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.assertTrue;
022
023import java.io.IOException;
024import java.util.List;
025import org.apache.hadoop.conf.Configuration;
026import org.apache.hadoop.fs.FileSystem;
027import org.apache.hadoop.fs.LocatedFileStatus;
028import org.apache.hadoop.fs.Path;
029import org.apache.hadoop.fs.RemoteIterator;
030import org.apache.hadoop.hbase.HBaseClassTestRule;
031import org.apache.hadoop.hbase.HBaseTestingUtil;
032import org.apache.hadoop.hbase.HConstants;
033import org.apache.hadoop.hbase.TableName;
034import org.apache.hadoop.hbase.client.RegionInfo;
035import org.apache.hadoop.hbase.client.Table;
036import org.apache.hadoop.hbase.client.TableDescriptor;
037import org.apache.hadoop.hbase.errorhandling.ForeignExceptionDispatcher;
038import org.apache.hadoop.hbase.io.HFileLink;
039import org.apache.hadoop.hbase.mob.MobUtils;
040import org.apache.hadoop.hbase.monitoring.MonitoredTask;
041import org.apache.hadoop.hbase.regionserver.HRegion;
042import org.apache.hadoop.hbase.regionserver.StoreFileInfo;
043import org.apache.hadoop.hbase.snapshot.SnapshotTestingUtils.SnapshotMock;
044import org.apache.hadoop.hbase.testclassification.MediumTests;
045import org.apache.hadoop.hbase.testclassification.RegionServerTests;
046import org.apache.hadoop.hbase.util.Bytes;
047import org.apache.hadoop.hbase.util.CommonFSUtils;
048import org.apache.hadoop.hbase.util.FSTableDescriptors;
049import org.apache.hadoop.hbase.wal.WALSplitUtil;
050import org.junit.After;
051import org.junit.AfterClass;
052import org.junit.Assert;
053import org.junit.Before;
054import org.junit.BeforeClass;
055import org.junit.ClassRule;
056import org.junit.Test;
057import org.junit.experimental.categories.Category;
058import org.mockito.Mockito;
059import org.slf4j.Logger;
060import org.slf4j.LoggerFactory;
061
062import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription;
063
064/**
065 * Test the restore/clone operation from a file-system point of view.
066 */
067@Category({ RegionServerTests.class, MediumTests.class })
068public class TestRestoreSnapshotHelper {
069
070  @ClassRule
071  public static final HBaseClassTestRule CLASS_RULE =
072    HBaseClassTestRule.forClass(TestRestoreSnapshotHelper.class);
073
074  private static final Logger LOG = LoggerFactory.getLogger(TestRestoreSnapshotHelper.class);
075
076  protected final static HBaseTestingUtil TEST_UTIL = new HBaseTestingUtil();
077  protected final static String TEST_HFILE = "abc";
078
079  protected Configuration conf;
080  protected Path archiveDir;
081  protected FileSystem fs;
082  protected Path rootDir;
083
084  protected void setupConf(Configuration conf) {
085  }
086
087  @BeforeClass
088  public static void setupCluster() throws Exception {
089    TEST_UTIL.startMiniCluster();
090  }
091
092  @AfterClass
093  public static void tearDownCluster() throws Exception {
094    TEST_UTIL.shutdownMiniCluster();
095  }
096
097  @Before
098  public void setup() throws Exception {
099    rootDir = TEST_UTIL.getDataTestDir("testRestore");
100    archiveDir = new Path(rootDir, HConstants.HFILE_ARCHIVE_DIRECTORY);
101    fs = TEST_UTIL.getTestFileSystem();
102    conf = TEST_UTIL.getConfiguration();
103    setupConf(conf);
104    CommonFSUtils.setRootDir(conf, rootDir);
105  }
106
107  @After
108  public void tearDown() throws Exception {
109    fs.delete(TEST_UTIL.getDataTestDir(), true);
110  }
111
112  protected SnapshotMock createSnapshotMock() throws IOException {
113    return new SnapshotMock(TEST_UTIL.getConfiguration(), fs, rootDir);
114  }
115
116  @Test
117  public void testRestore() throws IOException {
118    restoreAndVerify("snapshot", "testRestore");
119  }
120
121  @Test
122  public void testRestoreWithNamespace() throws IOException {
123    restoreAndVerify("snapshot", "namespace1:testRestoreWithNamespace");
124  }
125
126  @Test
127  public void testNoHFileLinkInRootDir() throws IOException {
128    rootDir = TEST_UTIL.getDefaultRootDirPath();
129    CommonFSUtils.setRootDir(conf, rootDir);
130    fs = rootDir.getFileSystem(conf);
131
132    TableName tableName = TableName.valueOf("testNoHFileLinkInRootDir");
133    String snapshotName = tableName.getNameAsString() + "-snapshot";
134    createTableAndSnapshot(tableName, snapshotName);
135
136    Path restoreDir = new Path("/hbase/.tmp-restore");
137    RestoreSnapshotHelper.copySnapshotForScanner(conf, fs, rootDir, restoreDir, snapshotName);
138    checkNoHFileLinkInTableDir(tableName);
139  }
140
141  @Test
142  public void testSkipReplayAndUpdateSeqId() throws Exception {
143    rootDir = TEST_UTIL.getDefaultRootDirPath();
144    CommonFSUtils.setRootDir(conf, rootDir);
145    TableName tableName = TableName.valueOf("testSkipReplayAndUpdateSeqId");
146    String snapshotName = "testSkipReplayAndUpdateSeqId";
147    createTableAndSnapshot(tableName, snapshotName);
148    // put some data in the table
149    Table table = TEST_UTIL.getConnection().getTable(tableName);
150    TEST_UTIL.loadTable(table, Bytes.toBytes("A"));
151
152    Configuration conf = TEST_UTIL.getConfiguration();
153    Path rootDir = CommonFSUtils.getRootDir(conf);
154    Path restoreDir = new Path("/hbase/.tmp-restore/testScannerWithRestoreScanner2");
155    // restore snapshot.
156    final RestoreSnapshotHelper.RestoreMetaChanges meta =
157      RestoreSnapshotHelper.copySnapshotForScanner(conf, fs, rootDir, restoreDir, snapshotName);
158    TableDescriptor htd = meta.getTableDescriptor();
159    final List<RegionInfo> restoredRegions = meta.getRegionsToAdd();
160    for (RegionInfo restoredRegion : restoredRegions) {
161      // open restored region
162      HRegion region = HRegion.newHRegion(CommonFSUtils.getTableDir(restoreDir, tableName), null,
163        fs, conf, restoredRegion, htd, null);
164      // set restore flag
165      region.setRestoredRegion(true);
166      region.initialize();
167      Path recoveredEdit =
168        CommonFSUtils.getWALRegionDir(conf, tableName, region.getRegionInfo().getEncodedName());
169      long maxSeqId = WALSplitUtil.getMaxRegionSequenceId(fs, recoveredEdit);
170
171      // open restored region without set restored flag
172      HRegion region2 = HRegion.newHRegion(CommonFSUtils.getTableDir(restoreDir, tableName), null,
173        fs, conf, restoredRegion, htd, null);
174      region2.initialize();
175      long maxSeqId2 = WALSplitUtil.getMaxRegionSequenceId(fs, recoveredEdit);
176      Assert.assertTrue(maxSeqId2 > maxSeqId);
177    }
178  }
179
180  protected void createTableAndSnapshot(TableName tableName, String snapshotName)
181    throws IOException {
182    byte[] column = Bytes.toBytes("A");
183    Table table = TEST_UTIL.createTable(tableName, column, 2);
184    TEST_UTIL.loadTable(table, column);
185    TEST_UTIL.getAdmin().snapshot(snapshotName, tableName);
186  }
187
188  private void checkNoHFileLinkInTableDir(TableName tableName) throws IOException {
189    Path[] tableDirs = new Path[] { CommonFSUtils.getTableDir(rootDir, tableName),
190      CommonFSUtils.getTableDir(new Path(rootDir, HConstants.HFILE_ARCHIVE_DIRECTORY), tableName),
191      CommonFSUtils.getTableDir(MobUtils.getMobHome(rootDir), tableName) };
192    for (Path tableDir : tableDirs) {
193      Assert.assertFalse(hasHFileLink(tableDir));
194    }
195  }
196
197  private boolean hasHFileLink(Path tableDir) throws IOException {
198    if (fs.exists(tableDir)) {
199      RemoteIterator<LocatedFileStatus> iterator = fs.listFiles(tableDir, true);
200      while (iterator.hasNext()) {
201        LocatedFileStatus fileStatus = iterator.next();
202        if (fileStatus.isFile() && HFileLink.isHFileLink(fileStatus.getPath())) {
203          return true;
204        }
205      }
206    }
207    return false;
208  }
209
210  private void restoreAndVerify(final String snapshotName, final String tableName)
211    throws IOException {
212    // Test Rolling-Upgrade like Snapshot.
213    // half machines writing using v1 and the others using v2 format.
214    SnapshotMock snapshotMock = createSnapshotMock();
215    SnapshotMock.SnapshotBuilder builder = snapshotMock.createSnapshotV2("snapshot", tableName);
216    builder.addRegionV1();
217    builder.addRegionV2();
218    builder.addRegionV2();
219    builder.addRegionV1();
220    Path snapshotDir = builder.commit();
221    TableDescriptor htd = builder.getTableDescriptor();
222    SnapshotDescription desc = builder.getSnapshotDescription();
223
224    // Test clone a snapshot
225    TableDescriptor htdClone = snapshotMock.createHtd("testtb-clone");
226    testRestore(snapshotDir, desc, htdClone);
227    verifyRestore(rootDir, htd, htdClone);
228
229    // Test clone a clone ("link to link")
230    SnapshotDescription cloneDesc =
231      SnapshotDescription.newBuilder().setName("cloneSnapshot").setTable("testtb-clone").build();
232    Path cloneDir = CommonFSUtils.getTableDir(rootDir, htdClone.getTableName());
233    TableDescriptor htdClone2 = snapshotMock.createHtd("testtb-clone2");
234    testRestore(cloneDir, cloneDesc, htdClone2);
235    verifyRestore(rootDir, htd, htdClone2);
236  }
237
238  private void verifyRestore(final Path rootDir, final TableDescriptor sourceHtd,
239    final TableDescriptor htdClone) throws IOException {
240    List<String> files = SnapshotTestingUtils.listHFileNames(fs,
241      CommonFSUtils.getTableDir(rootDir, htdClone.getTableName()));
242    assertEquals(12, files.size());
243    for (int i = 0; i < files.size(); i += 2) {
244      String linkFile = files.get(i);
245      String refFile = files.get(i + 1);
246      assertTrue(linkFile + " should be a HFileLink", HFileLink.isHFileLink(linkFile));
247      assertTrue(refFile + " should be a Referene", StoreFileInfo.isReference(refFile));
248      assertEquals(sourceHtd.getTableName(), HFileLink.getReferencedTableName(linkFile));
249      Path refPath = getReferredToFile(refFile);
250      LOG.debug("get reference name for file " + refFile + " = " + refPath);
251      assertTrue(refPath.getName() + " should be a HFileLink",
252        HFileLink.isHFileLink(refPath.getName()));
253      assertEquals(linkFile, refPath.getName());
254    }
255  }
256
257  /**
258   * Execute the restore operation
259   * @param snapshotDir The snapshot directory to use as "restore source"
260   * @param sd          The snapshot descriptor
261   * @param htdClone    The HTableDescriptor of the table to restore/clone.
262   */
263  private void testRestore(final Path snapshotDir, final SnapshotDescription sd,
264    final TableDescriptor htdClone) throws IOException {
265    LOG.debug("pre-restore table=" + htdClone.getTableName() + " snapshot=" + snapshotDir);
266    CommonFSUtils.logFileSystemState(fs, rootDir, LOG);
267
268    new FSTableDescriptors(conf).createTableDescriptor(htdClone);
269    RestoreSnapshotHelper helper = getRestoreHelper(rootDir, snapshotDir, sd, htdClone);
270    helper.restoreHdfsRegions();
271
272    LOG.debug("post-restore table=" + htdClone.getTableName() + " snapshot=" + snapshotDir);
273    CommonFSUtils.logFileSystemState(fs, rootDir, LOG);
274  }
275
276  /**
277   * Initialize the restore helper, based on the snapshot and table information provided.
278   */
279  private RestoreSnapshotHelper getRestoreHelper(final Path rootDir, final Path snapshotDir,
280    final SnapshotDescription sd, final TableDescriptor htdClone) throws IOException {
281    ForeignExceptionDispatcher monitor = Mockito.mock(ForeignExceptionDispatcher.class);
282    MonitoredTask status = Mockito.mock(MonitoredTask.class);
283
284    SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, sd);
285    return new RestoreSnapshotHelper(conf, fs, manifest, htdClone, rootDir, monitor, status);
286  }
287
288  private Path getReferredToFile(final String referenceName) {
289    Path fakeBasePath = new Path(new Path("table", "region"), "cf");
290    return StoreFileInfo.getReferredToFile(new Path(fakeBasePath, referenceName));
291  }
292}