001/* 002 * Licensed to the Apache Software Foundation (ASF) under one 003 * or more contributor license agreements. See the NOTICE file 004 * distributed with this work for additional information 005 * regarding copyright ownership. The ASF licenses this file 006 * to you under the Apache License, Version 2.0 (the 007 * "License"); you may not use this file except in compliance 008 * with the License. You may obtain a copy of the License at 009 * 010 * http://www.apache.org/licenses/LICENSE-2.0 011 * 012 * Unless required by applicable law or agreed to in writing, software 013 * distributed under the License is distributed on an "AS IS" BASIS, 014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. 015 * See the License for the specific language governing permissions and 016 * limitations under the License. 017 */ 018package org.apache.hadoop.hbase.snapshot; 019 020import static org.junit.Assert.assertEquals; 021import static org.junit.Assert.assertFalse; 022import static org.junit.Assert.assertTrue; 023 024import java.io.IOException; 025import java.util.ArrayList; 026import java.util.Collections; 027import java.util.HashSet; 028import java.util.List; 029import java.util.Map; 030import java.util.Set; 031import java.util.TreeSet; 032import org.apache.hadoop.conf.Configuration; 033import org.apache.hadoop.fs.FSDataInputStream; 034import org.apache.hadoop.fs.FSDataOutputStream; 035import org.apache.hadoop.fs.FileStatus; 036import org.apache.hadoop.fs.FileSystem; 037import org.apache.hadoop.fs.Path; 038import org.apache.hadoop.fs.PathFilter; 039import org.apache.hadoop.hbase.HBaseTestingUtil; 040import org.apache.hadoop.hbase.HConstants; 041import org.apache.hadoop.hbase.TableName; 042import org.apache.hadoop.hbase.TableNotEnabledException; 043import org.apache.hadoop.hbase.client.Admin; 044import org.apache.hadoop.hbase.client.BufferedMutator; 045import org.apache.hadoop.hbase.client.ColumnFamilyDescriptorBuilder; 046import org.apache.hadoop.hbase.client.Durability; 047import org.apache.hadoop.hbase.client.Put; 048import org.apache.hadoop.hbase.client.RegionInfo; 049import org.apache.hadoop.hbase.client.RegionInfoBuilder; 050import org.apache.hadoop.hbase.client.RegionReplicaUtil; 051import org.apache.hadoop.hbase.client.SnapshotDescription; 052import org.apache.hadoop.hbase.client.SnapshotType; 053import org.apache.hadoop.hbase.client.Table; 054import org.apache.hadoop.hbase.client.TableDescriptor; 055import org.apache.hadoop.hbase.client.TableDescriptorBuilder; 056import org.apache.hadoop.hbase.errorhandling.ForeignExceptionDispatcher; 057import org.apache.hadoop.hbase.io.HFileLink; 058import org.apache.hadoop.hbase.master.HMaster; 059import org.apache.hadoop.hbase.master.MasterFileSystem; 060import org.apache.hadoop.hbase.mob.MobUtils; 061import org.apache.hadoop.hbase.regionserver.HRegion; 062import org.apache.hadoop.hbase.regionserver.HRegionFileSystem; 063import org.apache.hadoop.hbase.regionserver.HRegionServer; 064import org.apache.hadoop.hbase.util.Bytes; 065import org.apache.hadoop.hbase.util.CommonFSUtils; 066import org.apache.hadoop.hbase.util.EnvironmentEdgeManager; 067import org.apache.hadoop.hbase.util.FSTableDescriptors; 068import org.apache.hadoop.hbase.util.FSVisitor; 069import org.apache.hadoop.hbase.util.MD5Hash; 070import org.apache.yetus.audience.InterfaceAudience; 071import org.junit.Assert; 072import org.slf4j.Logger; 073import org.slf4j.LoggerFactory; 074 075import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil; 076import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.IsSnapshotDoneRequest; 077import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos; 078import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotRegionManifest; 079 080/** 081 * Utilities class for snapshots 082 */ 083@InterfaceAudience.Private 084public final class SnapshotTestingUtils { 085 private static final Logger LOG = LoggerFactory.getLogger(SnapshotTestingUtils.class); 086 087 // default number of regions (and keys) given by getSplitKeys() and createTable() 088 private static byte[] KEYS = Bytes.toBytes("0123456"); 089 090 private SnapshotTestingUtils() { 091 // private constructor for utility class 092 } 093 094 /** 095 * Assert that we don't have any snapshots lists if the admin operation fails 096 */ 097 public static void assertNoSnapshots(Admin admin) throws IOException { 098 assertEquals("Have some previous snapshots", 0, admin.listSnapshots().size()); 099 } 100 101 /** 102 * Make sure that there is only one snapshot returned from the master and its name and table match 103 * the passed in parameters. 104 */ 105 public static List<SnapshotDescription> assertExistsMatchingSnapshot(Admin admin, 106 String snapshotName, TableName tableName) throws IOException { 107 // list the snapshot 108 List<SnapshotDescription> snapshots = admin.listSnapshots(); 109 110 List<SnapshotDescription> returnedSnapshots = new ArrayList<>(); 111 for (SnapshotDescription sd : snapshots) { 112 if (snapshotName.equals(sd.getName()) && tableName.equals(sd.getTableName())) { 113 returnedSnapshots.add(sd); 114 } 115 } 116 117 Assert.assertTrue("No matching snapshots found.", returnedSnapshots.size() > 0); 118 return returnedSnapshots; 119 } 120 121 /** 122 * Make sure that there is only one snapshot returned from the master 123 */ 124 public static void assertOneSnapshotThatMatches(Admin admin, 125 SnapshotProtos.SnapshotDescription snapshot) throws IOException { 126 assertOneSnapshotThatMatches(admin, snapshot.getName(), TableName.valueOf(snapshot.getTable())); 127 } 128 129 /** 130 * Make sure that there is only one snapshot returned from the master and its name and table match 131 * the passed in parameters. 132 */ 133 public static List<SnapshotDescription> assertOneSnapshotThatMatches(Admin admin, 134 String snapshotName, TableName tableName) throws IOException { 135 // list the snapshot 136 List<SnapshotDescription> snapshots = admin.listSnapshots(); 137 138 assertEquals("Should only have 1 snapshot", 1, snapshots.size()); 139 assertEquals(snapshotName, snapshots.get(0).getName()); 140 assertEquals(tableName, snapshots.get(0).getTableName()); 141 142 return snapshots; 143 } 144 145 /** 146 * Make sure that there is only one snapshot returned from the master and its name and table match 147 * the passed in parameters. 148 */ 149 public static List<SnapshotDescription> assertOneSnapshotThatMatches(Admin admin, byte[] snapshot, 150 TableName tableName) throws IOException { 151 return assertOneSnapshotThatMatches(admin, Bytes.toString(snapshot), tableName); 152 } 153 154 public static void confirmSnapshotValid(HBaseTestingUtil testUtil, 155 SnapshotProtos.SnapshotDescription snapshotDescriptor, TableName tableName, byte[] family) 156 throws IOException { 157 MasterFileSystem mfs = testUtil.getHBaseCluster().getMaster().getMasterFileSystem(); 158 confirmSnapshotValid(snapshotDescriptor, tableName, family, mfs.getRootDir(), 159 testUtil.getAdmin(), mfs.getFileSystem()); 160 } 161 162 /** 163 * Confirm that the snapshot contains references to all the files that should be in the snapshot. 164 */ 165 public static void confirmSnapshotValid(SnapshotProtos.SnapshotDescription snapshotDescriptor, 166 TableName tableName, byte[] testFamily, Path rootDir, Admin admin, FileSystem fs) 167 throws IOException { 168 ArrayList nonEmptyTestFamilies = new ArrayList(1); 169 nonEmptyTestFamilies.add(testFamily); 170 confirmSnapshotValid(snapshotDescriptor, tableName, nonEmptyTestFamilies, null, rootDir, admin, 171 fs); 172 } 173 174 /** 175 * Confirm that the snapshot has no references files but only metadata. 176 */ 177 public static void confirmEmptySnapshotValid( 178 SnapshotProtos.SnapshotDescription snapshotDescriptor, TableName tableName, byte[] testFamily, 179 Path rootDir, Admin admin, FileSystem fs) throws IOException { 180 ArrayList emptyTestFamilies = new ArrayList(1); 181 emptyTestFamilies.add(testFamily); 182 confirmSnapshotValid(snapshotDescriptor, tableName, null, emptyTestFamilies, rootDir, admin, 183 fs); 184 } 185 186 /** 187 * Confirm that the snapshot contains references to all the files that should be in the snapshot. 188 * This method also perform some redundant check like the existence of the snapshotinfo or the 189 * regioninfo which are done always by the MasterSnapshotVerifier, at the end of the snapshot 190 * operation. 191 */ 192 public static void confirmSnapshotValid(SnapshotProtos.SnapshotDescription snapshotDescriptor, 193 TableName tableName, List<byte[]> nonEmptyTestFamilies, List<byte[]> emptyTestFamilies, 194 Path rootDir, Admin admin, FileSystem fs) throws IOException { 195 final Configuration conf = admin.getConfiguration(); 196 197 // check snapshot dir 198 Path snapshotDir = 199 SnapshotDescriptionUtils.getCompletedSnapshotDir(snapshotDescriptor, rootDir); 200 assertTrue("target snapshot directory, '" + snapshotDir + "', doesn't exist.", 201 fs.exists(snapshotDir)); 202 203 SnapshotProtos.SnapshotDescription desc = 204 SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir); 205 206 // Extract regions and families with store files 207 final Set<byte[]> snapshotFamilies = new TreeSet<>(Bytes.BYTES_COMPARATOR); 208 209 SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, desc); 210 Map<String, SnapshotRegionManifest> regionManifests = manifest.getRegionManifestsMap(); 211 for (SnapshotRegionManifest regionManifest : regionManifests.values()) { 212 SnapshotReferenceUtil.visitRegionStoreFiles(regionManifest, 213 new SnapshotReferenceUtil.StoreFileVisitor() { 214 @Override 215 public void storeFile(final RegionInfo regionInfo, final String family, 216 final SnapshotRegionManifest.StoreFile storeFile) throws IOException { 217 snapshotFamilies.add(Bytes.toBytes(family)); 218 } 219 }); 220 } 221 // Verify that there are store files in the specified families 222 if (nonEmptyTestFamilies != null) { 223 for (final byte[] familyName : nonEmptyTestFamilies) { 224 assertTrue("Expected snapshot to contain family '" + Bytes.toString(familyName) 225 + "', but it does not.", snapshotFamilies.contains(familyName)); 226 } 227 } 228 229 // Verify that there are no store files in the specified families 230 if (emptyTestFamilies != null) { 231 for (final byte[] familyName : emptyTestFamilies) { 232 assertFalse("Expected snapshot to skip empty family '" + Bytes.toString(familyName) 233 + "', but it is present.", snapshotFamilies.contains(familyName)); 234 } 235 } 236 237 // check the region snapshot for all the regions 238 List<RegionInfo> regions = admin.getRegions(tableName); 239 // remove the non-default regions 240 RegionReplicaUtil.removeNonDefaultRegions(regions); 241 boolean hasMob = 242 regionManifests.containsKey(MobUtils.getMobRegionInfo(tableName).getEncodedName()); 243 if (hasMob) { 244 assertEquals("Wrong number of regions.", regions.size(), regionManifests.size() - 1); 245 } else { 246 // if create snapshot when table splitting, parent region will be included to the snapshot 247 // region manifest. we should exclude the parent regions. 248 int regionCountExclusiveSplitParent = 0; 249 for (SnapshotRegionManifest snapshotRegionManifest : regionManifests.values()) { 250 RegionInfo hri = ProtobufUtil.toRegionInfo(snapshotRegionManifest.getRegionInfo()); 251 if (hri.isOffline() && (hri.isSplit() || hri.isSplitParent())) { 252 continue; 253 } 254 regionCountExclusiveSplitParent++; 255 } 256 assertEquals("Wrong number of regions.", regions.size(), regionCountExclusiveSplitParent); 257 } 258 259 // Verify Regions (redundant check, see MasterSnapshotVerifier) 260 for (RegionInfo info : regions) { 261 String regionName = info.getEncodedName(); 262 assertTrue("Missing region name: '" + regionName + "'", 263 regionManifests.containsKey(regionName)); 264 } 265 } 266 267 /* 268 * Take snapshot with maximum of numTries attempts, ignoring CorruptedSnapshotException except for 269 * the last CorruptedSnapshotException 270 */ 271 public static void snapshot(Admin admin, final String snapshotName, final TableName tableName, 272 final SnapshotType type, final int numTries) throws IOException { 273 snapshot(admin, snapshotName, tableName, type, numTries, null); 274 } 275 276 /* 277 * Take snapshot having snapshot properties with maximum of numTries attempts, ignoring 278 * CorruptedSnapshotException except for the last CorruptedSnapshotException 279 */ 280 public static void snapshot(Admin admin, final String snapshotName, final TableName tableName, 281 final SnapshotType type, final int numTries, Map<String, Object> snapshotProps) 282 throws IOException { 283 int tries = 0; 284 CorruptedSnapshotException lastEx = null; 285 while (tries++ < numTries) { 286 try { 287 admin.snapshot(snapshotName, tableName, type, snapshotProps); 288 return; 289 } catch (CorruptedSnapshotException cse) { 290 LOG.warn("Got CorruptedSnapshotException", cse); 291 lastEx = cse; 292 } 293 } 294 throw lastEx; 295 } 296 297 public static void cleanupSnapshot(Admin admin, byte[] tableName) throws IOException { 298 SnapshotTestingUtils.cleanupSnapshot(admin, Bytes.toString(tableName)); 299 } 300 301 public static void cleanupSnapshot(Admin admin, String snapshotName) throws IOException { 302 // delete the taken snapshot 303 admin.deleteSnapshot(snapshotName); 304 assertNoSnapshots(admin); 305 } 306 307 /** 308 * Expect the snapshot to throw an error when checking if the snapshot is complete 309 * @param master master to check 310 * @param snapshot the {@link SnapshotDescription} request to pass to the master 311 * @param clazz expected exception from the master 312 */ 313 public static void expectSnapshotDoneException(HMaster master, IsSnapshotDoneRequest snapshot, 314 Class<? extends HBaseSnapshotException> clazz) { 315 try { 316 master.getMasterRpcServices().isSnapshotDone(null, snapshot); 317 Assert.fail("didn't fail to lookup a snapshot"); 318 } catch (org.apache.hbase.thirdparty.com.google.protobuf.ServiceException se) { 319 try { 320 throw ProtobufUtil.handleRemoteException(se); 321 } catch (HBaseSnapshotException e) { 322 assertEquals("Threw wrong snapshot exception!", clazz, e.getClass()); 323 } catch (Throwable t) { 324 Assert.fail("Threw an unexpected exception:" + t); 325 } 326 } 327 } 328 329 /** 330 * List all the HFiles in the given table 331 * @param fs FileSystem where the table lives 332 * @param tableDir directory of the table 333 * @return array of the current HFiles in the table (could be a zero-length array) 334 * @throws IOException on unexecpted error reading the FS 335 */ 336 public static ArrayList<String> listHFileNames(final FileSystem fs, final Path tableDir) 337 throws IOException { 338 final ArrayList<String> hfiles = new ArrayList<>(); 339 FSVisitor.visitTableStoreFiles(fs, tableDir, new FSVisitor.StoreFileVisitor() { 340 @Override 341 public void storeFile(final String region, final String family, final String hfileName) 342 throws IOException { 343 hfiles.add(hfileName); 344 } 345 }); 346 Collections.sort(hfiles); 347 return hfiles; 348 } 349 350 /** 351 * Take a snapshot of the specified table and verify that the given family is not empty. Note that 352 * this will leave the table disabled in the case of an offline snapshot. 353 */ 354 public static void createSnapshotAndValidate(Admin admin, TableName tableName, String familyName, 355 String snapshotNameString, Path rootDir, FileSystem fs, boolean onlineSnapshot) 356 throws Exception { 357 ArrayList<byte[]> nonEmptyFamilyNames = new ArrayList<>(1); 358 nonEmptyFamilyNames.add(Bytes.toBytes(familyName)); 359 createSnapshotAndValidate(admin, tableName, nonEmptyFamilyNames, /* emptyFamilyNames= */ null, 360 snapshotNameString, rootDir, fs, onlineSnapshot); 361 } 362 363 /** 364 * Take a snapshot of the specified table and verify the given families. Note that this will leave 365 * the table disabled in the case of an offline snapshot. 366 */ 367 public static void createSnapshotAndValidate(Admin admin, TableName tableName, 368 List<byte[]> nonEmptyFamilyNames, List<byte[]> emptyFamilyNames, String snapshotNameString, 369 Path rootDir, FileSystem fs, boolean onlineSnapshot) throws Exception { 370 if (!onlineSnapshot) { 371 try { 372 LOG.info("prepping for offline snapshot."); 373 admin.disableTable(tableName); 374 } catch (TableNotEnabledException tne) { 375 LOG.info("In attempting to disable " + tableName + " it turns out that the this table is " 376 + "already disabled."); 377 } 378 } 379 LOG.info("taking snapshot."); 380 admin.snapshot(snapshotNameString, tableName); 381 382 LOG.info("Confirming snapshot exists."); 383 List<SnapshotDescription> snapshots = 384 SnapshotTestingUtils.assertExistsMatchingSnapshot(admin, snapshotNameString, tableName); 385 if (snapshots == null || snapshots.size() != 1) { 386 Assert.fail("Incorrect number of snapshots for table " + tableName); 387 } 388 389 LOG.info("validating snapshot."); 390 SnapshotTestingUtils.confirmSnapshotValid( 391 ProtobufUtil.createHBaseProtosSnapshotDesc(snapshots.get(0)), tableName, nonEmptyFamilyNames, 392 emptyFamilyNames, rootDir, admin, fs); 393 } 394 395 /** 396 * Corrupt the specified snapshot by deleting some files. 397 * @param util {@link HBaseTestingUtil} 398 * @param snapshotName name of the snapshot to corrupt 399 * @return array of the corrupted HFiles 400 * @throws IOException on unexecpted error reading the FS 401 */ 402 public static ArrayList corruptSnapshot(final HBaseTestingUtil util, final String snapshotName) 403 throws IOException { 404 final MasterFileSystem mfs = util.getHBaseCluster().getMaster().getMasterFileSystem(); 405 final FileSystem fs = mfs.getFileSystem(); 406 407 Path snapshotDir = 408 SnapshotDescriptionUtils.getCompletedSnapshotDir(snapshotName, mfs.getRootDir()); 409 SnapshotProtos.SnapshotDescription snapshotDesc = 410 SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir); 411 final TableName table = TableName.valueOf(snapshotDesc.getTable()); 412 413 final ArrayList corruptedFiles = new ArrayList(); 414 final Configuration conf = util.getConfiguration(); 415 SnapshotReferenceUtil.visitTableStoreFiles(conf, fs, snapshotDir, snapshotDesc, 416 new SnapshotReferenceUtil.StoreFileVisitor() { 417 @Override 418 public void storeFile(final RegionInfo regionInfo, final String family, 419 final SnapshotRegionManifest.StoreFile storeFile) throws IOException { 420 String region = regionInfo.getEncodedName(); 421 String hfile = storeFile.getName(); 422 HFileLink link = HFileLink.build(conf, table, region, family, hfile); 423 if (corruptedFiles.size() % 2 == 0) { 424 fs.delete(link.getAvailablePath(fs), true); 425 corruptedFiles.add(hfile); 426 } 427 } 428 }); 429 430 assertTrue(corruptedFiles.size() > 0); 431 return corruptedFiles; 432 } 433 434 // ========================================================================== 435 // Snapshot Mock 436 // ========================================================================== 437 public static class SnapshotMock { 438 protected final static String TEST_FAMILY = "cf"; 439 public final static int TEST_NUM_REGIONS = 4; 440 441 private final Configuration conf; 442 private final FileSystem fs; 443 private final Path rootDir; 444 445 static class RegionData { 446 public RegionInfo hri; 447 public Path tableDir; 448 public Path[] files; 449 450 public RegionData(final Path tableDir, final RegionInfo hri, final int nfiles) { 451 this.tableDir = tableDir; 452 this.hri = hri; 453 this.files = new Path[nfiles]; 454 } 455 } 456 457 public static class SnapshotBuilder { 458 private final RegionData[] tableRegions; 459 private final SnapshotProtos.SnapshotDescription desc; 460 private final TableDescriptor htd; 461 private final Configuration conf; 462 private final FileSystem fs; 463 private final Path rootDir; 464 private Path snapshotDir; 465 private int snapshotted = 0; 466 467 public SnapshotBuilder(final Configuration conf, final FileSystem fs, final Path rootDir, 468 final TableDescriptor htd, final SnapshotProtos.SnapshotDescription desc, 469 final RegionData[] tableRegions) throws IOException { 470 this.fs = fs; 471 this.conf = conf; 472 this.rootDir = rootDir; 473 this.htd = htd; 474 this.desc = desc; 475 this.tableRegions = tableRegions; 476 this.snapshotDir = SnapshotDescriptionUtils.getWorkingSnapshotDir(desc, rootDir, conf); 477 FSTableDescriptors.createTableDescriptorForTableDirectory( 478 this.snapshotDir.getFileSystem(conf), snapshotDir, htd, false); 479 } 480 481 public TableDescriptor getTableDescriptor() { 482 return this.htd; 483 } 484 485 public SnapshotProtos.SnapshotDescription getSnapshotDescription() { 486 return this.desc; 487 } 488 489 public Path getSnapshotsDir() { 490 return this.snapshotDir; 491 } 492 493 public Path[] addRegion() throws IOException { 494 return addRegion(desc); 495 } 496 497 public Path[] addRegionV1() throws IOException { 498 return addRegion( 499 desc.toBuilder().setVersion(SnapshotManifestV1.DESCRIPTOR_VERSION).build()); 500 } 501 502 public Path[] addRegionV2() throws IOException { 503 return addRegion( 504 desc.toBuilder().setVersion(SnapshotManifestV2.DESCRIPTOR_VERSION).build()); 505 } 506 507 private Path[] addRegion(final SnapshotProtos.SnapshotDescription desc) throws IOException { 508 if (this.snapshotted == tableRegions.length) { 509 throw new UnsupportedOperationException("No more regions in the table"); 510 } 511 512 RegionData regionData = tableRegions[this.snapshotted++]; 513 ForeignExceptionDispatcher monitor = new ForeignExceptionDispatcher(desc.getName()); 514 SnapshotManifest manifest = SnapshotManifest.create(conf, fs, snapshotDir, desc, monitor); 515 manifest.addTableDescriptor(htd); 516 manifest.addRegion(regionData.tableDir, regionData.hri); 517 return regionData.files; 518 } 519 520 private void corruptFile(Path p) throws IOException { 521 String manifestName = p.getName(); 522 523 // Rename the original region-manifest file 524 Path newP = new Path(p.getParent(), manifestName + "1"); 525 fs.rename(p, newP); 526 527 // Create a new region-manifest file 528 FSDataOutputStream out = fs.create(p); 529 530 // Copy the first 25 bytes of the original region-manifest into the new one, 531 // make it a corrupted region-manifest file. 532 FSDataInputStream input = fs.open(newP); 533 byte[] buffer = new byte[25]; 534 int len = input.read(0, buffer, 0, 25); 535 if (len > 1) { 536 out.write(buffer, 0, len - 1); 537 } 538 out.close(); 539 540 // Delete the original region-manifest 541 fs.delete(newP); 542 } 543 544 /** 545 * Corrupt one region-manifest file 546 * @throws IOException on unexecpted error from the FS 547 */ 548 public void corruptOneRegionManifest() throws IOException { 549 FileStatus[] manifestFiles = CommonFSUtils.listStatus(fs, snapshotDir, new PathFilter() { 550 @Override 551 public boolean accept(Path path) { 552 return path.getName().startsWith(SnapshotManifestV2.SNAPSHOT_MANIFEST_PREFIX); 553 } 554 }); 555 556 if (manifestFiles.length == 0) return; 557 558 // Just choose the first one 559 Path p = manifestFiles[0].getPath(); 560 corruptFile(p); 561 } 562 563 public void missOneRegionSnapshotFile() throws IOException { 564 FileStatus[] manifestFiles = CommonFSUtils.listStatus(fs, snapshotDir); 565 for (FileStatus fileStatus : manifestFiles) { 566 String fileName = fileStatus.getPath().getName(); 567 if ( 568 fileName.endsWith(SnapshotDescriptionUtils.SNAPSHOTINFO_FILE) 569 || fileName.endsWith(".tabledesc") 570 || fileName.endsWith(SnapshotDescriptionUtils.SNAPSHOT_TMP_DIR_NAME) 571 ) { 572 fs.delete(fileStatus.getPath(), true); 573 } 574 } 575 } 576 577 /** 578 * Corrupt data-manifest file 579 * @throws IOException on unexecpted error from the FS 580 */ 581 public void corruptDataManifest() throws IOException { 582 FileStatus[] manifestFiles = CommonFSUtils.listStatus(fs, snapshotDir, new PathFilter() { 583 @Override 584 public boolean accept(Path path) { 585 return path.getName().startsWith(SnapshotManifest.DATA_MANIFEST_NAME); 586 } 587 }); 588 589 if (manifestFiles.length == 0) return; 590 591 // Just choose the first one 592 Path p = manifestFiles[0].getPath(); 593 corruptFile(p); 594 } 595 596 public Path commit() throws IOException { 597 ForeignExceptionDispatcher monitor = new ForeignExceptionDispatcher(desc.getName()); 598 SnapshotManifest manifest = SnapshotManifest.create(conf, fs, snapshotDir, desc, monitor); 599 manifest.addTableDescriptor(htd); 600 manifest.consolidate(); 601 Path finishedDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(desc, rootDir); 602 SnapshotDescriptionUtils.completeSnapshot(finishedDir, snapshotDir, fs, 603 snapshotDir.getFileSystem(conf), conf); 604 snapshotDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(desc, rootDir); 605 return snapshotDir; 606 } 607 608 public void consolidate() throws IOException { 609 ForeignExceptionDispatcher monitor = new ForeignExceptionDispatcher(desc.getName()); 610 SnapshotManifest manifest = SnapshotManifest.create(conf, fs, snapshotDir, desc, monitor); 611 manifest.addTableDescriptor(htd); 612 manifest.consolidate(); 613 } 614 } 615 616 public SnapshotMock(final Configuration conf, final FileSystem fs, final Path rootDir) { 617 this.fs = fs; 618 this.conf = conf; 619 this.rootDir = rootDir; 620 } 621 622 public SnapshotBuilder createSnapshotV1(final String snapshotName, final String tableName) 623 throws IOException { 624 return createSnapshot(snapshotName, tableName, SnapshotManifestV1.DESCRIPTOR_VERSION); 625 } 626 627 public SnapshotBuilder createSnapshotV1(final String snapshotName, final String tableName, 628 final int numRegions) throws IOException { 629 return createSnapshot(snapshotName, tableName, numRegions, 630 SnapshotManifestV1.DESCRIPTOR_VERSION); 631 } 632 633 public SnapshotBuilder createSnapshotV2(final String snapshotName, final String tableName) 634 throws IOException { 635 return createSnapshot(snapshotName, tableName, SnapshotManifestV2.DESCRIPTOR_VERSION); 636 } 637 638 public SnapshotBuilder createSnapshotV2(final String snapshotName, final String tableName, 639 final int numRegions) throws IOException { 640 return createSnapshot(snapshotName, tableName, numRegions, 641 SnapshotManifestV2.DESCRIPTOR_VERSION); 642 } 643 644 public SnapshotBuilder createSnapshotV2(final String snapshotName, final String tableName, 645 final int numRegions, final long ttl) throws IOException { 646 return createSnapshot(snapshotName, tableName, numRegions, 647 SnapshotManifestV2.DESCRIPTOR_VERSION, ttl); 648 } 649 650 private SnapshotBuilder createSnapshot(final String snapshotName, final String tableName, 651 final int version) throws IOException { 652 return createSnapshot(snapshotName, tableName, TEST_NUM_REGIONS, version); 653 } 654 655 private SnapshotBuilder createSnapshot(final String snapshotName, final String tableName, 656 final int numRegions, final int version) throws IOException { 657 TableDescriptor htd = createHtd(tableName); 658 RegionData[] regions = createTable(htd, numRegions); 659 660 SnapshotProtos.SnapshotDescription desc = SnapshotProtos.SnapshotDescription.newBuilder() 661 .setTable(htd.getTableName().getNameAsString()).setName(snapshotName).setVersion(version) 662 .build(); 663 664 Path workingDir = SnapshotDescriptionUtils.getWorkingSnapshotDir(desc, rootDir, conf); 665 FileSystem workingFs = workingDir.getFileSystem(conf); 666 SnapshotDescriptionUtils.writeSnapshotInfo(desc, workingDir, workingFs); 667 return new SnapshotBuilder(conf, fs, rootDir, htd, desc, regions); 668 } 669 670 private SnapshotBuilder createSnapshot(final String snapshotName, final String tableName, 671 final int numRegions, final int version, final long ttl) throws IOException { 672 TableDescriptor htd = createHtd(tableName); 673 RegionData[] regions = createTable(htd, numRegions); 674 SnapshotProtos.SnapshotDescription desc = SnapshotProtos.SnapshotDescription.newBuilder() 675 .setTable(htd.getTableName().getNameAsString()).setName(snapshotName).setVersion(version) 676 .setCreationTime(EnvironmentEdgeManager.currentTime()).setTtl(ttl).build(); 677 Path workingDir = SnapshotDescriptionUtils.getWorkingSnapshotDir(desc, rootDir, conf); 678 SnapshotDescriptionUtils.writeSnapshotInfo(desc, workingDir, fs); 679 return new SnapshotBuilder(conf, fs, rootDir, htd, desc, regions); 680 } 681 682 public TableDescriptor createHtd(final String tableName) { 683 return TableDescriptorBuilder.newBuilder(TableName.valueOf(tableName)) 684 .setColumnFamily(ColumnFamilyDescriptorBuilder.of(TEST_FAMILY)).build(); 685 } 686 687 private RegionData[] createTable(final TableDescriptor htd, final int nregions) 688 throws IOException { 689 Path tableDir = CommonFSUtils.getTableDir(rootDir, htd.getTableName()); 690 new FSTableDescriptors(conf).createTableDescriptorForTableDirectory(tableDir, htd, false); 691 692 assertTrue(nregions % 2 == 0); 693 RegionData[] regions = new RegionData[nregions]; 694 for (int i = 0; i < regions.length; i += 2) { 695 byte[] startKey = Bytes.toBytes(0 + i * 2); 696 byte[] endKey = Bytes.toBytes(1 + i * 2); 697 698 // First region, simple with one plain hfile. 699 RegionInfo hri = RegionInfoBuilder.newBuilder(htd.getTableName()).setStartKey(startKey) 700 .setEndKey(endKey).build(); 701 HRegionFileSystem rfs = HRegionFileSystem.createRegionOnFileSystem(conf, fs, tableDir, hri); 702 regions[i] = new RegionData(tableDir, hri, 3); 703 for (int j = 0; j < regions[i].files.length; ++j) { 704 Path storeFile = createStoreFile(rfs.createTempName()); 705 regions[i].files[j] = rfs.commitStoreFile(TEST_FAMILY, storeFile); 706 } 707 708 // Second region, used to test the split case. 709 // This region contains a reference to the hfile in the first region. 710 startKey = Bytes.toBytes(2 + i * 2); 711 endKey = Bytes.toBytes(3 + i * 2); 712 hri = RegionInfoBuilder.newBuilder(htd.getTableName()).build(); 713 rfs = HRegionFileSystem.createRegionOnFileSystem(conf, fs, tableDir, hri); 714 regions[i + 1] = new RegionData(tableDir, hri, regions[i].files.length); 715 for (int j = 0; j < regions[i].files.length; ++j) { 716 String refName = regions[i].files[j].getName() + '.' + regions[i].hri.getEncodedName(); 717 Path refFile = createStoreFile(new Path(rootDir, refName)); 718 regions[i + 1].files[j] = rfs.commitStoreFile(TEST_FAMILY, refFile); 719 } 720 } 721 return regions; 722 } 723 724 private Path createStoreFile(final Path storeFile) throws IOException { 725 FSDataOutputStream out = fs.create(storeFile); 726 try { 727 out.write(Bytes.toBytes(storeFile.toString())); 728 } finally { 729 out.close(); 730 } 731 return storeFile; 732 } 733 } 734 735 // ========================================================================== 736 // Table Helpers 737 // ========================================================================== 738 public static void waitForTableToBeOnline(final HBaseTestingUtil util, final TableName tableName) 739 throws IOException, InterruptedException { 740 HRegionServer rs = util.getRSForFirstRegionInTable(tableName); 741 List<HRegion> onlineRegions = rs.getRegions(tableName); 742 for (HRegion region : onlineRegions) { 743 region.waitForFlushesAndCompactions(); 744 } 745 // Wait up to 60 seconds for a table to be available. 746 util.waitFor(60000, util.predicateTableAvailable(tableName)); 747 } 748 749 public static void createTable(final HBaseTestingUtil util, final TableName tableName, 750 int regionReplication, int nRegions, final byte[]... families) 751 throws IOException, InterruptedException { 752 TableDescriptorBuilder builder = 753 TableDescriptorBuilder.newBuilder(tableName).setRegionReplication(regionReplication); 754 for (byte[] family : families) { 755 builder.setColumnFamily(ColumnFamilyDescriptorBuilder.of(family)); 756 } 757 byte[][] splitKeys = getSplitKeys(nRegions); 758 util.createTable(builder.build(), splitKeys); 759 assertEquals((splitKeys.length + 1) * regionReplication, 760 util.getAdmin().getRegions(tableName).size()); 761 } 762 763 public static byte[][] getSplitKeys() { 764 return getSplitKeys(KEYS.length); 765 } 766 767 public static byte[][] getSplitKeys(int nRegions) { 768 nRegions = nRegions < KEYS.length ? nRegions : (KEYS.length - 1); 769 final byte[][] splitKeys = new byte[nRegions - 1][]; 770 final int step = KEYS.length / nRegions; 771 int keyIndex = 1; 772 for (int i = 0; i < splitKeys.length; ++i) { 773 splitKeys[i] = new byte[] { KEYS[keyIndex] }; 774 keyIndex += step; 775 } 776 return splitKeys; 777 } 778 779 public static void createTable(final HBaseTestingUtil util, final TableName tableName, 780 final byte[]... families) throws IOException, InterruptedException { 781 createTable(util, tableName, 1, families); 782 } 783 784 public static void createTable(final HBaseTestingUtil util, final TableName tableName, 785 final int regionReplication, final byte[]... families) 786 throws IOException, InterruptedException { 787 createTable(util, tableName, regionReplication, KEYS.length, families); 788 } 789 790 public static void createPreSplitTable(final HBaseTestingUtil util, final TableName tableName, 791 final int nRegions, final byte[]... families) throws IOException, InterruptedException { 792 createTable(util, tableName, 1, nRegions, families); 793 } 794 795 public static void loadData(final HBaseTestingUtil util, final TableName tableName, int rows, 796 byte[]... families) throws IOException, InterruptedException { 797 BufferedMutator mutator = util.getConnection().getBufferedMutator(tableName); 798 loadData(util, mutator, rows, families); 799 } 800 801 public static void loadData(final HBaseTestingUtil util, final BufferedMutator mutator, int rows, 802 byte[]... families) throws IOException, InterruptedException { 803 // Ensure one row per region 804 assertTrue(rows >= KEYS.length); 805 for (byte k0 : KEYS) { 806 byte[] k = new byte[] { k0 }; 807 byte[] value = Bytes.add(Bytes.toBytes(EnvironmentEdgeManager.currentTime()), k); 808 byte[] key = Bytes.add(k, Bytes.toBytes(MD5Hash.getMD5AsHex(value))); 809 final byte[][] families1 = families; 810 final byte[] key1 = key; 811 final byte[] value1 = value; 812 mutator.mutate(createPut(families1, key1, value1)); 813 rows--; 814 } 815 816 // Add other extra rows. more rows, more files 817 while (rows-- > 0) { 818 byte[] value = 819 Bytes.add(Bytes.toBytes(EnvironmentEdgeManager.currentTime()), Bytes.toBytes(rows)); 820 byte[] key = Bytes.toBytes(MD5Hash.getMD5AsHex(value)); 821 final byte[][] families1 = families; 822 final byte[] key1 = key; 823 final byte[] value1 = value; 824 mutator.mutate(createPut(families1, key1, value1)); 825 } 826 mutator.flush(); 827 828 waitForTableToBeOnline(util, mutator.getName()); 829 } 830 831 private static Put createPut(final byte[][] families, final byte[] key, final byte[] value) { 832 byte[] q = Bytes.toBytes("q"); 833 Put put = new Put(key); 834 put.setDurability(Durability.SKIP_WAL); 835 for (byte[] family : families) { 836 put.addColumn(family, q, value); 837 } 838 return put; 839 } 840 841 public static void deleteAllSnapshots(final Admin admin) throws IOException { 842 // Delete all the snapshots 843 for (SnapshotDescription snapshot : admin.listSnapshots()) { 844 admin.deleteSnapshot(snapshot.getName()); 845 } 846 SnapshotTestingUtils.assertNoSnapshots(admin); 847 } 848 849 public static void deleteArchiveDirectory(final HBaseTestingUtil util) throws IOException { 850 // Ensure the archiver to be empty 851 MasterFileSystem mfs = util.getMiniHBaseCluster().getMaster().getMasterFileSystem(); 852 Path archiveDir = new Path(mfs.getRootDir(), HConstants.HFILE_ARCHIVE_DIRECTORY); 853 mfs.getFileSystem().delete(archiveDir, true); 854 } 855 856 public static void verifyRowCount(final HBaseTestingUtil util, final TableName tableName, 857 long expectedRows) throws IOException { 858 Table table = util.getConnection().getTable(tableName); 859 try { 860 assertEquals(expectedRows, util.countRows(table)); 861 } finally { 862 table.close(); 863 } 864 } 865 866 public static void verifyReplicasCameOnline(TableName tableName, Admin admin, 867 int regionReplication) throws IOException { 868 List<RegionInfo> regions = admin.getRegions(tableName); 869 HashSet<RegionInfo> set = new HashSet<>(); 870 for (RegionInfo hri : regions) { 871 set.add(RegionReplicaUtil.getRegionInfoForDefaultReplica(hri)); 872 for (int i = 0; i < regionReplication; i++) { 873 RegionInfo replica = RegionReplicaUtil.getRegionInfoForReplica(hri, i); 874 if (!regions.contains(replica)) { 875 Assert.fail(replica + " is not contained in the list of online regions"); 876 } 877 } 878 } 879 assertEquals(getSplitKeys().length + 1, set.size()); 880 } 881}