HBASE-24033 Add ut for loading the corrupt recovered hfiles (#1322)
[hbase.git] / hbase-server / src / test / java / org / apache / hadoop / hbase / regionserver / TestHRegionFileSystem.java
blob565134b4ce407c7ca9b3831e9ad1aa1d63fba390
1 /**
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org.apache.hadoop.hbase.regionserver;
20 import static org.junit.Assert.assertEquals;
21 import static org.junit.Assert.assertFalse;
22 import static org.junit.Assert.assertNotNull;
23 import static org.junit.Assert.assertNull;
24 import static org.junit.Assert.assertTrue;
26 import java.io.IOException;
27 import java.net.URI;
28 import java.util.Collection;
29 import java.util.List;
30 import org.apache.hadoop.conf.Configuration;
31 import org.apache.hadoop.fs.FSDataInputStream;
32 import org.apache.hadoop.fs.FSDataOutputStream;
33 import org.apache.hadoop.fs.FileStatus;
34 import org.apache.hadoop.fs.FileSystem;
35 import org.apache.hadoop.fs.Path;
36 import org.apache.hadoop.fs.permission.FsPermission;
37 import org.apache.hadoop.hbase.HBaseClassTestRule;
38 import org.apache.hadoop.hbase.HBaseTestingUtility;
39 import org.apache.hadoop.hbase.TableName;
40 import org.apache.hadoop.hbase.client.Admin;
41 import org.apache.hadoop.hbase.client.ColumnFamilyDescriptorBuilder;
42 import org.apache.hadoop.hbase.client.Connection;
43 import org.apache.hadoop.hbase.client.Put;
44 import org.apache.hadoop.hbase.client.RegionInfo;
45 import org.apache.hadoop.hbase.client.RegionInfoBuilder;
46 import org.apache.hadoop.hbase.client.Table;
47 import org.apache.hadoop.hbase.fs.HFileSystem;
48 import org.apache.hadoop.hbase.testclassification.LargeTests;
49 import org.apache.hadoop.hbase.testclassification.RegionServerTests;
50 import org.apache.hadoop.hbase.util.Bytes;
51 import org.apache.hadoop.hbase.util.FSUtils;
52 import org.apache.hadoop.util.Progressable;
53 import org.junit.ClassRule;
54 import org.junit.Rule;
55 import org.junit.Test;
56 import org.junit.experimental.categories.Category;
57 import org.junit.rules.TestName;
58 import org.slf4j.Logger;
59 import org.slf4j.LoggerFactory;
61 @Category({RegionServerTests.class, LargeTests.class})
62 public class TestHRegionFileSystem {
64 @ClassRule
65 public static final HBaseClassTestRule CLASS_RULE =
66 HBaseClassTestRule.forClass(TestHRegionFileSystem.class);
68 private static HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
69 private static final Logger LOG = LoggerFactory.getLogger(TestHRegionFileSystem.class);
71 public static final byte[] FAMILY_NAME = Bytes.toBytes("info");
72 private static final byte[][] FAMILIES = {
73 Bytes.add(FAMILY_NAME, Bytes.toBytes("-A")),
74 Bytes.add(FAMILY_NAME, Bytes.toBytes("-B")) };
75 private static final TableName TABLE_NAME = TableName.valueOf("TestTable");
77 @Rule
78 public TestName name = new TestName();
80 @Test
81 public void testBlockStoragePolicy() throws Exception {
82 TEST_UTIL = new HBaseTestingUtility();
83 Configuration conf = TEST_UTIL.getConfiguration();
84 TEST_UTIL.startMiniCluster();
85 Table table = TEST_UTIL.createTable(TABLE_NAME, FAMILIES);
86 assertEquals("Should start with empty table", 0, TEST_UTIL.countRows(table));
87 HRegionFileSystem regionFs = getHRegionFS(TEST_UTIL.getConnection(), table, conf);
88 // the original block storage policy would be HOT
89 String spA = regionFs.getStoragePolicyName(Bytes.toString(FAMILIES[0]));
90 String spB = regionFs.getStoragePolicyName(Bytes.toString(FAMILIES[1]));
91 LOG.debug("Storage policy of cf 0: [" + spA + "].");
92 LOG.debug("Storage policy of cf 1: [" + spB + "].");
93 assertEquals("HOT", spA);
94 assertEquals("HOT", spB);
96 // Recreate table and make sure storage policy could be set through configuration
97 TEST_UTIL.shutdownMiniCluster();
98 TEST_UTIL.getConfiguration().set(HStore.BLOCK_STORAGE_POLICY_KEY, "WARM");
99 TEST_UTIL.startMiniCluster();
100 table = TEST_UTIL.createTable(TABLE_NAME, FAMILIES);
101 regionFs = getHRegionFS(TEST_UTIL.getConnection(), table, conf);
103 try (Admin admin = TEST_UTIL.getConnection().getAdmin()) {
104 spA = regionFs.getStoragePolicyName(Bytes.toString(FAMILIES[0]));
105 spB = regionFs.getStoragePolicyName(Bytes.toString(FAMILIES[1]));
106 LOG.debug("Storage policy of cf 0: [" + spA + "].");
107 LOG.debug("Storage policy of cf 1: [" + spB + "].");
108 assertEquals("WARM", spA);
109 assertEquals("WARM", spB);
111 // alter table cf schema to change storage policies
112 // and make sure it could override settings in conf
113 ColumnFamilyDescriptorBuilder cfdA =
114 ColumnFamilyDescriptorBuilder.newBuilder(FAMILIES[0]);
115 // alter through setting HStore#BLOCK_STORAGE_POLICY_KEY in HColumnDescriptor
116 cfdA.setValue(HStore.BLOCK_STORAGE_POLICY_KEY, "ONE_SSD");
117 admin.modifyColumnFamily(TABLE_NAME, cfdA.build());
118 while (TEST_UTIL.getMiniHBaseCluster().getMaster().getAssignmentManager().
119 getRegionStates().hasRegionsInTransition()) {
120 Thread.sleep(200);
121 LOG.debug("Waiting on table to finish schema altering");
123 // alter through HColumnDescriptor#setStoragePolicy
124 ColumnFamilyDescriptorBuilder cfdB =
125 ColumnFamilyDescriptorBuilder.newBuilder(FAMILIES[1]);
126 cfdB.setStoragePolicy("ALL_SSD");
127 admin.modifyColumnFamily(TABLE_NAME, cfdB.build());
128 while (TEST_UTIL.getMiniHBaseCluster().getMaster().getAssignmentManager().getRegionStates()
129 .hasRegionsInTransition()) {
130 Thread.sleep(200);
131 LOG.debug("Waiting on table to finish schema altering");
133 spA = regionFs.getStoragePolicyName(Bytes.toString(FAMILIES[0]));
134 spB = regionFs.getStoragePolicyName(Bytes.toString(FAMILIES[1]));
135 LOG.debug("Storage policy of cf 0: [" + spA + "].");
136 LOG.debug("Storage policy of cf 1: [" + spB + "].");
137 assertNotNull(spA);
138 assertEquals("ONE_SSD", spA);
139 assertNotNull(spB);
140 assertEquals("ALL_SSD", spB);
142 // flush memstore snapshot into 3 files
143 for (long i = 0; i < 3; i++) {
144 Put put = new Put(Bytes.toBytes(i));
145 put.addColumn(FAMILIES[0], Bytes.toBytes(i), Bytes.toBytes(i));
146 table.put(put);
147 admin.flush(TABLE_NAME);
149 // there should be 3 files in store dir
150 FileSystem fs = TEST_UTIL.getDFSCluster().getFileSystem();
151 Path storePath = regionFs.getStoreDir(Bytes.toString(FAMILIES[0]));
152 FileStatus[] storeFiles = FSUtils.listStatus(fs, storePath);
153 assertNotNull(storeFiles);
154 assertEquals(3, storeFiles.length);
155 // store temp dir still exists but empty
156 Path storeTempDir = new Path(regionFs.getTempDir(), Bytes.toString(FAMILIES[0]));
157 assertTrue(fs.exists(storeTempDir));
158 FileStatus[] tempFiles = FSUtils.listStatus(fs, storeTempDir);
159 assertNull(tempFiles);
160 // storage policy of cf temp dir and 3 store files should be ONE_SSD
161 assertEquals("ONE_SSD",
162 ((HFileSystem) regionFs.getFileSystem()).getStoragePolicyName(storeTempDir));
163 for (FileStatus status : storeFiles) {
164 assertEquals("ONE_SSD",
165 ((HFileSystem) regionFs.getFileSystem()).getStoragePolicyName(status.getPath()));
168 // change storage policies by calling raw api directly
169 regionFs.setStoragePolicy(Bytes.toString(FAMILIES[0]), "ALL_SSD");
170 regionFs.setStoragePolicy(Bytes.toString(FAMILIES[1]), "ONE_SSD");
171 spA = regionFs.getStoragePolicyName(Bytes.toString(FAMILIES[0]));
172 spB = regionFs.getStoragePolicyName(Bytes.toString(FAMILIES[1]));
173 LOG.debug("Storage policy of cf 0: [" + spA + "].");
174 LOG.debug("Storage policy of cf 1: [" + spB + "].");
175 assertNotNull(spA);
176 assertEquals("ALL_SSD", spA);
177 assertNotNull(spB);
178 assertEquals("ONE_SSD", spB);
179 } finally {
180 table.close();
181 TEST_UTIL.deleteTable(TABLE_NAME);
182 TEST_UTIL.shutdownMiniCluster();
186 private HRegionFileSystem getHRegionFS(Connection conn, Table table, Configuration conf)
187 throws IOException {
188 FileSystem fs = TEST_UTIL.getDFSCluster().getFileSystem();
189 Path tableDir = FSUtils.getTableDir(TEST_UTIL.getDefaultRootDirPath(), table.getName());
190 List<Path> regionDirs = FSUtils.getRegionDirs(fs, tableDir);
191 assertEquals(1, regionDirs.size());
192 List<Path> familyDirs = FSUtils.getFamilyDirs(fs, regionDirs.get(0));
193 assertEquals(2, familyDirs.size());
194 RegionInfo hri =
195 conn.getRegionLocator(table.getName()).getAllRegionLocations().get(0).getRegion();
196 HRegionFileSystem regionFs = new HRegionFileSystem(conf, new HFileSystem(fs), tableDir, hri);
197 return regionFs;
200 @Test
201 public void testOnDiskRegionCreation() throws IOException {
202 Path rootDir = TEST_UTIL.getDataTestDirOnTestFS(name.getMethodName());
203 FileSystem fs = TEST_UTIL.getTestFileSystem();
204 Configuration conf = TEST_UTIL.getConfiguration();
206 // Create a Region
207 RegionInfo hri = RegionInfoBuilder.newBuilder(TableName.valueOf(name.getMethodName())).build();
208 HRegionFileSystem regionFs = HRegionFileSystem.createRegionOnFileSystem(conf, fs,
209 FSUtils.getTableDir(rootDir, hri.getTable()), hri);
211 // Verify if the region is on disk
212 Path regionDir = regionFs.getRegionDir();
213 assertTrue("The region folder should be created", fs.exists(regionDir));
215 // Verify the .regioninfo
216 RegionInfo hriVerify = HRegionFileSystem.loadRegionInfoFileContent(fs, regionDir);
217 assertEquals(hri, hriVerify);
219 // Open the region
220 regionFs = HRegionFileSystem.openRegionFromFileSystem(conf, fs,
221 FSUtils.getTableDir(rootDir, hri.getTable()), hri, false);
222 assertEquals(regionDir, regionFs.getRegionDir());
224 // Delete the region
225 HRegionFileSystem.deleteRegionFromFileSystem(conf, fs,
226 FSUtils.getTableDir(rootDir, hri.getTable()), hri);
227 assertFalse("The region folder should be removed", fs.exists(regionDir));
229 fs.delete(rootDir, true);
232 @Test
233 public void testNonIdempotentOpsWithRetries() throws IOException {
234 Path rootDir = TEST_UTIL.getDataTestDirOnTestFS(name.getMethodName());
235 FileSystem fs = TEST_UTIL.getTestFileSystem();
236 Configuration conf = TEST_UTIL.getConfiguration();
238 // Create a Region
239 RegionInfo hri = RegionInfoBuilder.newBuilder(TableName.valueOf(name.getMethodName())).build();
240 HRegionFileSystem regionFs = HRegionFileSystem.createRegionOnFileSystem(conf, fs, rootDir, hri);
241 assertTrue(fs.exists(regionFs.getRegionDir()));
243 regionFs = new HRegionFileSystem(conf, new MockFileSystemForCreate(), rootDir, hri);
244 boolean result = regionFs.createDir(new Path("/foo/bar"));
245 assertTrue("Couldn't create the directory", result);
247 regionFs = new HRegionFileSystem(conf, new MockFileSystem(), rootDir, hri);
248 result = regionFs.rename(new Path("/foo/bar"), new Path("/foo/bar2"));
249 assertTrue("Couldn't rename the directory", result);
251 regionFs = new HRegionFileSystem(conf, new MockFileSystem(), rootDir, hri);
252 result = regionFs.deleteDir(new Path("/foo/bar"));
253 assertTrue("Couldn't delete the directory", result);
254 fs.delete(rootDir, true);
257 static class MockFileSystemForCreate extends MockFileSystem {
258 @Override
259 public boolean exists(Path path) {
260 return false;
265 * a mock fs which throws exception for first 3 times, and then process the call (returns the
266 * excepted result).
268 static class MockFileSystem extends FileSystem {
269 int retryCount;
270 final static int successRetryCount = 3;
272 public MockFileSystem() {
273 retryCount = 0;
276 @Override
277 public FSDataOutputStream append(Path arg0, int arg1, Progressable arg2) throws IOException {
278 throw new IOException("");
281 @Override
282 public FSDataOutputStream create(Path arg0, FsPermission arg1, boolean arg2, int arg3,
283 short arg4, long arg5, Progressable arg6) throws IOException {
284 LOG.debug("Create, " + retryCount);
285 if (retryCount++ < successRetryCount) throw new IOException("Something bad happen");
286 return null;
289 @Override
290 public boolean delete(Path arg0) throws IOException {
291 if (retryCount++ < successRetryCount) throw new IOException("Something bad happen");
292 return true;
295 @Override
296 public boolean delete(Path arg0, boolean arg1) throws IOException {
297 if (retryCount++ < successRetryCount) throw new IOException("Something bad happen");
298 return true;
301 @Override
302 public FileStatus getFileStatus(Path arg0) throws IOException {
303 FileStatus fs = new FileStatus();
304 return fs;
307 @Override
308 public boolean exists(Path path) {
309 return true;
312 @Override
313 public URI getUri() {
314 throw new RuntimeException("Something bad happen");
317 @Override
318 public Path getWorkingDirectory() {
319 throw new RuntimeException("Something bad happen");
322 @Override
323 public FileStatus[] listStatus(Path arg0) throws IOException {
324 throw new IOException("Something bad happen");
327 @Override
328 public boolean mkdirs(Path arg0, FsPermission arg1) throws IOException {
329 LOG.debug("mkdirs, " + retryCount);
330 if (retryCount++ < successRetryCount) throw new IOException("Something bad happen");
331 return true;
334 @Override
335 public FSDataInputStream open(Path arg0, int arg1) throws IOException {
336 throw new IOException("Something bad happen");
339 @Override
340 public boolean rename(Path arg0, Path arg1) throws IOException {
341 LOG.debug("rename, " + retryCount);
342 if (retryCount++ < successRetryCount) throw new IOException("Something bad happen");
343 return true;
346 @Override
347 public void setWorkingDirectory(Path arg0) {
348 throw new RuntimeException("Something bad happen");
352 @Test
353 public void testTempAndCommit() throws IOException {
354 Path rootDir = TEST_UTIL.getDataTestDirOnTestFS("testTempAndCommit");
355 FileSystem fs = TEST_UTIL.getTestFileSystem();
356 Configuration conf = TEST_UTIL.getConfiguration();
358 // Create a Region
359 String familyName = "cf";
361 RegionInfo hri = RegionInfoBuilder.newBuilder(TableName.valueOf(name.getMethodName())).build();
362 HRegionFileSystem regionFs = HRegionFileSystem.createRegionOnFileSystem(conf, fs, rootDir, hri);
364 // New region, no store files
365 Collection<StoreFileInfo> storeFiles = regionFs.getStoreFiles(familyName);
366 assertEquals(0, storeFiles != null ? storeFiles.size() : 0);
368 // Create a new file in temp (no files in the family)
369 Path buildPath = regionFs.createTempName();
370 fs.createNewFile(buildPath);
371 storeFiles = regionFs.getStoreFiles(familyName);
372 assertEquals(0, storeFiles != null ? storeFiles.size() : 0);
374 // commit the file
375 Path dstPath = regionFs.commitStoreFile(familyName, buildPath);
376 storeFiles = regionFs.getStoreFiles(familyName);
377 assertEquals(0, storeFiles != null ? storeFiles.size() : 0);
378 assertFalse(fs.exists(buildPath));
380 fs.delete(rootDir, true);