HBASE-25522 Remove deprecated methods in ReplicationPeerConfig (#2898)

Signed-off-by: Viraj Jasani <vjasani@apache.org>
This commit is contained in:
Baiqiang Zhao 2021-01-24 23:37:03 +08:00 committed by GitHub
parent f39636c05e
commit 0b0cdcac53
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
13 changed files with 125 additions and 215 deletions

View File

@ -79,41 +79,6 @@ public class ReplicationPeerConfig {
return Collections.unmodifiableMap(newTableCFsMap); return Collections.unmodifiableMap(newTableCFsMap);
} }
/**
* @deprecated as release of 2.0.0, and it will be removed in 3.0.0. Use
* {@link ReplicationPeerConfigBuilder} to create new ReplicationPeerConfig.
*/
@Deprecated
public ReplicationPeerConfig() {
this.peerData = new TreeMap<>(Bytes.BYTES_COMPARATOR);
this.configuration = new HashMap<>(0);
this.serial = false;
}
/**
* Set the clusterKey which is the concatenation of the slave cluster's:
* hbase.zookeeper.quorum:hbase.zookeeper.property.clientPort:zookeeper.znode.parent
* @deprecated as release of 2.0.0, and it will be removed in 3.0.0. Use
* {@link ReplicationPeerConfigBuilder#setClusterKey(String)} instead.
*/
@Deprecated
public ReplicationPeerConfig setClusterKey(String clusterKey) {
this.clusterKey = clusterKey;
return this;
}
/**
* Sets the ReplicationEndpoint plugin class for this peer.
* @param replicationEndpointImpl a class implementing ReplicationEndpoint
* @deprecated as release of 2.0.0, and it will be removed in 3.0.0. Use
* {@link ReplicationPeerConfigBuilder#setReplicationEndpointImpl(String)} instead.
*/
@Deprecated
public ReplicationPeerConfig setReplicationEndpointImpl(String replicationEndpointImpl) {
this.replicationEndpointImpl = replicationEndpointImpl;
return this;
}
public String getClusterKey() { public String getClusterKey() {
return clusterKey; return clusterKey;
} }
@ -134,88 +99,26 @@ public class ReplicationPeerConfig {
return (Map<TableName, List<String>>) tableCFsMap; return (Map<TableName, List<String>>) tableCFsMap;
} }
/**
* @deprecated as release of 2.0.0, and it will be removed in 3.0.0. Use
* {@link ReplicationPeerConfigBuilder#setTableCFsMap(Map)} instead.
*/
@Deprecated
public ReplicationPeerConfig setTableCFsMap(Map<TableName,
? extends Collection<String>> tableCFsMap) {
this.tableCFsMap = tableCFsMap;
return this;
}
public Set<String> getNamespaces() { public Set<String> getNamespaces() {
return this.namespaces; return this.namespaces;
} }
/**
* @deprecated as release of 2.0.0, and it will be removed in 3.0.0. Use
* {@link ReplicationPeerConfigBuilder#setNamespaces(Set)} instead.
*/
@Deprecated
public ReplicationPeerConfig setNamespaces(Set<String> namespaces) {
this.namespaces = namespaces;
return this;
}
public long getBandwidth() { public long getBandwidth() {
return this.bandwidth; return this.bandwidth;
} }
/**
* @deprecated as release of 2.0.0, and it will be removed in 3.0.0. Use
* {@link ReplicationPeerConfigBuilder#setBandwidth(long)} instead.
*/
@Deprecated
public ReplicationPeerConfig setBandwidth(long bandwidth) {
this.bandwidth = bandwidth;
return this;
}
public boolean replicateAllUserTables() { public boolean replicateAllUserTables() {
return this.replicateAllUserTables; return this.replicateAllUserTables;
} }
/**
* @deprecated as release of 2.0.0, and it will be removed in 3.0.0. Use
* {@link ReplicationPeerConfigBuilder#setReplicateAllUserTables(boolean)} instead.
*/
@Deprecated
public ReplicationPeerConfig setReplicateAllUserTables(boolean replicateAllUserTables) {
this.replicateAllUserTables = replicateAllUserTables;
return this;
}
public Map<TableName, List<String>> getExcludeTableCFsMap() { public Map<TableName, List<String>> getExcludeTableCFsMap() {
return (Map<TableName, List<String>>) excludeTableCFsMap; return (Map<TableName, List<String>>) excludeTableCFsMap;
} }
/**
* @deprecated as release of 2.0.0, and it will be removed in 3.0.0. Use
* {@link ReplicationPeerConfigBuilder#setExcludeTableCFsMap(Map)} instead.
*/
@Deprecated
public ReplicationPeerConfig setExcludeTableCFsMap(Map<TableName,
? extends Collection<String>> tableCFsMap) {
this.excludeTableCFsMap = tableCFsMap;
return this;
}
public Set<String> getExcludeNamespaces() { public Set<String> getExcludeNamespaces() {
return this.excludeNamespaces; return this.excludeNamespaces;
} }
/**
* @deprecated as release of 2.0.0, and it will be removed in 3.0.0. Use
* {@link ReplicationPeerConfigBuilder#setExcludeNamespaces(Set)} instead.
*/
@Deprecated
public ReplicationPeerConfig setExcludeNamespaces(Set<String> namespaces) {
this.excludeNamespaces = namespaces;
return this;
}
public String getRemoteWALDir() { public String getRemoteWALDir() {
return this.remoteWALDir; return this.remoteWALDir;
} }

View File

@ -44,6 +44,7 @@ import org.apache.hadoop.hbase.ServerName;
import org.apache.hadoop.hbase.TableName; import org.apache.hadoop.hbase.TableName;
import org.apache.hadoop.hbase.replication.ReplicationException; import org.apache.hadoop.hbase.replication.ReplicationException;
import org.apache.hadoop.hbase.replication.ReplicationPeerConfig; import org.apache.hadoop.hbase.replication.ReplicationPeerConfig;
import org.apache.hadoop.hbase.replication.ReplicationPeerConfigBuilder;
import org.apache.hadoop.hbase.replication.ReplicationPeerDescription; import org.apache.hadoop.hbase.replication.ReplicationPeerDescription;
import org.apache.hadoop.hbase.replication.ReplicationQueueStorage; import org.apache.hadoop.hbase.replication.ReplicationQueueStorage;
import org.apache.hadoop.hbase.replication.ReplicationStorageFactory; import org.apache.hadoop.hbase.replication.ReplicationStorageFactory;
@ -108,10 +109,8 @@ public class TestAsyncReplicationAdminApi extends TestAsyncAdminBase {
@Test @Test
public void testAddRemovePeer() throws Exception { public void testAddRemovePeer() throws Exception {
ReplicationPeerConfig rpc1 = new ReplicationPeerConfig(); ReplicationPeerConfig rpc1 = ReplicationPeerConfig.newBuilder().setClusterKey(KEY_ONE).build();
rpc1.setClusterKey(KEY_ONE); ReplicationPeerConfig rpc2 = ReplicationPeerConfig.newBuilder().setClusterKey(KEY_TWO).build();
ReplicationPeerConfig rpc2 = new ReplicationPeerConfig();
rpc2.setClusterKey(KEY_TWO);
// Add a valid peer // Add a valid peer
admin.addReplicationPeer(ID_ONE, rpc1).join(); admin.addReplicationPeer(ID_ONE, rpc1).join();
// try adding the same (fails) // try adding the same (fails)
@ -142,10 +141,11 @@ public class TestAsyncReplicationAdminApi extends TestAsyncAdminBase {
@Test @Test
public void testPeerConfig() throws Exception { public void testPeerConfig() throws Exception {
ReplicationPeerConfig config = new ReplicationPeerConfig(); ReplicationPeerConfig config = ReplicationPeerConfig.newBuilder()
config.setClusterKey(KEY_ONE); .setClusterKey(KEY_ONE)
config.getConfiguration().put("key1", "value1"); .putConfiguration("key1", "value1")
config.getConfiguration().put("key2", "value2"); .putConfiguration("key2", "value2")
.build();
admin.addReplicationPeer(ID_ONE, config).join(); admin.addReplicationPeer(ID_ONE, config).join();
List<ReplicationPeerDescription> peers = admin.listReplicationPeers().get(); List<ReplicationPeerDescription> peers = admin.listReplicationPeers().get();
@ -160,8 +160,7 @@ public class TestAsyncReplicationAdminApi extends TestAsyncAdminBase {
@Test @Test
public void testEnableDisablePeer() throws Exception { public void testEnableDisablePeer() throws Exception {
ReplicationPeerConfig rpc1 = new ReplicationPeerConfig(); ReplicationPeerConfig rpc1 = ReplicationPeerConfig.newBuilder().setClusterKey(KEY_ONE).build();
rpc1.setClusterKey(KEY_ONE);
admin.addReplicationPeer(ID_ONE, rpc1).join(); admin.addReplicationPeer(ID_ONE, rpc1).join();
List<ReplicationPeerDescription> peers = admin.listReplicationPeers().get(); List<ReplicationPeerDescription> peers = admin.listReplicationPeers().get();
assertEquals(1, peers.size()); assertEquals(1, peers.size());
@ -176,8 +175,8 @@ public class TestAsyncReplicationAdminApi extends TestAsyncAdminBase {
@Test @Test
public void testAppendPeerTableCFs() throws Exception { public void testAppendPeerTableCFs() throws Exception {
ReplicationPeerConfig rpc1 = new ReplicationPeerConfig(); ReplicationPeerConfigBuilder rpcBuilder =
rpc1.setClusterKey(KEY_ONE); ReplicationPeerConfig.newBuilder().setClusterKey(KEY_ONE);
final TableName tableName1 = TableName.valueOf(tableName.getNameAsString() + "t1"); final TableName tableName1 = TableName.valueOf(tableName.getNameAsString() + "t1");
final TableName tableName2 = TableName.valueOf(tableName.getNameAsString() + "t2"); final TableName tableName2 = TableName.valueOf(tableName.getNameAsString() + "t2");
final TableName tableName3 = TableName.valueOf(tableName.getNameAsString() + "t3"); final TableName tableName3 = TableName.valueOf(tableName.getNameAsString() + "t3");
@ -186,9 +185,9 @@ public class TestAsyncReplicationAdminApi extends TestAsyncAdminBase {
final TableName tableName6 = TableName.valueOf(tableName.getNameAsString() + "t6"); final TableName tableName6 = TableName.valueOf(tableName.getNameAsString() + "t6");
// Add a valid peer // Add a valid peer
admin.addReplicationPeer(ID_ONE, rpc1).join(); admin.addReplicationPeer(ID_ONE, rpcBuilder.build()).join();
rpc1.setReplicateAllUserTables(false); rpcBuilder.setReplicateAllUserTables(false);
admin.updateReplicationPeerConfig(ID_ONE, rpc1).join(); admin.updateReplicationPeerConfig(ID_ONE, rpcBuilder.build()).join();
Map<TableName, List<String>> tableCFs = new HashMap<>(); Map<TableName, List<String>> tableCFs = new HashMap<>();
@ -280,16 +279,16 @@ public class TestAsyncReplicationAdminApi extends TestAsyncAdminBase {
@Test @Test
public void testRemovePeerTableCFs() throws Exception { public void testRemovePeerTableCFs() throws Exception {
ReplicationPeerConfig rpc1 = new ReplicationPeerConfig(); ReplicationPeerConfigBuilder rpcBuilder =
rpc1.setClusterKey(KEY_ONE); ReplicationPeerConfig.newBuilder().setClusterKey(KEY_ONE);
final TableName tableName1 = TableName.valueOf(tableName.getNameAsString() + "t1"); final TableName tableName1 = TableName.valueOf(tableName.getNameAsString() + "t1");
final TableName tableName2 = TableName.valueOf(tableName.getNameAsString() + "t2"); final TableName tableName2 = TableName.valueOf(tableName.getNameAsString() + "t2");
final TableName tableName3 = TableName.valueOf(tableName.getNameAsString() + "t3"); final TableName tableName3 = TableName.valueOf(tableName.getNameAsString() + "t3");
final TableName tableName4 = TableName.valueOf(tableName.getNameAsString() + "t4"); final TableName tableName4 = TableName.valueOf(tableName.getNameAsString() + "t4");
// Add a valid peer // Add a valid peer
admin.addReplicationPeer(ID_ONE, rpc1).join(); admin.addReplicationPeer(ID_ONE, rpcBuilder.build()).join();
rpc1.setReplicateAllUserTables(false); rpcBuilder.setReplicateAllUserTables(false);
admin.updateReplicationPeerConfig(ID_ONE, rpc1).join(); admin.updateReplicationPeerConfig(ID_ONE, rpcBuilder.build()).join();
Map<TableName, List<String>> tableCFs = new HashMap<>(); Map<TableName, List<String>> tableCFs = new HashMap<>();
try { try {
@ -369,30 +368,28 @@ public class TestAsyncReplicationAdminApi extends TestAsyncAdminBase {
String ns1 = "ns1"; String ns1 = "ns1";
String ns2 = "ns2"; String ns2 = "ns2";
ReplicationPeerConfig rpc = new ReplicationPeerConfig(); ReplicationPeerConfigBuilder rpcBuilder =
rpc.setClusterKey(KEY_ONE); ReplicationPeerConfig.newBuilder().setClusterKey(KEY_ONE);
admin.addReplicationPeer(ID_ONE, rpc).join(); admin.addReplicationPeer(ID_ONE, rpcBuilder.build()).join();
rpc.setReplicateAllUserTables(false); rpcBuilder.setReplicateAllUserTables(false);
admin.updateReplicationPeerConfig(ID_ONE, rpc).join(); admin.updateReplicationPeerConfig(ID_ONE, rpcBuilder.build()).join();
// add ns1 and ns2 to peer config // add ns1 and ns2 to peer config
rpc = admin.getReplicationPeerConfig(ID_ONE).get();
Set<String> namespaces = new HashSet<>(); Set<String> namespaces = new HashSet<>();
namespaces.add(ns1); namespaces.add(ns1);
namespaces.add(ns2); namespaces.add(ns2);
rpc.setNamespaces(namespaces); rpcBuilder.setNamespaces(namespaces);
admin.updateReplicationPeerConfig(ID_ONE, rpc).join(); admin.updateReplicationPeerConfig(ID_ONE, rpcBuilder.build()).join();
namespaces = admin.getReplicationPeerConfig(ID_ONE).get().getNamespaces(); namespaces = admin.getReplicationPeerConfig(ID_ONE).get().getNamespaces();
assertEquals(2, namespaces.size()); assertEquals(2, namespaces.size());
assertTrue(namespaces.contains(ns1)); assertTrue(namespaces.contains(ns1));
assertTrue(namespaces.contains(ns2)); assertTrue(namespaces.contains(ns2));
// update peer config only contains ns1 // update peer config only contains ns1
rpc = admin.getReplicationPeerConfig(ID_ONE).get();
namespaces = new HashSet<>(); namespaces = new HashSet<>();
namespaces.add(ns1); namespaces.add(ns1);
rpc.setNamespaces(namespaces); rpcBuilder.setNamespaces(namespaces);
admin.updateReplicationPeerConfig(ID_ONE, rpc).join(); admin.updateReplicationPeerConfig(ID_ONE, rpcBuilder.build()).join();
namespaces = admin.getReplicationPeerConfig(ID_ONE).get().getNamespaces(); namespaces = admin.getReplicationPeerConfig(ID_ONE).get().getNamespaces();
assertEquals(1, namespaces.size()); assertEquals(1, namespaces.size());
assertTrue(namespaces.contains(ns1)); assertTrue(namespaces.contains(ns1));
@ -407,40 +404,36 @@ public class TestAsyncReplicationAdminApi extends TestAsyncAdminBase {
final TableName tableName1 = TableName.valueOf(ns1 + ":" + tableName.getNameAsString() + "1"); final TableName tableName1 = TableName.valueOf(ns1 + ":" + tableName.getNameAsString() + "1");
final TableName tableName2 = TableName.valueOf(ns2 + ":" + tableName.getNameAsString() + "2"); final TableName tableName2 = TableName.valueOf(ns2 + ":" + tableName.getNameAsString() + "2");
ReplicationPeerConfig rpc = new ReplicationPeerConfig(); ReplicationPeerConfigBuilder rpcBuilder =
rpc.setClusterKey(KEY_ONE); ReplicationPeerConfig.newBuilder().setClusterKey(KEY_ONE);
admin.addReplicationPeer(ID_ONE, rpc).join(); admin.addReplicationPeer(ID_ONE, rpcBuilder.build()).join();
rpc.setReplicateAllUserTables(false); rpcBuilder.setReplicateAllUserTables(false);
admin.updateReplicationPeerConfig(ID_ONE, rpc).join(); admin.updateReplicationPeerConfig(ID_ONE, rpcBuilder.build()).join();
rpc = admin.getReplicationPeerConfig(ID_ONE).get();
Set<String> namespaces = new HashSet<String>(); Set<String> namespaces = new HashSet<String>();
namespaces.add(ns1); namespaces.add(ns1);
rpc.setNamespaces(namespaces); rpcBuilder.setNamespaces(namespaces);
admin.updateReplicationPeerConfig(ID_ONE, rpc).get(); admin.updateReplicationPeerConfig(ID_ONE, rpcBuilder.build()).get();
rpc = admin.getReplicationPeerConfig(ID_ONE).get();
Map<TableName, List<String>> tableCfs = new HashMap<>(); Map<TableName, List<String>> tableCfs = new HashMap<>();
tableCfs.put(tableName1, new ArrayList<>()); tableCfs.put(tableName1, new ArrayList<>());
rpc.setTableCFsMap(tableCfs); rpcBuilder.setTableCFsMap(tableCfs);
try { try {
admin.updateReplicationPeerConfig(ID_ONE, rpc).join(); admin.updateReplicationPeerConfig(ID_ONE, rpcBuilder.build()).join();
fail( fail(
"Test case should fail, because table " + tableName1 + " conflict with namespace " + ns1); "Test case should fail, because table " + tableName1 + " conflict with namespace " + ns1);
} catch (CompletionException e) { } catch (CompletionException e) {
// OK // OK
} }
rpc = admin.getReplicationPeerConfig(ID_ONE).get();
tableCfs.clear(); tableCfs.clear();
tableCfs.put(tableName2, new ArrayList<>()); tableCfs.put(tableName2, new ArrayList<>());
rpc.setTableCFsMap(tableCfs); rpcBuilder.setTableCFsMap(tableCfs);
admin.updateReplicationPeerConfig(ID_ONE, rpc).get(); admin.updateReplicationPeerConfig(ID_ONE, rpcBuilder.build()).get();
rpc = admin.getReplicationPeerConfig(ID_ONE).get();
namespaces.clear(); namespaces.clear();
namespaces.add(ns2); namespaces.add(ns2);
rpc.setNamespaces(namespaces); rpcBuilder.setNamespaces(namespaces);
try { try {
admin.updateReplicationPeerConfig(ID_ONE, rpc).join(); admin.updateReplicationPeerConfig(ID_ONE, rpcBuilder.build()).join();
fail( fail(
"Test case should fail, because namespace " + ns2 + " conflict with table " + tableName2); "Test case should fail, because namespace " + ns2 + " conflict with table " + tableName2);
} catch (CompletionException e) { } catch (CompletionException e) {
@ -452,15 +445,14 @@ public class TestAsyncReplicationAdminApi extends TestAsyncAdminBase {
@Test @Test
public void testPeerBandwidth() throws Exception { public void testPeerBandwidth() throws Exception {
ReplicationPeerConfig rpc = new ReplicationPeerConfig(); ReplicationPeerConfigBuilder rpcBuilder =
rpc.setClusterKey(KEY_ONE); ReplicationPeerConfig.newBuilder().setClusterKey(KEY_ONE);
admin.addReplicationPeer(ID_ONE, rpc).join(); admin.addReplicationPeer(ID_ONE, rpcBuilder.build()).join();;
rpc = admin.getReplicationPeerConfig(ID_ONE).get(); assertEquals(0, admin.getReplicationPeerConfig(ID_ONE).get().getBandwidth());
assertEquals(0, rpc.getBandwidth());
rpc.setBandwidth(2097152); rpcBuilder.setBandwidth(2097152);
admin.updateReplicationPeerConfig(ID_ONE, rpc).join(); admin.updateReplicationPeerConfig(ID_ONE, rpcBuilder.build()).join();
assertEquals(2097152, admin.getReplicationPeerConfig(ID_ONE).join().getBandwidth()); assertEquals(2097152, admin.getReplicationPeerConfig(ID_ONE).join().getBandwidth());
admin.removeReplicationPeer(ID_ONE).join(); admin.removeReplicationPeer(ID_ONE).join();

View File

@ -24,8 +24,8 @@ import static org.junit.Assert.assertTrue;
import static org.junit.Assert.fail; import static org.junit.Assert.fail;
import java.io.IOException; import java.io.IOException;
import java.util.Collection;
import java.util.HashMap; import java.util.HashMap;
import java.util.List;
import java.util.Map; import java.util.Map;
import java.util.concurrent.CompletionException; import java.util.concurrent.CompletionException;
import java.util.concurrent.ForkJoinPool; import java.util.concurrent.ForkJoinPool;
@ -38,6 +38,7 @@ import org.apache.hadoop.hbase.HConstants;
import org.apache.hadoop.hbase.TableName; import org.apache.hadoop.hbase.TableName;
import org.apache.hadoop.hbase.TableNotFoundException; import org.apache.hadoop.hbase.TableNotFoundException;
import org.apache.hadoop.hbase.replication.ReplicationPeerConfig; import org.apache.hadoop.hbase.replication.ReplicationPeerConfig;
import org.apache.hadoop.hbase.replication.ReplicationPeerConfigBuilder;
import org.apache.hadoop.hbase.testclassification.ClientTests; import org.apache.hadoop.hbase.testclassification.ClientTests;
import org.apache.hadoop.hbase.testclassification.LargeTests; import org.apache.hadoop.hbase.testclassification.LargeTests;
import org.apache.hadoop.hbase.util.Bytes; import org.apache.hadoop.hbase.util.Bytes;
@ -86,8 +87,8 @@ public class TestAsyncReplicationAdminApiWithClusters extends TestAsyncAdminBase
ConnectionFactory.createAsyncConnection(TEST_UTIL2.getConfiguration()).get(); ConnectionFactory.createAsyncConnection(TEST_UTIL2.getConfiguration()).get();
admin2 = connection.getAdmin(); admin2 = connection.getAdmin();
ReplicationPeerConfig rpc = new ReplicationPeerConfig(); ReplicationPeerConfig rpc = ReplicationPeerConfig.newBuilder()
rpc.setClusterKey(TEST_UTIL2.getClusterKey()); .setClusterKey(TEST_UTIL2.getClusterKey()).build();
ASYNC_CONN.getAdmin().addReplicationPeer(ID_SECOND, rpc).join(); ASYNC_CONN.getAdmin().addReplicationPeer(ID_SECOND, rpc).join();
} }
@ -231,30 +232,30 @@ public class TestAsyncReplicationAdminApiWithClusters extends TestAsyncAdminBase
assertFalse("Table should not exists in the peer cluster", assertFalse("Table should not exists in the peer cluster",
admin2.tableExists(tableName2).get()); admin2.tableExists(tableName2).get());
Map<TableName, ? extends Collection<String>> tableCfs = new HashMap<>(); Map<TableName, List<String>> tableCfs = new HashMap<>();
tableCfs.put(tableName, null); tableCfs.put(tableName, null);
ReplicationPeerConfig rpc = admin.getReplicationPeerConfig(ID_SECOND).get(); ReplicationPeerConfigBuilder rpcBuilder = ReplicationPeerConfig
rpc.setReplicateAllUserTables(false); .newBuilder(admin.getReplicationPeerConfig(ID_SECOND).get())
rpc.setTableCFsMap(tableCfs); .setReplicateAllUserTables(false)
.setTableCFsMap(tableCfs);
try { try {
// Only add tableName to replication peer config // Only add tableName to replication peer config
admin.updateReplicationPeerConfig(ID_SECOND, rpc).join(); admin.updateReplicationPeerConfig(ID_SECOND, rpcBuilder.build()).join();
admin.enableTableReplication(tableName2).join(); admin.enableTableReplication(tableName2).join();
assertFalse("Table should not be created if user has set table cfs explicitly for the " assertFalse("Table should not be created if user has set table cfs explicitly for the "
+ "peer and this is not part of that collection", admin2.tableExists(tableName2).get()); + "peer and this is not part of that collection", admin2.tableExists(tableName2).get());
// Add tableName2 to replication peer config, too // Add tableName2 to replication peer config, too
tableCfs.put(tableName2, null); tableCfs.put(tableName2, null);
rpc.setTableCFsMap(tableCfs); rpcBuilder.setTableCFsMap(tableCfs);
admin.updateReplicationPeerConfig(ID_SECOND, rpc).join(); admin.updateReplicationPeerConfig(ID_SECOND, rpcBuilder.build()).join();
admin.enableTableReplication(tableName2).join(); admin.enableTableReplication(tableName2).join();
assertTrue( assertTrue(
"Table should be created if user has explicitly added table into table cfs collection", "Table should be created if user has explicitly added table into table cfs collection",
admin2.tableExists(tableName2).get()); admin2.tableExists(tableName2).get());
} finally { } finally {
rpc.setTableCFsMap(null); rpcBuilder.setTableCFsMap(null).setReplicateAllUserTables(true).build();
rpc.setReplicateAllUserTables(true); admin.updateReplicationPeerConfig(ID_SECOND, rpcBuilder.build()).join();
admin.updateReplicationPeerConfig(ID_SECOND, rpc).join();
} }
} }
} }

View File

@ -395,8 +395,8 @@ public class TestReplicaWithCluster {
try (Connection connection = ConnectionFactory.createConnection(HTU.getConfiguration()); try (Connection connection = ConnectionFactory.createConnection(HTU.getConfiguration());
Admin admin = connection.getAdmin()) { Admin admin = connection.getAdmin()) {
ReplicationPeerConfig rpc = new ReplicationPeerConfig(); ReplicationPeerConfig rpc = ReplicationPeerConfig.newBuilder()
rpc.setClusterKey(HTU2.getClusterKey()); .setClusterKey(HTU2.getClusterKey()).build();
admin.addReplicationPeer("2", rpc); admin.addReplicationPeer("2", rpc);
} }

View File

@ -623,7 +623,8 @@ public class TestMasterReplication {
try (Connection conn = ConnectionFactory.createConnection(configurations[masterClusterNumber]); try (Connection conn = ConnectionFactory.createConnection(configurations[masterClusterNumber]);
Admin admin = conn.getAdmin()) { Admin admin = conn.getAdmin()) {
admin.addReplicationPeer(id, admin.addReplicationPeer(id,
new ReplicationPeerConfig().setClusterKey(utilities[slaveClusterNumber].getClusterKey())); ReplicationPeerConfig.newBuilder().
setClusterKey(utilities[slaveClusterNumber].getClusterKey()).build());
} }
} }
@ -633,9 +634,10 @@ public class TestMasterReplication {
Admin admin = conn.getAdmin()) { Admin admin = conn.getAdmin()) {
admin.addReplicationPeer( admin.addReplicationPeer(
id, id,
new ReplicationPeerConfig().setClusterKey(utilities[slaveClusterNumber].getClusterKey()) ReplicationPeerConfig.newBuilder()
.setReplicateAllUserTables(false) .setClusterKey(utilities[slaveClusterNumber].getClusterKey())
.setTableCFsMap(ReplicationPeerConfigUtil.parseTableCFsFromConfig(tableCfs))); .setReplicateAllUserTables(false)
.setTableCFsMap(ReplicationPeerConfigUtil.parseTableCFsFromConfig(tableCfs)).build());
} }
} }

View File

@ -145,9 +145,9 @@ public class TestMultiSlaveReplication {
Table htable2 = utility2.getConnection().getTable(tableName); Table htable2 = utility2.getConnection().getTable(tableName);
Table htable3 = utility3.getConnection().getTable(tableName); Table htable3 = utility3.getConnection().getTable(tableName);
ReplicationPeerConfig rpc = new ReplicationPeerConfig(); ReplicationPeerConfigBuilder rpcBuilder =
rpc.setClusterKey(utility2.getClusterKey()); ReplicationPeerConfig.newBuilder().setClusterKey(utility2.getClusterKey());
admin1.addReplicationPeer("1", rpc); admin1.addReplicationPeer("1", rpcBuilder.build());
// put "row" and wait 'til it got around, then delete // put "row" and wait 'til it got around, then delete
putAndWait(row, famName, htable1, htable2); putAndWait(row, famName, htable1, htable2);
@ -163,9 +163,8 @@ public class TestMultiSlaveReplication {
// after the log was rolled put a new row // after the log was rolled put a new row
putAndWait(row3, famName, htable1, htable2); putAndWait(row3, famName, htable1, htable2);
rpc = new ReplicationPeerConfig(); rpcBuilder.setClusterKey(utility3.getClusterKey());
rpc.setClusterKey(utility3.getClusterKey()); admin1.addReplicationPeer("2", rpcBuilder.build());
admin1.addReplicationPeer("2", rpc);
// put a row, check it was replicated to all clusters // put a row, check it was replicated to all clusters
putAndWait(row1, famName, htable1, htable2, htable3); putAndWait(row1, famName, htable1, htable2, htable3);

View File

@ -142,8 +142,10 @@ public class TestReplicationEndpoint extends TestReplicationBase {
public void testCustomReplicationEndpoint() throws Exception { public void testCustomReplicationEndpoint() throws Exception {
// test installing a custom replication endpoint other than the default one. // test installing a custom replication endpoint other than the default one.
hbaseAdmin.addReplicationPeer("testCustomReplicationEndpoint", hbaseAdmin.addReplicationPeer("testCustomReplicationEndpoint",
new ReplicationPeerConfig().setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF1)) ReplicationPeerConfig.newBuilder()
.setReplicationEndpointImpl(ReplicationEndpointForTest.class.getName())); .setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF1))
.setReplicationEndpointImpl(ReplicationEndpointForTest.class.getName())
.build());
// check whether the class has been constructed and started // check whether the class has been constructed and started
Waiter.waitFor(CONF1, 60000, new Waiter.Predicate<Exception>() { Waiter.waitFor(CONF1, 60000, new Waiter.Predicate<Exception>() {
@ -184,8 +186,10 @@ public class TestReplicationEndpoint extends TestReplicationBase {
int peerCount = hbaseAdmin.listReplicationPeers().size(); int peerCount = hbaseAdmin.listReplicationPeers().size();
final String id = "testReplicationEndpointReturnsFalseOnReplicate"; final String id = "testReplicationEndpointReturnsFalseOnReplicate";
hbaseAdmin.addReplicationPeer(id, hbaseAdmin.addReplicationPeer(id,
new ReplicationPeerConfig().setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF1)) ReplicationPeerConfig.newBuilder()
.setReplicationEndpointImpl(ReplicationEndpointReturningFalse.class.getName())); .setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF1))
.setReplicationEndpointImpl(ReplicationEndpointReturningFalse.class.getName())
.build());
// This test is flakey and then there is so much stuff flying around in here its, hard to // This test is flakey and then there is so much stuff flying around in here its, hard to
// debug. Peer needs to be up for the edit to make it across. This wait on // debug. Peer needs to be up for the edit to make it across. This wait on
// peer count seems to be a hack that has us not progress till peer is up. // peer count seems to be a hack that has us not progress till peer is up.
@ -236,8 +240,10 @@ public class TestReplicationEndpoint extends TestReplicationBase {
} }
hbaseAdmin.addReplicationPeer(id, hbaseAdmin.addReplicationPeer(id,
new ReplicationPeerConfig().setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF2)) ReplicationPeerConfig.newBuilder()
.setReplicationEndpointImpl(InterClusterReplicationEndpointForTest.class.getName())); .setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF2))
.setReplicationEndpointImpl(InterClusterReplicationEndpointForTest.class.getName())
.build());
final int numEdits = totEdits; final int numEdits = totEdits;
Waiter.waitFor(CONF1, 30000, new Waiter.ExplainingPredicate<Exception>() { Waiter.waitFor(CONF1, 30000, new Waiter.ExplainingPredicate<Exception>() {
@ -260,13 +266,15 @@ public class TestReplicationEndpoint extends TestReplicationBase {
@Test @Test
public void testWALEntryFilterFromReplicationEndpoint() throws Exception { public void testWALEntryFilterFromReplicationEndpoint() throws Exception {
ReplicationPeerConfig rpc = ReplicationPeerConfig rpc = ReplicationPeerConfig.newBuilder()
new ReplicationPeerConfig().setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF1)) .setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF1))
.setReplicationEndpointImpl(ReplicationEndpointWithWALEntryFilter.class.getName()); .setReplicationEndpointImpl(ReplicationEndpointWithWALEntryFilter.class.getName())
// test that we can create mutliple WALFilters reflectively // test that we can create mutliple WALFilters reflectively
rpc.getConfiguration().put(BaseReplicationEndpoint.REPLICATION_WALENTRYFILTER_CONFIG_KEY, .putConfiguration(BaseReplicationEndpoint.REPLICATION_WALENTRYFILTER_CONFIG_KEY,
EverythingPassesWALEntryFilter.class.getName() + "," + EverythingPassesWALEntryFilter.class.getName() + "," +
EverythingPassesWALEntryFilterSubclass.class.getName()); EverythingPassesWALEntryFilterSubclass.class.getName())
.build();
hbaseAdmin.addReplicationPeer("testWALEntryFilterFromReplicationEndpoint", rpc); hbaseAdmin.addReplicationPeer("testWALEntryFilterFromReplicationEndpoint", rpc);
// now replicate some data. // now replicate some data.
try (Connection connection = ConnectionFactory.createConnection(CONF1)) { try (Connection connection = ConnectionFactory.createConnection(CONF1)) {
@ -290,23 +298,25 @@ public class TestReplicationEndpoint extends TestReplicationBase {
@Test(expected = IOException.class) @Test(expected = IOException.class)
public void testWALEntryFilterAddValidation() throws Exception { public void testWALEntryFilterAddValidation() throws Exception {
ReplicationPeerConfig rpc = ReplicationPeerConfig rpc = ReplicationPeerConfig.newBuilder()
new ReplicationPeerConfig().setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF1)) .setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF1))
.setReplicationEndpointImpl(ReplicationEndpointWithWALEntryFilter.class.getName()); .setReplicationEndpointImpl(ReplicationEndpointWithWALEntryFilter.class.getName())
// test that we can create mutliple WALFilters reflectively // test that we can create mutliple WALFilters reflectively
rpc.getConfiguration().put(BaseReplicationEndpoint.REPLICATION_WALENTRYFILTER_CONFIG_KEY, .putConfiguration(BaseReplicationEndpoint.REPLICATION_WALENTRYFILTER_CONFIG_KEY,
"IAmNotARealWalEntryFilter"); "IAmNotARealWalEntryFilter")
.build();
hbaseAdmin.addReplicationPeer("testWALEntryFilterAddValidation", rpc); hbaseAdmin.addReplicationPeer("testWALEntryFilterAddValidation", rpc);
} }
@Test(expected = IOException.class) @Test(expected = IOException.class)
public void testWALEntryFilterUpdateValidation() throws Exception { public void testWALEntryFilterUpdateValidation() throws Exception {
ReplicationPeerConfig rpc = ReplicationPeerConfig rpc = ReplicationPeerConfig.newBuilder()
new ReplicationPeerConfig().setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF1)) .setClusterKey(ZKConfig.getZooKeeperClusterKey(CONF1))
.setReplicationEndpointImpl(ReplicationEndpointWithWALEntryFilter.class.getName()); .setReplicationEndpointImpl(ReplicationEndpointWithWALEntryFilter.class.getName())
// test that we can create mutliple WALFilters reflectively // test that we can create mutliple WALFilters reflectively
rpc.getConfiguration().put(BaseReplicationEndpoint.REPLICATION_WALENTRYFILTER_CONFIG_KEY, .putConfiguration(BaseReplicationEndpoint.REPLICATION_WALENTRYFILTER_CONFIG_KEY,
"IAmNotARealWalEntryFilter"); "IAmNotARealWalEntryFilter")
.build();
hbaseAdmin.updateReplicationPeerConfig("testWALEntryFilterUpdateValidation", rpc); hbaseAdmin.updateReplicationPeerConfig("testWALEntryFilterUpdateValidation", rpc);
} }

View File

@ -143,8 +143,8 @@ public class TestReplicationWithTags {
connection1 = ConnectionFactory.createConnection(conf1); connection1 = ConnectionFactory.createConnection(conf1);
replicationAdmin = connection1.getAdmin(); replicationAdmin = connection1.getAdmin();
ReplicationPeerConfig rpc = new ReplicationPeerConfig(); ReplicationPeerConfig rpc = ReplicationPeerConfig.newBuilder()
rpc.setClusterKey(utility2.getClusterKey()); .setClusterKey(utility2.getClusterKey()).build();
replicationAdmin.addReplicationPeer("2", rpc); replicationAdmin.addReplicationPeer("2", rpc);
TableDescriptor tableDescriptor = TableDescriptorBuilder.newBuilder(TABLE_NAME) TableDescriptor tableDescriptor = TableDescriptorBuilder.newBuilder(TABLE_NAME)

View File

@ -101,8 +101,8 @@ public class TestGlobalReplicationThrottler {
utility2.setZkCluster(miniZK); utility2.setZkCluster(miniZK);
new ZKWatcher(conf2, "cluster2", null, true); new ZKWatcher(conf2, "cluster2", null, true);
ReplicationPeerConfig rpc = new ReplicationPeerConfig(); ReplicationPeerConfig rpc = ReplicationPeerConfig.newBuilder()
rpc.setClusterKey(utility2.getClusterKey()); .setClusterKey(utility2.getClusterKey()).build();
utility1.startMiniCluster(); utility1.startMiniCluster();
utility2.startMiniCluster(); utility2.startMiniCluster();

View File

@ -71,8 +71,8 @@ public class TestReplicator extends TestReplicationBase {
// Replace the peer set up for us by the base class with a wrapper for this test // Replace the peer set up for us by the base class with a wrapper for this test
hbaseAdmin.addReplicationPeer("testReplicatorBatching", hbaseAdmin.addReplicationPeer("testReplicatorBatching",
new ReplicationPeerConfig().setClusterKey(UTIL2.getClusterKey()) ReplicationPeerConfig.newBuilder().setClusterKey(UTIL2.getClusterKey())
.setReplicationEndpointImpl(ReplicationEndpointForTest.class.getName())); .setReplicationEndpointImpl(ReplicationEndpointForTest.class.getName()).build());
ReplicationEndpointForTest.setBatchCount(0); ReplicationEndpointForTest.setBatchCount(0);
ReplicationEndpointForTest.setEntriesCount(0); ReplicationEndpointForTest.setEntriesCount(0);
@ -120,8 +120,9 @@ public class TestReplicator extends TestReplicationBase {
// Replace the peer set up for us by the base class with a wrapper for this test // Replace the peer set up for us by the base class with a wrapper for this test
hbaseAdmin.addReplicationPeer("testReplicatorWithErrors", hbaseAdmin.addReplicationPeer("testReplicatorWithErrors",
new ReplicationPeerConfig().setClusterKey(UTIL2.getClusterKey()) ReplicationPeerConfig.newBuilder().setClusterKey(UTIL2.getClusterKey())
.setReplicationEndpointImpl(FailureInjectingReplicationEndpointForTest.class.getName())); .setReplicationEndpointImpl(FailureInjectingReplicationEndpointForTest.class.getName())
.build());
FailureInjectingReplicationEndpointForTest.setBatchCount(0); FailureInjectingReplicationEndpointForTest.setBatchCount(0);
FailureInjectingReplicationEndpointForTest.setEntriesCount(0); FailureInjectingReplicationEndpointForTest.setEntriesCount(0);

View File

@ -3056,7 +3056,8 @@ public class TestAccessController extends SecureTestUtil {
@Override @Override
public Object run() throws Exception { public Object run() throws Exception {
ACCESS_CONTROLLER.preUpdateReplicationPeerConfig( ACCESS_CONTROLLER.preUpdateReplicationPeerConfig(
ObserverContextImpl.createAndPrepare(CP_ENV), "test", new ReplicationPeerConfig()); ObserverContextImpl.createAndPrepare(CP_ENV), "test",
ReplicationPeerConfig.newBuilder().build());
return null; return null;
} }
}; };

View File

@ -137,8 +137,9 @@ public class TestVisibilityLabelReplicationWithExpAsString extends TestVisibilit
TEST_UTIL1.startMiniCluster(1); TEST_UTIL1.startMiniCluster(1);
admin = TEST_UTIL.getAdmin(); admin = TEST_UTIL.getAdmin();
ReplicationPeerConfig rpc = new ReplicationPeerConfig(); ReplicationPeerConfig rpc = ReplicationPeerConfig.newBuilder()
rpc.setClusterKey(TEST_UTIL1.getClusterKey()); .setClusterKey(TEST_UTIL1.getClusterKey())
.build();
admin.addReplicationPeer("2", rpc); admin.addReplicationPeer("2", rpc);
TableDescriptor tableDescriptor = TableDescriptor tableDescriptor =

View File

@ -190,8 +190,8 @@ public class TestVisibilityLabelsReplication {
TEST_UTIL1.startMiniCluster(1); TEST_UTIL1.startMiniCluster(1);
admin = TEST_UTIL.getAdmin(); admin = TEST_UTIL.getAdmin();
ReplicationPeerConfig rpc = new ReplicationPeerConfig(); ReplicationPeerConfig rpc = ReplicationPeerConfig.newBuilder()
rpc.setClusterKey(TEST_UTIL1.getClusterKey()); .setClusterKey(TEST_UTIL1.getClusterKey()).build();
admin.addReplicationPeer("2", rpc); admin.addReplicationPeer("2", rpc);
Admin hBaseAdmin = TEST_UTIL.getAdmin(); Admin hBaseAdmin = TEST_UTIL.getAdmin();