From b0480db1aefc3d8b20860abe9dfee0f3e89e5893 Mon Sep 17 00:00:00 2001 From: Devaraj Das Date: Thu, 6 Mar 2014 23:33:23 +0000 Subject: [PATCH] HBASE-10672. Table snapshot should handle tables whose REGION_REPLICATION is greater than one git-svn-id: https://svn.apache.org/repos/asf/hbase/branches/hbase-10070@1575096 13f79535-47bb-0310-9956-ffa450edef68 --- .../hbase/client/RegionReplicaUtil.java | 17 ++++++++ .../apache/hadoop/hbase/master/HMaster.java | 24 ++++------- .../master/handler/CreateTableHandler.java | 31 +++++++++++++- .../DisabledTableSnapshotHandler.java | 7 +++- .../snapshot/MasterSnapshotVerifier.java | 3 ++ .../snapshot/RegionServerSnapshotManager.java | 13 +++++- .../hadoop/hbase/util/ModifyRegionUtils.java | 10 +---- .../client/TestCloneSnapshotFromClient.java | 11 ++++- ...eSnapshotFromClientWithRegionReplicas.java | 30 ++++++++++++++ .../client/TestRestoreSnapshotFromClient.java | 14 ++++++- ...eSnapshotFromClientWithRegionReplicas.java | 30 ++++++++++++++ .../hbase/client/TestSnapshotFromClient.java | 15 +++++-- ...tSnapshotFromClientWithRegionReplicas.java | 30 ++++++++++++++ .../hbase/snapshot/SnapshotTestingUtils.java | 41 +++++++++++++++++-- 14 files changed, 238 insertions(+), 38 deletions(-) create mode 100644 hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestCloneSnapshotFromClientWithRegionReplicas.java create mode 100644 hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestRestoreSnapshotFromClientWithRegionReplicas.java create mode 100644 hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSnapshotFromClientWithRegionReplicas.java diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/RegionReplicaUtil.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/RegionReplicaUtil.java index cb7a2347687..84883db540e 100644 --- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/RegionReplicaUtil.java +++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/RegionReplicaUtil.java @@ -18,6 +18,9 @@ package org.apache.hadoop.hbase.client; +import java.util.Collection; +import java.util.Iterator; + import org.apache.hadoop.classification.InterfaceAudience; import org.apache.hadoop.hbase.HRegionInfo; @@ -71,4 +74,18 @@ public class RegionReplicaUtil { public static boolean isDefaultReplica(HRegionInfo hri) { return hri.getReplicaId() == DEFAULT_REPLICA_ID; } + + /** + * Removes the non-default replicas from the passed regions collection + * @param regions + */ + public static void removeNonDefaultRegions(Collection regions) { + Iterator iterator = regions.iterator(); + while (iterator.hasNext()) { + HRegionInfo hri = iterator.next(); + if (!RegionReplicaUtil.isDefaultReplica(hri)) { + iterator.remove(); + } + } + } } diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java index a00776099b7..446405583ff 100644 --- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java +++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java @@ -1299,31 +1299,21 @@ public class HMaster extends HRegionServer implements MasterServices, Server { private HRegionInfo[] getHRegionInfos(HTableDescriptor hTableDescriptor, byte[][] splitKeys) { - HRegionInfo[] hRegionInfos = null; - int numRegionReplicas = hTableDescriptor.getRegionReplication(); - if (numRegionReplicas <= 0) { - LOG.warn("Invalid number of replicas per region in the table descriptor. Setting it to 1."); - numRegionReplicas = 1; - } long regionId = System.currentTimeMillis(); + HRegionInfo[] hRegionInfos = null; if (splitKeys == null || splitKeys.length == 0) { - hRegionInfos = new HRegionInfo[numRegionReplicas]; - for (int i = 0; i < numRegionReplicas; i++) { - hRegionInfos[i] = new HRegionInfo(hTableDescriptor.getTableName(), null, null, - false, regionId, (short)i); - } + hRegionInfos = new HRegionInfo[]{new HRegionInfo(hTableDescriptor.getTableName(), null, null, + false, regionId)}; } else { int numRegions = splitKeys.length + 1; - hRegionInfos = new HRegionInfo[numRegions * numRegionReplicas]; + hRegionInfos = new HRegionInfo[numRegions]; byte[] startKey = null; byte[] endKey = null; for (int i = 0; i < numRegions; i++) { endKey = (i == splitKeys.length) ? null : splitKeys[i]; - for (int j = 0; j < numRegionReplicas; j++) { - hRegionInfos[i*numRegionReplicas + j] = - new HRegionInfo(hTableDescriptor.getTableName(), startKey, endKey, - false, regionId, (short)j); - } + hRegionInfos[i] = + new HRegionInfo(hTableDescriptor.getTableName(), startKey, endKey, + false, regionId); startKey = endKey; } } diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/handler/CreateTableHandler.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/handler/CreateTableHandler.java index 6ddd0e7fdca..d6a5d6a8965 100644 --- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/handler/CreateTableHandler.java +++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/handler/CreateTableHandler.java @@ -20,6 +20,7 @@ package org.apache.hadoop.hbase.master.handler; import java.io.IOException; import java.io.InterruptedIOException; +import java.util.ArrayList; import java.util.List; import org.apache.commons.logging.Log; @@ -38,6 +39,7 @@ import org.apache.hadoop.hbase.TableExistsException; import org.apache.hadoop.hbase.catalog.CatalogTracker; import org.apache.hadoop.hbase.catalog.MetaEditor; import org.apache.hadoop.hbase.catalog.MetaReader; +import org.apache.hadoop.hbase.client.RegionReplicaUtil; import org.apache.hadoop.hbase.executor.EventHandler; import org.apache.hadoop.hbase.executor.EventType; import org.apache.hadoop.hbase.master.AssignmentManager; @@ -84,6 +86,7 @@ public class CreateTableHandler extends EventHandler { , EventType.C_M_CREATE_TABLE.toString()); } + @Override public CreateTableHandler prepare() throws NotAllMetaRegionsOnlineException, TableExistsException, IOException { int timeout = conf.getInt("hbase.client.catalog.timeout", 10000); @@ -240,8 +243,10 @@ public class CreateTableHandler extends EventHandler { if (regionInfos != null && regionInfos.size() > 0) { // 4. Add regions to META addRegionsToMeta(this.catalogTracker, regionInfos); + // 5. Add replicas if needed + regionInfos = addReplicas(hTableDescriptor, regionInfos); - // 5. Trigger immediate assignment of the regions in round-robin fashion + // 6. Trigger immediate assignment of the regions in round-robin fashion ModifyRegionUtils.assignRegions(assignmentManager, regionInfos); } @@ -255,6 +260,30 @@ public class CreateTableHandler extends EventHandler { } } + /** + * Create any replicas for the regions (the default replicas that was + * already created is passed to the method) + * @param hTableDescriptor + * @param regions default replicas + * @return the combined list of default and non-default replicas + */ + protected List addReplicas(HTableDescriptor hTableDescriptor, + List regions) { + int numRegionReplicas = hTableDescriptor.getRegionReplication() - 1; + if (numRegionReplicas <= 0) { + return regions; + } + List hRegionInfos = + new ArrayList((numRegionReplicas+1)*regions.size()); + for (int i = 0; i < regions.size(); i++) { + for (int j = 1; j <= numRegionReplicas; j++) { + hRegionInfos.add(RegionReplicaUtil.getRegionInfoForReplica(regions.get(i), j)); + } + } + hRegionInfos.addAll(regions); + return hRegionInfos; + } + private void releaseTableLock() { if (this.tableLock != null) { try { diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/snapshot/DisabledTableSnapshotHandler.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/snapshot/DisabledTableSnapshotHandler.java index d7c79566c20..f0d0b9515ed 100644 --- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/snapshot/DisabledTableSnapshotHandler.java +++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/snapshot/DisabledTableSnapshotHandler.java @@ -30,6 +30,7 @@ import org.apache.hadoop.classification.InterfaceAudience; import org.apache.hadoop.classification.InterfaceStability; import org.apache.hadoop.hbase.HRegionInfo; import org.apache.hadoop.hbase.ServerName; +import org.apache.hadoop.hbase.client.RegionReplicaUtil; import org.apache.hadoop.hbase.errorhandling.ForeignException; import org.apache.hadoop.hbase.errorhandling.ForeignExceptionListener; import org.apache.hadoop.hbase.errorhandling.TimeoutExceptionInjector; @@ -84,7 +85,11 @@ public class DisabledTableSnapshotHandler extends TakeSnapshotHandler { // extract each pair to separate lists Set regions = new HashSet(); for (Pair p : regionsAndLocations) { - regions.add(p.getFirst()); + // Don't include non-default regions + HRegionInfo hri = p.getFirst(); + if (RegionReplicaUtil.isDefaultReplica(hri)) { + regions.add(hri); + } } // 2. for each region, write all the info to disk diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/snapshot/MasterSnapshotVerifier.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/snapshot/MasterSnapshotVerifier.java index d5d99938093..fc1f862f677 100644 --- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/snapshot/MasterSnapshotVerifier.java +++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/snapshot/MasterSnapshotVerifier.java @@ -33,6 +33,7 @@ import org.apache.hadoop.hbase.HRegionInfo; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.ServerName; import org.apache.hadoop.hbase.catalog.MetaReader; +import org.apache.hadoop.hbase.client.RegionReplicaUtil; import org.apache.hadoop.hbase.master.MasterServices; import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.SnapshotDescription; import org.apache.hadoop.hbase.protobuf.generated.SnapshotProtos.SnapshotRegionManifest; @@ -151,6 +152,8 @@ public final class MasterSnapshotVerifier { private void verifyRegions(final SnapshotManifest manifest) throws IOException { List regions = MetaReader.getTableRegions(this.services.getCatalogTracker(), tableName); + // Remove the non-default regions + RegionReplicaUtil.removeNonDefaultRegions(regions); Map regionManifests = manifest.getRegionManifestsMap(); if (regionManifests == null) { diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/snapshot/RegionServerSnapshotManager.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/snapshot/RegionServerSnapshotManager.java index e78d6902fa9..4e2c6c2a238 100644 --- a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/snapshot/RegionServerSnapshotManager.java +++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/snapshot/RegionServerSnapshotManager.java @@ -20,6 +20,7 @@ package org.apache.hadoop.hbase.regionserver.snapshot; import java.io.IOException; import java.util.ArrayList; import java.util.Collection; +import java.util.Iterator; import java.util.List; import java.util.concurrent.Callable; import java.util.concurrent.ExecutionException; @@ -36,6 +37,7 @@ import org.apache.hadoop.classification.InterfaceStability; import org.apache.hadoop.conf.Configuration; import org.apache.hadoop.hbase.DaemonThreadFactory; import org.apache.hadoop.hbase.TableName; +import org.apache.hadoop.hbase.client.RegionReplicaUtil; import org.apache.hadoop.hbase.errorhandling.ForeignException; import org.apache.hadoop.hbase.errorhandling.ForeignExceptionDispatcher; import org.apache.hadoop.hbase.master.snapshot.MasterSnapshotVerifier; @@ -220,7 +222,16 @@ public class RegionServerSnapshotManager extends RegionServerProcedureManager { * @throws IOException */ private List getRegionsToSnapshot(SnapshotDescription snapshot) throws IOException { - return rss.getOnlineRegions(TableName.valueOf(snapshot.getTable())); + List onlineRegions = rss.getOnlineRegions(TableName.valueOf(snapshot.getTable())); + Iterator iterator = onlineRegions.iterator(); + // remove the non-default regions + while (iterator.hasNext()) { + HRegion r = iterator.next(); + if (!RegionReplicaUtil.isDefaultReplica(r.getRegionInfo())) { + iterator.remove(); + } + } + return onlineRegions; } /** diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/util/ModifyRegionUtils.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/util/ModifyRegionUtils.java index 3b1f1cfb6dc..46d8e9066de 100644 --- a/hbase-server/src/main/java/org/apache/hadoop/hbase/util/ModifyRegionUtils.java +++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/util/ModifyRegionUtils.java @@ -143,13 +143,7 @@ public abstract class ModifyRegionUtils { CompletionService completionService = new ExecutorCompletionService(exec); List regionInfos = new ArrayList(); - int defaultReplicas = 0; for (final HRegionInfo newRegion : newRegions) { - regionInfos.add(newRegion); - if (!RegionReplicaUtil.isDefaultReplica(newRegion)) { - continue; - } - defaultReplicas++; completionService.submit(new Callable() { @Override public HRegionInfo call() throws IOException { @@ -159,8 +153,8 @@ public abstract class ModifyRegionUtils { } try { // wait for all regions to finish creation - for (int i = 0; i < defaultReplicas; i++) { - completionService.take().get(); + for (int i = 0; i < regionNumber; i++) { + regionInfos.add(completionService.take().get()); } } catch (InterruptedException e) { LOG.error("Caught " + e + " during region creation"); diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestCloneSnapshotFromClient.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestCloneSnapshotFromClient.java index ccf4c7ec0ed..e16d99ad8a9 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestCloneSnapshotFromClient.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestCloneSnapshotFromClient.java @@ -99,7 +99,7 @@ public class TestCloneSnapshotFromClient { snapshotName2 = Bytes.toBytes("snaptb2-" + tid); // create Table and disable it - SnapshotTestingUtils.createTable(TEST_UTIL, tableName, FAMILY); + SnapshotTestingUtils.createTable(TEST_UTIL, tableName, getNumReplicas(), FAMILY); admin.disableTable(tableName); // take an empty snapshot @@ -132,6 +132,10 @@ public class TestCloneSnapshotFromClient { } } + protected int getNumReplicas() { + return 1; + } + @After public void tearDown() throws Exception { if (admin.tableExists(tableName)) { @@ -168,9 +172,14 @@ public class TestCloneSnapshotFromClient { admin.cloneSnapshot(snapshotName, tableName); SnapshotTestingUtils.verifyRowCount(TEST_UTIL, tableName, snapshotRows); + verifyReplicasCameOnline(tableName); TEST_UTIL.deleteTable(tableName); } + protected void verifyReplicasCameOnline(TableName tableName) throws IOException { + SnapshotTestingUtils.verifyReplicasCameOnline(tableName, admin, getNumReplicas()); + } + @Test public void testCloneSnapshotCrossNamespace() throws IOException, InterruptedException { String nsName = "testCloneSnapshotCrossNamespace"; diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestCloneSnapshotFromClientWithRegionReplicas.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestCloneSnapshotFromClientWithRegionReplicas.java new file mode 100644 index 00000000000..933740da1ab --- /dev/null +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestCloneSnapshotFromClientWithRegionReplicas.java @@ -0,0 +1,30 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ +package org.apache.hadoop.hbase.client; + +import org.apache.hadoop.hbase.LargeTests; +import org.junit.experimental.categories.Category; + +@Category(LargeTests.class) +public class TestCloneSnapshotFromClientWithRegionReplicas extends + TestCloneSnapshotFromClient { + @Override + protected int getNumReplicas() { + return 3; + } +} diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestRestoreSnapshotFromClient.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestRestoreSnapshotFromClient.java index 296bea69385..68923e7e289 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestRestoreSnapshotFromClient.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestRestoreSnapshotFromClient.java @@ -104,7 +104,7 @@ public class TestRestoreSnapshotFromClient { snapshotName2 = Bytes.toBytes("snaptb2-" + tid); // create Table and disable it - SnapshotTestingUtils.createTable(TEST_UTIL, tableName, FAMILY); + SnapshotTestingUtils.createTable(TEST_UTIL, tableName, getNumReplicas(), FAMILY); admin.disableTable(tableName); // take an empty snapshot @@ -143,23 +143,31 @@ public class TestRestoreSnapshotFromClient { admin.restoreSnapshot(snapshotName0); admin.enableTable(tableName); SnapshotTestingUtils.verifyRowCount(TEST_UTIL, tableName, snapshot0Rows); + SnapshotTestingUtils.verifyReplicasCameOnline(tableName, admin, getNumReplicas()); // Restore from emptySnapshot admin.disableTable(tableName); admin.restoreSnapshot(emptySnapshot); admin.enableTable(tableName); SnapshotTestingUtils.verifyRowCount(TEST_UTIL, tableName, 0); + SnapshotTestingUtils.verifyReplicasCameOnline(tableName, admin, getNumReplicas()); // Restore from snapshot-1 admin.disableTable(tableName); admin.restoreSnapshot(snapshotName1); admin.enableTable(tableName); SnapshotTestingUtils.verifyRowCount(TEST_UTIL, tableName, snapshot1Rows); + SnapshotTestingUtils.verifyReplicasCameOnline(tableName, admin, getNumReplicas()); // Restore from snapshot-1 TEST_UTIL.deleteTable(tableName); admin.restoreSnapshot(snapshotName1); SnapshotTestingUtils.verifyRowCount(TEST_UTIL, tableName, snapshot1Rows); + SnapshotTestingUtils.verifyReplicasCameOnline(tableName, admin, getNumReplicas()); + } + + protected int getNumReplicas() { + return 1; } @Test @@ -224,6 +232,7 @@ public class TestRestoreSnapshotFromClient { TableName.valueOf("clonedtb-" + System.currentTimeMillis()); admin.cloneSnapshot(snapshotName0, clonedTableName); SnapshotTestingUtils.verifyRowCount(TEST_UTIL, clonedTableName, snapshot0Rows); + SnapshotTestingUtils.verifyReplicasCameOnline(clonedTableName, admin, getNumReplicas()); admin.disableTable(clonedTableName); admin.snapshot(snapshotName2, clonedTableName); TEST_UTIL.deleteTable(clonedTableName); @@ -231,6 +240,7 @@ public class TestRestoreSnapshotFromClient { admin.cloneSnapshot(snapshotName2, clonedTableName); SnapshotTestingUtils.verifyRowCount(TEST_UTIL, clonedTableName, snapshot0Rows); + SnapshotTestingUtils.verifyReplicasCameOnline(clonedTableName, admin, getNumReplicas()); TEST_UTIL.deleteTable(clonedTableName); } @@ -241,12 +251,14 @@ public class TestRestoreSnapshotFromClient { admin.cloneSnapshot(snapshotName0, tableName); SnapshotTestingUtils.verifyRowCount(TEST_UTIL, tableName, snapshot0Rows); + SnapshotTestingUtils.verifyReplicasCameOnline(tableName, admin, getNumReplicas()); waitCleanerRun(); admin.disableTable(tableName); admin.restoreSnapshot(snapshotName0); admin.enableTable(tableName); SnapshotTestingUtils.verifyRowCount(TEST_UTIL, tableName, snapshot0Rows); + SnapshotTestingUtils.verifyReplicasCameOnline(tableName, admin, getNumReplicas()); } @Test diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestRestoreSnapshotFromClientWithRegionReplicas.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestRestoreSnapshotFromClientWithRegionReplicas.java new file mode 100644 index 00000000000..7ab14ec66c9 --- /dev/null +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestRestoreSnapshotFromClientWithRegionReplicas.java @@ -0,0 +1,30 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ +package org.apache.hadoop.hbase.client; + +import org.apache.hadoop.hbase.LargeTests; +import org.junit.experimental.categories.Category; + +@Category(LargeTests.class) +public class TestRestoreSnapshotFromClientWithRegionReplicas extends + TestRestoreSnapshotFromClient { + @Override + protected int getNumReplicas() { + return 3; + } +} diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSnapshotFromClient.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSnapshotFromClient.java index 75f1c1088d6..873bfb3914b 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSnapshotFromClient.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSnapshotFromClient.java @@ -27,6 +27,7 @@ import org.apache.commons.logging.LogFactory; import org.apache.hadoop.conf.Configuration; import org.apache.hadoop.fs.FileSystem; import org.apache.hadoop.fs.Path; +import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.TableName; import org.apache.hadoop.hbase.HBaseTestingUtility; import org.apache.hadoop.hbase.HConstants; @@ -57,11 +58,11 @@ import com.google.common.collect.Lists; @Category(LargeTests.class) public class TestSnapshotFromClient { private static final Log LOG = LogFactory.getLog(TestSnapshotFromClient.class); - private static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); + protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); private static final int NUM_RS = 2; private static final String STRING_TABLE_NAME = "test"; - private static final byte[] TEST_FAM = Bytes.toBytes("fam"); - private static final TableName TABLE_NAME = + protected static final byte[] TEST_FAM = Bytes.toBytes("fam"); + protected static final TableName TABLE_NAME = TableName.valueOf(STRING_TABLE_NAME); /** @@ -93,7 +94,13 @@ public class TestSnapshotFromClient { @Before public void setup() throws Exception { - UTIL.createTable(TABLE_NAME, TEST_FAM); + HTableDescriptor htd = new HTableDescriptor(TABLE_NAME); + htd.setRegionReplication(getNumReplicas()); + UTIL.createTable(htd, new byte[][]{TEST_FAM}, UTIL.getConfiguration()); + } + + protected int getNumReplicas() { + return 1; } @After diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSnapshotFromClientWithRegionReplicas.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSnapshotFromClientWithRegionReplicas.java new file mode 100644 index 00000000000..08d42a289d6 --- /dev/null +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSnapshotFromClientWithRegionReplicas.java @@ -0,0 +1,30 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ +package org.apache.hadoop.hbase.client; + +import org.apache.hadoop.hbase.LargeTests; +import org.junit.experimental.categories.Category; + +@Category(LargeTests.class) +public class TestSnapshotFromClientWithRegionReplicas extends TestSnapshotFromClient { + + @Override + protected int getNumReplicas() { + return 3; + } +} diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/snapshot/SnapshotTestingUtils.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/snapshot/SnapshotTestingUtils.java index c68a5679450..2f75ea633e5 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/snapshot/SnapshotTestingUtils.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/snapshot/SnapshotTestingUtils.java @@ -25,6 +25,7 @@ import java.io.IOException; import java.util.Arrays; import java.util.ArrayList; import java.util.HashSet; + import java.util.List; import java.util.Map; import java.util.Set; @@ -48,6 +49,7 @@ import org.apache.hadoop.hbase.client.HBaseAdmin; import org.apache.hadoop.hbase.client.HTable; import org.apache.hadoop.hbase.client.Put; import org.apache.hadoop.hbase.errorhandling.ForeignExceptionDispatcher; +import org.apache.hadoop.hbase.client.RegionReplicaUtil; import org.apache.hadoop.hbase.io.HFileLink; import org.apache.hadoop.hbase.master.HMaster; import org.apache.hadoop.hbase.master.MasterFileSystem; @@ -223,6 +225,8 @@ public class SnapshotTestingUtils { // check the region snapshot for all the regions List regions = admin.getTableRegions(tableName); + // remove the non-default regions + RegionReplicaUtil.removeNonDefaultRegions(regions); assertEquals(regions.size(), regionManifests.size()); // Verify Regions (redundant check, see MasterSnapshotVerifier) @@ -629,19 +633,32 @@ public class SnapshotTestingUtils { } public static void createTable(final HBaseTestingUtility util, final TableName tableName, - final byte[]... families) throws IOException, InterruptedException { + int regionReplication, final byte[]... families) throws IOException, InterruptedException { HTableDescriptor htd = new HTableDescriptor(tableName); + htd.setRegionReplication(regionReplication); for (byte[] family: families) { HColumnDescriptor hcd = new HColumnDescriptor(family); htd.addFamily(hcd); } + byte[][] splitKeys = getSplitKeys(); + util.getHBaseAdmin().createTable(htd, splitKeys); + waitForTableToBeOnline(util, tableName); + assertEquals((splitKeys.length + 1) * regionReplication, + util.getHBaseAdmin().getTableRegions(tableName).size()); + } + + public static byte[][] getSplitKeys() { byte[][] splitKeys = new byte[KEYS.length-2][]; + byte[] hex = Bytes.toBytes("123456789abcde"); for (int i = 0; i < splitKeys.length; ++i) { splitKeys[i] = new byte[] { KEYS[i+1] }; } - util.getHBaseAdmin().createTable(htd, splitKeys); - waitForTableToBeOnline(util, tableName); - assertEquals(KEYS.length-1, util.getHBaseAdmin().getTableRegions(tableName).size()); + return splitKeys; + } + + public static void createTable(final HBaseTestingUtility util, final TableName tableName, + final byte[]... families) throws IOException, InterruptedException { + createTable(util, tableName, 1, families); } public static void loadData(final HBaseTestingUtility util, final TableName tableName, int rows, @@ -711,4 +728,20 @@ public class SnapshotTestingUtils { table.close(); } } + + public static void verifyReplicasCameOnline(TableName tableName, HBaseAdmin admin, + int regionReplication) throws IOException { + List regions = admin.getTableRegions(tableName); + HashSet set = new HashSet(); + for (HRegionInfo hri : regions) { + set.add(RegionReplicaUtil.getRegionInfoForDefaultReplica(hri)); + for (int i = 0; i < regionReplication; i++) { + HRegionInfo replica = RegionReplicaUtil.getRegionInfoForReplica(hri, i); + if (!regions.contains(replica)) { + Assert.fail(replica + " is not contained in the list of online regions"); + } + } + } + assert(set.size() == getSplitKeys().length + 1); + } }