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