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.client;
019
020import java.util.List;
021import java.util.regex.Pattern;
022import org.apache.hadoop.conf.Configuration;
023import org.apache.hadoop.fs.FileSystem;
024import org.apache.hadoop.fs.Path;
025import org.apache.hadoop.hbase.HBaseClassTestRule;
026import org.apache.hadoop.hbase.HBaseTestingUtility;
027import org.apache.hadoop.hbase.HColumnDescriptor;
028import org.apache.hadoop.hbase.HConstants;
029import org.apache.hadoop.hbase.HRegionInfo;
030import org.apache.hadoop.hbase.HTableDescriptor;
031import org.apache.hadoop.hbase.TableName;
032import org.apache.hadoop.hbase.master.snapshot.SnapshotManager;
033import org.apache.hadoop.hbase.regionserver.ConstantSizeRegionSplitPolicy;
034import org.apache.hadoop.hbase.snapshot.SnapshotTestingUtils;
035import org.apache.hadoop.hbase.testclassification.ClientTests;
036import org.apache.hadoop.hbase.testclassification.LargeTests;
037import org.apache.hadoop.hbase.util.Bytes;
038import org.apache.hadoop.hbase.util.Threads;
039import org.junit.After;
040import org.junit.AfterClass;
041import org.junit.Assert;
042import org.junit.Before;
043import org.junit.BeforeClass;
044import org.junit.ClassRule;
045import org.junit.Rule;
046import org.junit.Test;
047import org.junit.experimental.categories.Category;
048import org.junit.rules.TestName;
049import org.slf4j.Logger;
050import org.slf4j.LoggerFactory;
051
052/**
053 * Test to verify that the cloned table is independent of the table from which it was cloned
054 */
055@Category({LargeTests.class, ClientTests.class})
056public class TestSnapshotCloneIndependence {
057
058  @ClassRule
059  public static final HBaseClassTestRule CLASS_RULE =
060      HBaseClassTestRule.forClass(TestSnapshotCloneIndependence.class);
061
062  private static final Logger LOG = LoggerFactory.getLogger(TestSnapshotCloneIndependence.class);
063
064  @Rule
065  public TestName testName = new TestName();
066
067  protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility();
068
069  protected static final int NUM_RS = 2;
070  private static final String STRING_TABLE_NAME = "test";
071  private static final String TEST_FAM_STR = "fam";
072  protected static final byte[] TEST_FAM = Bytes.toBytes(TEST_FAM_STR);
073  private static final int CLEANER_INTERVAL = 100;
074
075  private FileSystem fs;
076  private Path rootDir;
077  private Admin admin;
078  private TableName originalTableName;
079  private Table originalTable;
080  private TableName cloneTableName;
081  private int countOriginalTable;
082  String snapshotNameAsString;
083  byte[] snapshotName;
084
085  /**
086   * Setup the config for the cluster and start it
087   */
088  @BeforeClass
089  public static void setupCluster() throws Exception {
090    setupConf(UTIL.getConfiguration());
091    UTIL.startMiniCluster(NUM_RS);
092  }
093
094  static void setupConf(Configuration conf) {
095    // Up the handlers; this test needs more than usual.
096    conf.setInt(HConstants.REGION_SERVER_HIGH_PRIORITY_HANDLER_COUNT, 15);
097    // enable snapshot support
098    conf.setBoolean(SnapshotManager.HBASE_SNAPSHOT_ENABLED, true);
099    // change the flush size to a small amount, regulating number of store files
100    conf.setInt("hbase.hregion.memstore.flush.size", 25000);
101    // so make sure we get a compaction when doing a load, but keep around
102    // some files in the store
103    conf.setInt("hbase.hstore.compaction.min", 10);
104    conf.setInt("hbase.hstore.compactionThreshold", 10);
105    // block writes if we get to 12 store files
106    conf.setInt("hbase.hstore.blockingStoreFiles", 12);
107    conf.setInt("hbase.regionserver.msginterval", 100);
108    conf.setBoolean("hbase.master.enabletable.roundrobin", true);
109    // Avoid potentially aggressive splitting which would cause snapshot to fail
110    conf.set(HConstants.HBASE_REGION_SPLIT_POLICY_KEY,
111        ConstantSizeRegionSplitPolicy.class.getName());
112    // Execute cleaner frequently to induce failures
113    conf.setInt("hbase.master.cleaner.interval", CLEANER_INTERVAL);
114    conf.setInt("hbase.master.hfilecleaner.plugins.snapshot.period", CLEANER_INTERVAL);
115    // Effectively disable TimeToLiveHFileCleaner. Don't want to fully disable it because that
116    // will even trigger races between creating the directory containing back references and
117    // the back reference itself.
118    conf.setInt("hbase.master.hfilecleaner.ttl", CLEANER_INTERVAL);
119  }
120
121  @Before
122  public void setup() throws Exception {
123    fs = UTIL.getHBaseCluster().getMaster().getMasterFileSystem().getFileSystem();
124    rootDir = UTIL.getHBaseCluster().getMaster().getMasterFileSystem().getRootDir();
125
126    admin = UTIL.getAdmin();
127    originalTableName = TableName.valueOf("test" + testName.getMethodName());
128    cloneTableName = TableName.valueOf("test-clone-" + originalTableName);
129    snapshotNameAsString = "snapshot_" + originalTableName;
130    snapshotName = Bytes.toBytes(snapshotNameAsString);
131
132    originalTable = createTable(originalTableName, TEST_FAM);
133    loadData(originalTable, TEST_FAM);
134    countOriginalTable = countRows(originalTable);
135    System.out.println("Original table has: " + countOriginalTable + " rows");
136  }
137
138  @After
139  public void tearDown() throws Exception {
140    UTIL.deleteTable(originalTableName);
141    UTIL.deleteTable(cloneTableName);
142    SnapshotTestingUtils.deleteAllSnapshots(UTIL.getAdmin());
143    SnapshotTestingUtils.deleteArchiveDirectory(UTIL);
144  }
145
146  @AfterClass
147  public static void cleanupTest() throws Exception {
148    try {
149      UTIL.shutdownMiniCluster();
150    } catch (Exception e) {
151      LOG.warn("failure shutting down cluster", e);
152    }
153  }
154
155  /**
156   * Verify that adding data to the cloned table will not affect the original, and vice-versa when
157   * it is taken as an online snapshot.
158   */
159  @Test
160  public void testOnlineSnapshotAppendIndependent() throws Exception {
161    createAndCloneSnapshot(true);
162    runTestSnapshotAppendIndependent();
163  }
164
165  /**
166   * Verify that adding data to the cloned table will not affect the original, and vice-versa when
167   * it is taken as an offline snapshot.
168   */
169  @Test
170  public void testOfflineSnapshotAppendIndependent() throws Exception {
171    createAndCloneSnapshot(false);
172    runTestSnapshotAppendIndependent();
173  }
174
175  /**
176   * Verify that adding metadata to the cloned table will not affect the original, and vice-versa
177   * when it is taken as an online snapshot.
178   */
179  @Test
180  public void testOnlineSnapshotMetadataChangesIndependent() throws Exception {
181    createAndCloneSnapshot(true);
182    runTestSnapshotMetadataChangesIndependent();
183  }
184
185  /**
186   * Verify that adding netadata to the cloned table will not affect the original, and vice-versa
187   * when is taken as an online snapshot.
188   */
189  @Test
190  public void testOfflineSnapshotMetadataChangesIndependent() throws Exception {
191    createAndCloneSnapshot(false);
192    runTestSnapshotMetadataChangesIndependent();
193  }
194
195  /**
196   * Verify that region operations, in this case splitting a region, are independent between the
197   * cloned table and the original.
198   */
199  @Test
200  public void testOfflineSnapshotRegionOperationsIndependent() throws Exception {
201    createAndCloneSnapshot(false);
202    runTestRegionOperationsIndependent();
203  }
204
205  /**
206   * Verify that region operations, in this case splitting a region, are independent between the
207   * cloned table and the original.
208   */
209  @Test
210  public void testOnlineSnapshotRegionOperationsIndependent() throws Exception {
211    createAndCloneSnapshot(true);
212    runTestRegionOperationsIndependent();
213  }
214
215  @Test
216  public void testOfflineSnapshotDeleteIndependent() throws Exception {
217    createAndCloneSnapshot(false);
218    runTestSnapshotDeleteIndependent();
219  }
220
221  @Test
222  public void testOnlineSnapshotDeleteIndependent() throws Exception {
223    createAndCloneSnapshot(true);
224    runTestSnapshotDeleteIndependent();
225  }
226
227  private static void waitOnSplit(Connection c, final Table t, int originalCount) throws Exception {
228    for (int i = 0; i < 200; i++) {
229      Threads.sleepWithoutInterrupt(500);
230      try (RegionLocator locator = c.getRegionLocator(t.getName())) {
231        if (locator.getAllRegionLocations().size() > originalCount) {
232          return;
233        }
234      }
235    }
236    throw new Exception("Split did not increase the number of regions");
237  }
238
239  /**
240   * Takes the snapshot of originalTable and clones the snapshot to another tables.
241   * If {@code online} is false, the original table is disabled during taking snapshot, so also
242   * enables it again.
243   * @param online - Whether the table is online or not during the snapshot
244   */
245  private void createAndCloneSnapshot(boolean online) throws Exception {
246    SnapshotTestingUtils.createSnapshotAndValidate(admin, originalTableName, TEST_FAM_STR,
247      snapshotNameAsString, rootDir, fs, online);
248
249    // If offline, enable the table disabled by snapshot testing util.
250    if (!online) {
251      admin.enableTable(originalTableName);
252      UTIL.waitTableAvailable(originalTableName);
253    }
254
255    admin.cloneSnapshot(snapshotName, cloneTableName);
256    UTIL.waitUntilAllRegionsAssigned(cloneTableName);
257  }
258
259  /**
260   * Verify that adding data to original table or clone table doesn't affect other table.
261   */
262  private void runTestSnapshotAppendIndependent() throws Exception {
263    try (Table clonedTable = UTIL.getConnection().getTable(cloneTableName)) {
264      final int clonedTableRowCount = countRows(clonedTable);
265
266      Assert.assertEquals(
267        "The line counts of original and cloned tables do not match after clone. ",
268        countOriginalTable, clonedTableRowCount);
269
270      // Attempt to add data to the test
271      Put p = new Put(Bytes.toBytes("new-row-" + System.currentTimeMillis()));
272      p.addColumn(TEST_FAM, Bytes.toBytes("someQualifier"), Bytes.toBytes("someString"));
273      originalTable.put(p);
274
275      // Verify that the new row is not in the restored table
276      Assert.assertEquals("The row count of the original table was not modified by the put",
277        countOriginalTable + 1, countRows(originalTable));
278      Assert.assertEquals(
279        "The row count of the cloned table changed as a result of addition to the original",
280        clonedTableRowCount, countRows(clonedTable));
281
282      Put p2 = new Put(Bytes.toBytes("new-row-" + System.currentTimeMillis()));
283      p2.addColumn(TEST_FAM, Bytes.toBytes("someQualifier"), Bytes.toBytes("someString"));
284      clonedTable.put(p2);
285
286      // Verify that the row is not added to the original table.
287      Assert.assertEquals(
288        "The row count of the original table was modified by the put to the clone",
289        countOriginalTable + 1, countRows(originalTable));
290      Assert.assertEquals("The row count of the cloned table was not modified by the put",
291        clonedTableRowCount + 1, countRows(clonedTable));
292    }
293  }
294
295  /**
296   * Do a split, and verify that this only affects one table
297   */
298  private void runTestRegionOperationsIndependent() throws Exception {
299    // Verify that region information is the same pre-split
300    ((ClusterConnection) UTIL.getConnection()).clearRegionCache();
301    List<HRegionInfo> originalTableHRegions = admin.getTableRegions(originalTableName);
302
303    final int originalRegionCount = originalTableHRegions.size();
304    final int cloneTableRegionCount = admin.getTableRegions(cloneTableName).size();
305    Assert.assertEquals(
306      "The number of regions in the cloned table is different than in the original table.",
307      originalRegionCount, cloneTableRegionCount);
308
309    // Split a region on the parent table
310    admin.splitRegionAsync(originalTableHRegions.get(0).getRegionName()).get();
311    waitOnSplit(UTIL.getConnection(), originalTable, originalRegionCount);
312
313    // Verify that the cloned table region is not split
314    final int cloneTableRegionCount2 = admin.getTableRegions(cloneTableName).size();
315    Assert.assertEquals(
316      "The number of regions in the cloned table changed though none of its regions were split.",
317      cloneTableRegionCount, cloneTableRegionCount2);
318  }
319
320  /**
321   * Add metadata, and verify that this only affects one table
322   */
323  private void runTestSnapshotMetadataChangesIndependent() throws Exception {
324    // Add a new column family to the original table
325    byte[] TEST_FAM_2 = Bytes.toBytes("fam2");
326    HColumnDescriptor hcd = new HColumnDescriptor(TEST_FAM_2);
327
328    admin.disableTable(originalTableName);
329    admin.addColumnFamily(originalTableName, hcd);
330
331    // Verify that it is not in the snapshot
332    admin.enableTable(originalTableName);
333    UTIL.waitTableAvailable(originalTableName);
334
335    // get a description of the cloned table
336    // get a list of its families
337    // assert that the family is there
338    HTableDescriptor originalTableDescriptor = originalTable.getTableDescriptor();
339    HTableDescriptor clonedTableDescriptor = admin.getTableDescriptor(cloneTableName);
340
341    Assert.assertTrue("The original family was not found. There is something wrong. ",
342      originalTableDescriptor.hasFamily(TEST_FAM));
343    Assert.assertTrue("The original family was not found in the clone. There is something wrong. ",
344      clonedTableDescriptor.hasFamily(TEST_FAM));
345
346    Assert.assertTrue("The new family was not found. ",
347      originalTableDescriptor.hasFamily(TEST_FAM_2));
348    Assert.assertTrue("The new family was not found. ",
349      !clonedTableDescriptor.hasFamily(TEST_FAM_2));
350  }
351
352  /**
353   * Verify that deleting the snapshot does not affect either table.
354   */
355  private void runTestSnapshotDeleteIndependent() throws Exception {
356    // Ensure the original table does not reference the HFiles anymore
357    admin.majorCompact(originalTableName);
358
359    // Deleting the snapshot used to break the cloned table by deleting in-use HFiles
360    admin.deleteSnapshot(snapshotName);
361
362    // Wait for cleaner run and DFS heartbeats so that anything that is deletable is fully deleted
363    Pattern pattern = Pattern.compile(snapshotNameAsString);
364    do {
365      Thread.sleep(5000);
366    } while (!admin.listSnapshots(pattern).isEmpty());
367
368    try (Table original = UTIL.getConnection().getTable(originalTableName)) {
369      try (Table clonedTable = UTIL.getConnection().getTable(cloneTableName)) {
370        // Verify that all regions of both tables are readable
371        final int origTableRowCount = countRows(original);
372        final int clonedTableRowCount = countRows(clonedTable);
373        Assert.assertEquals(origTableRowCount, clonedTableRowCount);
374      }
375    }
376  }
377
378  protected Table createTable(final TableName table, byte[] family) throws Exception {
379   Table t = UTIL.createTable(table, family);
380    // Wait for everything to be ready with the table
381    UTIL.waitUntilAllRegionsAssigned(table);
382
383    // At this point the table should be good to go.
384    return t;
385  }
386
387  public void loadData(final Table table, byte[]... families) throws Exception {
388    UTIL.loadTable(originalTable, TEST_FAM);
389  }
390
391  protected int countRows(final Table table, final byte[]... families) throws Exception {
392    return UTIL.countRows(table, families);
393  }
394}