You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by vj...@apache.org on 2021/01/20 06:03:20 UTC

[hbase] branch branch-2 updated: HBASE-25478 : Implement retries when enabling tables in TestRegionReplicaReplicationEndpoint (#2896)

This is an automated email from the ASF dual-hosted git repository.

vjasani pushed a commit to branch branch-2
in repository https://gitbox.apache.org/repos/asf/hbase.git


The following commit(s) were added to refs/heads/branch-2 by this push:
     new 854f759  HBASE-25478 : Implement retries when enabling tables in TestRegionReplicaReplicationEndpoint (#2896)
854f759 is described below

commit 854f75982f8751993a83c39279d30fd241967caf
Author: Christine Feng <ch...@gmail.com>
AuthorDate: Tue Jan 19 22:02:43 2021 -0800

    HBASE-25478 : Implement retries when enabling tables in TestRegionReplicaReplicationEndpoint (#2896)
    
    Signed-off-by: stack <st...@apache.org>
    Signed-off-by: Viraj Jasani <vj...@apache.org>
---
 .../TestRegionReplicaReplicationEndpoint.java      | 47 +++++++++++++++++-----
 1 file changed, 36 insertions(+), 11 deletions(-)

diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/replication/regionserver/TestRegionReplicaReplicationEndpoint.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/replication/regionserver/TestRegionReplicaReplicationEndpoint.java
index 54a8dcc..9c52d3d 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/replication/regionserver/TestRegionReplicaReplicationEndpoint.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/replication/regionserver/TestRegionReplicaReplicationEndpoint.java
@@ -29,6 +29,7 @@ import java.io.IOException;
 import java.util.List;
 import java.util.UUID;
 import java.util.concurrent.Executors;
+import java.util.concurrent.TimeUnit;
 import java.util.concurrent.atomic.AtomicLong;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.FileSystem;
@@ -50,6 +51,7 @@ import org.apache.hadoop.hbase.client.Connection;
 import org.apache.hadoop.hbase.client.ConnectionFactory;
 import org.apache.hadoop.hbase.client.RegionLocator;
 import org.apache.hadoop.hbase.client.Table;
+import org.apache.hadoop.hbase.client.TableDescriptor;
 import org.apache.hadoop.hbase.client.replication.ReplicationAdmin;
 import org.apache.hadoop.hbase.regionserver.HRegion;
 import org.apache.hadoop.hbase.regionserver.HRegionServer;
@@ -66,6 +68,7 @@ import org.apache.hadoop.hbase.wal.WAL.Entry;
 import org.apache.hadoop.hbase.wal.WALEdit;
 import org.apache.hadoop.hbase.wal.WALKeyImpl;
 import org.apache.hadoop.hbase.zookeeper.ZKConfig;
+import org.apache.hbase.thirdparty.com.google.common.util.concurrent.Uninterruptibles;
 import org.junit.AfterClass;
 import org.junit.BeforeClass;
 import org.junit.ClassRule;
@@ -153,7 +156,7 @@ public class TestRegionReplicaReplicationEndpoint {
 
       HTableDescriptor htd = HTU.createTableDescriptor(
         "testReplicationPeerIsCreated_no_region_replicas");
-      HTU.getAdmin().createTable(htd);
+      createOrEnableTableWithRetries(htd, true);
       try {
         peerConfig = admin.getReplicationPeerConfig(peerId);
         fail("Should throw ReplicationException, because replication peer id=" + peerId
@@ -164,7 +167,7 @@ public class TestRegionReplicaReplicationEndpoint {
 
       htd = HTU.createTableDescriptor("testReplicationPeerIsCreated");
       htd.setRegionReplication(2);
-      HTU.getAdmin().createTable(htd);
+      createOrEnableTableWithRetries(htd, true);
 
       // assert peer configuration is correct
       peerConfig = admin.getReplicationPeerConfig(peerId);
@@ -196,7 +199,7 @@ public class TestRegionReplicaReplicationEndpoint {
       }
 
       HTableDescriptor htd = HTU.createTableDescriptor("testRegionReplicaReplicationPeerIsCreatedForModifyTable");
-      HTU.getAdmin().createTable(htd);
+      createOrEnableTableWithRetries(htd, true);
 
       // assert that replication peer is not created yet
       try {
@@ -210,7 +213,7 @@ public class TestRegionReplicaReplicationEndpoint {
       HTU.getAdmin().disableTable(htd.getTableName());
       htd.setRegionReplication(2);
       HTU.getAdmin().modifyTable(htd.getTableName(), htd);
-      HTU.getAdmin().enableTable(htd.getTableName());
+      createOrEnableTableWithRetries(htd, false);
 
       // assert peer configuration is correct
       peerConfig = admin.getReplicationPeerConfig(peerId);
@@ -229,7 +232,7 @@ public class TestRegionReplicaReplicationEndpoint {
         + regionReplication);
     HTableDescriptor htd = HTU.createTableDescriptor(tableName.toString());
     htd.setRegionReplication(regionReplication);
-    HTU.getAdmin().createTable(htd);
+    createOrEnableTableWithRetries(htd, true);
     TableName tableNameNoReplicas =
         TableName.valueOf("testRegionReplicaReplicationWithReplicas_NO_REPLICAS");
     HTU.deleteTableIfAny(tableNameNoReplicas);
@@ -320,7 +323,7 @@ public class TestRegionReplicaReplicationEndpoint {
     HTableDescriptor htd = HTU.createTableDescriptor(tableName);
     htd.setRegionReplication(regionReplication);
     htd.setRegionMemstoreReplication(false);
-    HTU.getAdmin().createTable(htd);
+    createOrEnableTableWithRetries(htd, true);
 
     Connection connection = ConnectionFactory.createConnection(HTU.getConfiguration());
     Table table = connection.getTable(tableName);
@@ -355,7 +358,7 @@ public class TestRegionReplicaReplicationEndpoint {
     final TableName tableName = TableName.valueOf(name.getMethodName());
     HTableDescriptor htd = HTU.createTableDescriptor(tableName);
     htd.setRegionReplication(regionReplication);
-    HTU.getAdmin().createTable(htd);
+    createOrEnableTableWithRetries(htd, true);
 
 
     Connection connection = ConnectionFactory.createConnection(HTU.getConfiguration());
@@ -407,13 +410,13 @@ public class TestRegionReplicaReplicationEndpoint {
     htd.setRegionReplication(regionReplication);
     HTU.deleteTableIfAny(tableName);
 
-    HTU.getAdmin().createTable(htd);
+    createOrEnableTableWithRetries(htd, true);
     TableName toBeDisabledTable = TableName.valueOf(
       dropTable ? "droppedTable" : (disableReplication ? "disableReplication" : "disabledTable"));
     HTU.deleteTableIfAny(toBeDisabledTable);
     htd = HTU.createTableDescriptor(toBeDisabledTable.toString());
     htd.setRegionReplication(regionReplication);
-    HTU.getAdmin().createTable(htd);
+    createOrEnableTableWithRetries(htd, true);
 
     // both tables are created, now pause replication
     HTU.getAdmin().disableReplicationPeer(ServerRegionReplicaUtil.REGION_REPLICA_REPLICATION_PEER);
@@ -453,7 +456,7 @@ public class TestRegionReplicaReplicationEndpoint {
     } else if (disableReplication) {
       htd.setRegionReplication(regionReplication - 2);
       HTU.getAdmin().modifyTable(toBeDisabledTable, htd);
-      HTU.getAdmin().enableTable(toBeDisabledTable);
+      createOrEnableTableWithRetries(htd, false);
     }
     sinkWriter.append(toBeDisabledTable, encodedRegionName,
       HConstants.EMPTY_BYTE_ARRAY, Lists.newArrayList(entry, entry));
@@ -486,7 +489,7 @@ public class TestRegionReplicaReplicationEndpoint {
       HTU.getAdmin().disableTable(toBeDisabledTable); // disable the table
       htd.setRegionReplication(regionReplication);
       HTU.getAdmin().modifyTable(toBeDisabledTable, htd);
-      HTU.getAdmin().enableTable(toBeDisabledTable);
+      createOrEnableTableWithRetries(htd, false);
     }
 
     try {
@@ -508,4 +511,26 @@ public class TestRegionReplicaReplicationEndpoint {
       connection.close();
     }
   }
+
+  private void createOrEnableTableWithRetries(TableDescriptor htd, boolean createTableOperation) {
+    // Helper function to run create/enable table operations with a retry feature
+    boolean continueToRetry = true;
+    int tries = 0;
+    while (continueToRetry && tries < 50) {
+      try {
+        continueToRetry = false;
+        if (createTableOperation) {
+          HTU.getAdmin().createTable(htd);
+        } else {
+          HTU.getAdmin().enableTable(htd.getTableName());
+        }
+      } catch (IOException e) {
+        if (e.getCause() instanceof ReplicationException) {
+          continueToRetry = true;
+          tries++;
+          Uninterruptibles.sleepUninterruptibly(1, TimeUnit.SECONDS);
+        }
+      }
+    }
+  }
 }