You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by zg...@apache.org on 2017/11/06 12:40:33 UTC
[1/2] hbase git commit: HBASE-18950 Remove Optional parameters in
AsyncAdmin interface
Repository: hbase
Updated Branches:
refs/heads/master bc3f3ee3b -> 888f2335c
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncAdminBase.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncAdminBase.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncAdminBase.java
index c3c4045..83ba244 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncAdminBase.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncAdminBase.java
@@ -21,10 +21,8 @@ import static org.apache.hadoop.hbase.client.AsyncProcess.START_LOG_ERRORS_AFTER
import java.util.Arrays;
import java.util.List;
-import java.util.Optional;
-import java.util.concurrent.ExecutionException;
+import java.util.concurrent.CompletableFuture;
import java.util.concurrent.ForkJoinPool;
-import java.util.concurrent.TimeUnit;
import java.util.function.Supplier;
import java.util.regex.Pattern;
@@ -41,8 +39,6 @@ import org.junit.Before;
import org.junit.BeforeClass;
import org.junit.Rule;
import org.junit.rules.TestName;
-import org.junit.runner.RunWith;
-import org.junit.runners.Parameterized;
import org.junit.runners.Parameterized.Parameter;
import org.junit.runners.Parameterized.Parameters;
@@ -106,7 +102,7 @@ public abstract class TestAsyncAdminBase {
@After
public void tearDown() throws Exception {
- admin.listTableNames(Optional.of(Pattern.compile(tableName.getNameAsString() + ".*")), false)
+ admin.listTableNames(Pattern.compile(tableName.getNameAsString() + ".*"), false)
.whenCompleteAsync((tables, err) -> {
if (tables != null) {
tables.forEach(table -> {
@@ -122,19 +118,21 @@ public abstract class TestAsyncAdminBase {
}
protected void createTableWithDefaultConf(TableName tableName) {
- createTableWithDefaultConf(tableName, Optional.empty());
+ createTableWithDefaultConf(tableName, null);
}
- protected void createTableWithDefaultConf(TableName tableName, Optional<byte[][]> splitKeys) {
+ protected void createTableWithDefaultConf(TableName tableName, byte[][] splitKeys) {
createTableWithDefaultConf(tableName, splitKeys, FAMILY);
}
- protected void createTableWithDefaultConf(TableName tableName, Optional<byte[][]> splitKeys,
+ protected void createTableWithDefaultConf(TableName tableName, byte[][] splitKeys,
byte[]... families) {
TableDescriptorBuilder builder = TableDescriptorBuilder.newBuilder(tableName);
for (byte[] family : families) {
builder.addColumnFamily(ColumnFamilyDescriptorBuilder.of(family));
}
- admin.createTable(builder.build(), splitKeys).join();
+ CompletableFuture<Void> future = splitKeys == null ? admin.createTable(builder.build())
+ : admin.createTable(builder.build(), splitKeys);
+ future.join();
}
}
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncClusterAdminApi.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncClusterAdminApi.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncClusterAdminApi.java
index 53de2b5..e7c439b 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncClusterAdminApi.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncClusterAdminApi.java
@@ -31,7 +31,6 @@ import java.util.Collection;
import java.util.EnumSet;
import java.util.List;
import java.util.Map;
-import java.util.Optional;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.ClusterStatus;
@@ -254,7 +253,7 @@ public class TestAsyncClusterAdminApi extends TestAsyncAdminBase {
List<RegionInfo> tableRegions = admin.getTableRegions(table).get();
List<RegionLoad> regionLoads = Lists.newArrayList();
for (ServerName serverName : servers) {
- regionLoads.addAll(admin.getRegionLoads(serverName, Optional.of(table)).get());
+ regionLoads.addAll(admin.getRegionLoads(serverName, table).get());
}
checkRegionsAndRegionLoads(tableRegions, regionLoads);
}
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncRegionAdminApi.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncRegionAdminApi.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncRegionAdminApi.java
index 262cac6..1ee1b94 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncRegionAdminApi.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncRegionAdminApi.java
@@ -200,7 +200,7 @@ public class TestAsyncRegionAdminApi extends TestAsyncAdminBase {
}
assertTrue(destServerName != null && !destServerName.equals(serverName));
- admin.move(hri.getRegionName(), Optional.of(destServerName)).get();
+ admin.move(hri.getRegionName(), destServerName).get();
long timeoutTime = System.currentTimeMillis() + 30000;
while (true) {
@@ -362,7 +362,7 @@ public class TestAsyncRegionAdminApi extends TestAsyncAdminBase {
@Test
public void testMergeRegions() throws Exception {
byte[][] splitRows = new byte[][] { Bytes.toBytes("3"), Bytes.toBytes("6") };
- createTableWithDefaultConf(tableName, Optional.of(splitRows));
+ createTableWithDefaultConf(tableName, splitRows);
RawAsyncTable metaTable = ASYNC_CONN.getRawTable(META_TABLE_NAME);
List<HRegionLocation> regionLocations =
@@ -419,8 +419,11 @@ public class TestAsyncRegionAdminApi extends TestAsyncAdminBase {
table.putAll(puts).join();
if (isSplitRegion) {
- admin.splitRegion(regionLocations.get(0).getRegionInfo().getRegionName(),
- Optional.ofNullable(splitPoint)).get();
+ if (splitPoint == null) {
+ admin.splitRegion(regionLocations.get(0).getRegionInfo().getRegionName()).get();
+ } else {
+ admin.splitRegion(regionLocations.get(0).getRegionInfo().getRegionName(), splitPoint).get();
+ }
} else {
if (splitPoint == null) {
admin.split(tableName).get();
@@ -450,7 +453,7 @@ public class TestAsyncRegionAdminApi extends TestAsyncAdminBase {
@Test
public void testCompactRegionServer() throws Exception {
byte[][] families = { Bytes.toBytes("f1"), Bytes.toBytes("f2"), Bytes.toBytes("f3") };
- createTableWithDefaultConf(tableName, Optional.empty(), families);
+ createTableWithDefaultConf(tableName, null, families);
loadData(tableName, families, 3000, 8);
List<HRegionServer> rsList =
@@ -491,7 +494,7 @@ public class TestAsyncRegionAdminApi extends TestAsyncAdminBase {
byte[] family = Bytes.toBytes("family");
byte[][] families =
{ family, Bytes.add(family, Bytes.toBytes("2")), Bytes.add(family, Bytes.toBytes("3")) };
- createTableWithDefaultConf(tableName, Optional.empty(), families);
+ createTableWithDefaultConf(tableName, null, families);
loadData(tableName, families, 3000, flushes);
List<Region> regions = new ArrayList<>();
@@ -506,15 +509,15 @@ public class TestAsyncRegionAdminApi extends TestAsyncAdminBase {
assertTrue(countBefore > 0); // there should be some data files
if (expectedState == CompactionState.MINOR) {
if (singleFamily) {
- admin.compact(tableName, Optional.of(family)).get();
+ admin.compact(tableName, family).get();
} else {
- admin.compact(tableName, Optional.empty()).get();
+ admin.compact(tableName).get();
}
} else {
if (singleFamily) {
- admin.majorCompact(tableName, Optional.of(family)).get();
+ admin.majorCompact(tableName, family).get();
} else {
- admin.majorCompact(tableName, Optional.empty()).get();
+ admin.majorCompact(tableName).get();
}
}
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncSnapshotAdminApi.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncSnapshotAdminApi.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncSnapshotAdminApi.java
index f789da5..5014e96 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncSnapshotAdminApi.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncSnapshotAdminApi.java
@@ -32,7 +32,6 @@ import org.junit.runners.Parameterized;
import java.io.IOException;
import java.util.Collections;
import java.util.List;
-import java.util.Optional;
import java.util.regex.Pattern;
@RunWith(Parameterized.class)
@@ -169,13 +168,10 @@ public class TestAsyncSnapshotAdminApi extends TestAsyncAdminBase {
admin.snapshot(snapshotName3, tableName).get();
Assert.assertEquals(admin.listSnapshots().get().size(), 3);
- Assert.assertEquals(admin.listSnapshots(Optional.of(Pattern.compile("(.*)"))).get().size(), 3);
- Assert.assertEquals(admin.listSnapshots(Optional.of(Pattern.compile("snapshotName(\\d+)")))
- .get().size(), 3);
- Assert.assertEquals(admin.listSnapshots(Optional.of(Pattern.compile("snapshotName[1|3]")))
- .get().size(), 2);
- Assert.assertEquals(admin.listSnapshots(Optional.of(Pattern.compile("snapshot(.*)"))).get()
- .size(), 3);
+ Assert.assertEquals(admin.listSnapshots(Pattern.compile("(.*)")).get().size(), 3);
+ Assert.assertEquals(admin.listSnapshots(Pattern.compile("snapshotName(\\d+)")).get().size(), 3);
+ Assert.assertEquals(admin.listSnapshots(Pattern.compile("snapshotName[1|3]")).get().size(), 2);
+ Assert.assertEquals(admin.listSnapshots(Pattern.compile("snapshot(.*)")).get().size(), 3);
Assert.assertEquals(
admin.listTableSnapshots(Pattern.compile("testListSnapshots"), Pattern.compile("s(.*)")).get()
.size(),
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncTableAdminApi.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncTableAdminApi.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncTableAdminApi.java
index 4df5947..7bbbd71 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncTableAdminApi.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestAsyncTableAdminApi.java
@@ -120,9 +120,9 @@ public class TestAsyncTableAdminApi extends TestAsyncAdminBase {
admin.deleteTable(tables[i]).join();
}
- tableDescs = admin.listTables(Optional.empty(), true).get();
+ tableDescs = admin.listTables(true).get();
assertTrue("Not found system tables", tableDescs.size() > 0);
- tableNames = admin.listTableNames(Optional.empty(), true).get();
+ tableNames = admin.listTableNames(true).get();
assertTrue("Not found system tables", tableNames.size() > 0);
}
@@ -169,7 +169,7 @@ public class TestAsyncTableAdminApi extends TestAsyncAdminBase {
assertEquals("Table should have only 1 region", 1, regionLocations.size());
final TableName tableName2 = TableName.valueOf(tableName.getNameAsString() + "_2");
- createTableWithDefaultConf(tableName2, Optional.of(new byte[][] { new byte[] { 42 } }));
+ createTableWithDefaultConf(tableName2, new byte[][] { new byte[] { 42 } });
regionLocations =
AsyncMetaTableAccessor.getTableHRegionLocations(metaTable, Optional.of(tableName2)).get();
assertEquals("Table should have only 2 region", 2, regionLocations.size());
@@ -208,7 +208,7 @@ public class TestAsyncTableAdminApi extends TestAsyncAdminBase {
new byte[] { 7, 7, 7 }, new byte[] { 8, 8, 8 }, new byte[] { 9, 9, 9 }, };
int expectedRegions = splitKeys.length + 1;
boolean tablesOnMaster = LoadBalancer.isTablesOnMaster(TEST_UTIL.getConfiguration());
- createTableWithDefaultConf(tableName, Optional.of(splitKeys));
+ createTableWithDefaultConf(tableName, splitKeys);
boolean tableAvailable = admin.isTableAvailable(tableName, splitKeys).get();
assertTrue("Table should be created with splitKyes + 1 rows in META", tableAvailable);
@@ -342,7 +342,7 @@ public class TestAsyncTableAdminApi extends TestAsyncAdminBase {
new byte[] { 3, 3, 3 }, new byte[] { 2, 2, 2 } };
final TableName tableName4 = TableName.valueOf(tableName.getNameAsString() + "_4");;
try {
- createTableWithDefaultConf(tableName4, Optional.of(splitKeys));
+ createTableWithDefaultConf(tableName4, splitKeys);
fail("Should not be able to create this table because of " + "duplicate split keys");
} catch (CompletionException e) {
assertTrue(e.getCause() instanceof IllegalArgumentException);
@@ -376,7 +376,7 @@ public class TestAsyncTableAdminApi extends TestAsyncAdminBase {
byte[][] splitKeys = new byte[1][];
splitKeys[0] = HConstants.EMPTY_BYTE_ARRAY;
try {
- createTableWithDefaultConf(tableName, Optional.of(splitKeys));
+ createTableWithDefaultConf(tableName, splitKeys);
fail("Test case should fail as empty split key is passed.");
} catch (CompletionException e) {
assertTrue(e.getCause() instanceof IllegalArgumentException);
@@ -390,7 +390,7 @@ public class TestAsyncTableAdminApi extends TestAsyncAdminBase {
splitKeys[1] = HConstants.EMPTY_BYTE_ARRAY;
splitKeys[2] = "region2".getBytes();
try {
- createTableWithDefaultConf(tableName, Optional.of(splitKeys));
+ createTableWithDefaultConf(tableName, splitKeys);
fail("Test case should fail as empty split key is passed.");
} catch (CompletionException e) {
assertTrue(e.getCause() instanceof IllegalArgumentException);
@@ -423,7 +423,7 @@ public class TestAsyncTableAdminApi extends TestAsyncAdminBase {
splitKeys[1] = Bytes.toBytes(8);
// Create & Fill the table
- createTableWithDefaultConf(tableName, Optional.of(splitKeys));
+ createTableWithDefaultConf(tableName, splitKeys);
RawAsyncTable table = ASYNC_CONN.getRawTable(tableName);
int expectedRows = 10;
for (int i = 0; i < expectedRows; i++) {
@@ -517,7 +517,7 @@ public class TestAsyncTableAdminApi extends TestAsyncAdminBase {
table1.get(get).get();
table2.get(get).get();
- admin.listTableNames(Optional.of(Pattern.compile(tableName.getNameAsString() + ".*")), false)
+ admin.listTableNames(Pattern.compile(tableName.getNameAsString() + ".*"), false)
.get().forEach(t -> admin.disableTable(t).join());
// Test that tables are disabled
@@ -541,7 +541,7 @@ public class TestAsyncTableAdminApi extends TestAsyncAdminBase {
assertEquals(TableState.State.DISABLED, getStateFromMeta(tableName1));
assertEquals(TableState.State.DISABLED, getStateFromMeta(tableName2));
- admin.listTableNames(Optional.of(Pattern.compile(tableName.getNameAsString() + ".*")), false)
+ admin.listTableNames(Pattern.compile(tableName.getNameAsString() + ".*"), false)
.get().forEach(t -> admin.enableTable(t).join());
// Test that tables are enabled
@@ -567,7 +567,7 @@ public class TestAsyncTableAdminApi extends TestAsyncAdminBase {
new byte[] { 4, 4, 4 }, new byte[] { 5, 5, 5 }, new byte[] { 6, 6, 6 },
new byte[] { 7, 7, 7 }, new byte[] { 8, 8, 8 }, new byte[] { 9, 9, 9 } };
int expectedRegions = splitKeys.length + 1;
- createTableWithDefaultConf(tableName, Optional.of(splitKeys));
+ createTableWithDefaultConf(tableName, splitKeys);
RawAsyncTable metaTable = ASYNC_CONN.getRawTable(META_TABLE_NAME);
List<HRegionLocation> regions =
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-server/src/test/java/org/apache/hadoop/hbase/coprocessor/TestMasterObserver.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/coprocessor/TestMasterObserver.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/coprocessor/TestMasterObserver.java
index 60b0260..d24711a 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/coprocessor/TestMasterObserver.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/coprocessor/TestMasterObserver.java
@@ -1543,7 +1543,7 @@ public class TestMasterObserver {
assertTrue("Found server", found);
LOG.info("Found " + destName);
master.getMasterRpcServices().moveRegion(null, RequestConverter.buildMoveRegionRequest(
- firstGoodPair.getRegionInfo().getEncodedNameAsBytes(),Bytes.toBytes(destName)));
+ firstGoodPair.getRegionInfo().getEncodedNameAsBytes(), ServerName.valueOf(destName)));
assertTrue("Coprocessor should have been called on region move",
cp.wasMoveCalled());
@@ -1565,11 +1565,12 @@ public class TestMasterObserver {
UTIL.waitUntilNoRegionsInTransition();
List<RegionInfo> openRegions = ProtobufUtil.getOnlineRegions(rs.getRSRpcServices());
int moveCnt = openRegions.size()/2;
- for (int i=0; i<moveCnt; i++) {
+ for (int i = 0; i < moveCnt; i++) {
RegionInfo info = openRegions.get(i);
if (!info.isMetaRegion()) {
- master.getMasterRpcServices().moveRegion(null, RequestConverter.buildMoveRegionRequest(
- openRegions.get(i).getEncodedNameAsBytes(), destRS));
+ master.getMasterRpcServices().moveRegion(null,
+ RequestConverter.buildMoveRegionRequest(openRegions.get(i).getEncodedNameAsBytes(),
+ ServerName.valueOf(Bytes.toString(destRS))));
}
}
//Make sure no regions are in transition now
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoad.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoad.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoad.java
index 53d9741..723b570 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoad.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoad.java
@@ -242,7 +242,7 @@ public class TestHRegionServerBulkLoad {
AdminProtos.AdminService.BlockingInterface server =
conn.getAdmin(getLocation().getServerName());
CompactRegionRequest request = RequestConverter.buildCompactRegionRequest(
- getLocation().getRegionInfo().getRegionName(), true, Optional.empty());
+ getLocation().getRegionInfo().getRegionName(), true, null);
server.compactRegion(null, request);
numCompactions.incrementAndGet();
return null;
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoadWithOldClient.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoadWithOldClient.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoadWithOldClient.java
index 7f486e4..da4b740 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoadWithOldClient.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoadWithOldClient.java
@@ -20,7 +20,6 @@ package org.apache.hadoop.hbase.regionserver;
import java.io.IOException;
import java.util.ArrayList;
import java.util.List;
-import java.util.Optional;
import java.util.concurrent.atomic.AtomicLong;
import org.apache.commons.logging.Log;
@@ -124,7 +123,7 @@ public class TestHRegionServerBulkLoadWithOldClient extends TestHRegionServerBul
conn.getAdmin(getLocation().getServerName());
CompactRegionRequest request =
RequestConverter.buildCompactRegionRequest(
- getLocation().getRegionInfo().getRegionName(), true, Optional.empty());
+ getLocation().getRegionInfo().getRegionName(), true, null);
server.compactRegion(null, request);
numCompactions.incrementAndGet();
return null;
[2/2] hbase git commit: HBASE-18950 Remove Optional parameters in
AsyncAdmin interface
Posted by zg...@apache.org.
HBASE-18950 Remove Optional parameters in AsyncAdmin interface
Project: http://git-wip-us.apache.org/repos/asf/hbase/repo
Commit: http://git-wip-us.apache.org/repos/asf/hbase/commit/888f2335
Tree: http://git-wip-us.apache.org/repos/asf/hbase/tree/888f2335
Diff: http://git-wip-us.apache.org/repos/asf/hbase/diff/888f2335
Branch: refs/heads/master
Commit: 888f2335c952040646ce820f6191f6433ec9411d
Parents: bc3f3ee
Author: Guanghao Zhang <zg...@apache.org>
Authored: Mon Oct 23 11:22:00 2017 +0800
Committer: Guanghao Zhang <zg...@apache.org>
Committed: Mon Nov 6 20:30:59 2017 +0800
----------------------------------------------------------------------
.../apache/hadoop/hbase/client/AsyncAdmin.java | 132 ++---
.../hadoop/hbase/client/AsyncHBaseAdmin.java | 114 ++++-
.../client/AsyncRpcRetryingCallerFactory.java | 8 +-
.../apache/hadoop/hbase/client/HBaseAdmin.java | 6 +-
.../hadoop/hbase/client/RawAsyncHBaseAdmin.java | 477 +++++++++++++------
.../hbase/shaded/protobuf/ProtobufUtil.java | 11 +-
.../hbase/shaded/protobuf/RequestConverter.java | 403 ++++++----------
...gionServerBulkLoadWithOldSecureEndpoint.java | 3 +-
.../hadoop/hbase/client/TestAsyncAdminBase.java | 18 +-
.../hbase/client/TestAsyncClusterAdminApi.java | 3 +-
.../hbase/client/TestAsyncRegionAdminApi.java | 23 +-
.../hbase/client/TestAsyncSnapshotAdminApi.java | 12 +-
.../hbase/client/TestAsyncTableAdminApi.java | 22 +-
.../hbase/coprocessor/TestMasterObserver.java | 9 +-
.../regionserver/TestHRegionServerBulkLoad.java | 2 +-
.../TestHRegionServerBulkLoadWithOldClient.java | 3 +-
16 files changed, 713 insertions(+), 533 deletions(-)
----------------------------------------------------------------------
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncAdmin.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncAdmin.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncAdmin.java
index 8fe02b9..baae6cf 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncAdmin.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncAdmin.java
@@ -64,38 +64,49 @@ public interface AsyncAdmin {
/**
* List all the userspace tables.
* @return - returns a list of TableDescriptors wrapped by a {@link CompletableFuture}.
- * @see #listTables(Optional, boolean)
*/
default CompletableFuture<List<TableDescriptor>> listTables() {
- return listTables(Optional.empty(), false);
+ return listTables(false);
}
/**
+ * List all the tables.
+ * @param includeSysTables False to match only against userspace tables
+ * @return - returns a list of TableDescriptors wrapped by a {@link CompletableFuture}.
+ */
+ CompletableFuture<List<TableDescriptor>> listTables(boolean includeSysTables);
+
+ /**
* List all the tables matching the given pattern.
* @param pattern The compiled regular expression to match against
* @param includeSysTables False to match only against userspace tables
* @return - returns a list of TableDescriptors wrapped by a {@link CompletableFuture}.
*/
- CompletableFuture<List<TableDescriptor>> listTables(Optional<Pattern> pattern,
- boolean includeSysTables);
+ CompletableFuture<List<TableDescriptor>> listTables(Pattern pattern, boolean includeSysTables);
/**
* List all of the names of userspace tables.
* @return a list of table names wrapped by a {@link CompletableFuture}.
- * @see #listTableNames(Optional, boolean)
+ * @see #listTableNames(Pattern, boolean)
*/
default CompletableFuture<List<TableName>> listTableNames() {
- return listTableNames(Optional.empty(), false);
+ return listTableNames(false);
}
/**
+ * List all of the names of tables.
+ * @param includeSysTables False to match only against userspace tables
+ * @return a list of table names wrapped by a {@link CompletableFuture}.
+ */
+ CompletableFuture<List<TableName>> listTableNames(boolean includeSysTables);
+
+ /**
* List all of the names of userspace tables.
* @param pattern The regular expression to match against
* @param includeSysTables False to match only against userspace tables
* @return a list of table names wrapped by a {@link CompletableFuture}.
*/
- CompletableFuture<List<TableName>> listTableNames(Optional<Pattern> pattern,
- boolean includeSysTables);
+ CompletableFuture<List<TableName>> listTableNames(Pattern pattern, boolean includeSysTables);
/**
* Method for getting the tableDescriptor
@@ -108,9 +119,7 @@ public interface AsyncAdmin {
* Creates a new table.
* @param desc table descriptor for table
*/
- default CompletableFuture<Void> createTable(TableDescriptor desc) {
- return createTable(desc, Optional.empty());
- }
+ CompletableFuture<Void> createTable(TableDescriptor desc);
/**
* Creates a new table with the specified number of regions. The start key specified will become
@@ -133,7 +142,7 @@ public interface AsyncAdmin {
* @param desc table descriptor for table
* @param splitKeys array of split keys for the initial regions of the table
*/
- CompletableFuture<Void> createTable(TableDescriptor desc, Optional<byte[][]> splitKeys);
+ CompletableFuture<Void> createTable(TableDescriptor desc, byte[][] splitKeys);
/**
* Deletes a table.
@@ -179,9 +188,7 @@ public interface AsyncAdmin {
* @return true if all regions of the table are available. The return value will be wrapped by a
* {@link CompletableFuture}.
*/
- default CompletableFuture<Boolean> isTableAvailable(TableName tableName) {
- return isTableAvailable(tableName, null);
- }
+ CompletableFuture<Boolean> isTableAvailable(TableName tableName);
/**
* Use this api to check if the table has been created with the specified number of splitkeys
@@ -274,9 +281,7 @@ public interface AsyncAdmin {
* was sent to HBase and may need some time to finish the compact operation.
* @param tableName table to compact
*/
- default CompletableFuture<Void> compact(TableName tableName) {
- return compact(tableName, Optional.empty());
- }
+ CompletableFuture<Void> compact(TableName tableName);
/**
* Compact a column family within a table. When the returned CompletableFuture is done, it only
@@ -286,16 +291,14 @@ public interface AsyncAdmin {
* @param columnFamily column family within a table. If not present, compact the table's all
* column families.
*/
- CompletableFuture<Void> compact(TableName tableName, Optional<byte[]> columnFamily);
+ CompletableFuture<Void> compact(TableName tableName, byte[] columnFamily);
/**
* Compact an individual region. When the returned CompletableFuture is done, it only means the
* compact request was sent to HBase and may need some time to finish the compact operation.
* @param regionName region to compact
*/
- default CompletableFuture<Void> compactRegion(byte[] regionName) {
- return compactRegion(regionName, Optional.empty());
- }
+ CompletableFuture<Void> compactRegion(byte[] regionName);
/**
* Compact a column family within a region. When the returned CompletableFuture is done, it only
@@ -305,16 +308,14 @@ public interface AsyncAdmin {
* @param columnFamily column family within a region. If not present, compact the region's all
* column families.
*/
- CompletableFuture<Void> compactRegion(byte[] regionName, Optional<byte[]> columnFamily);
+ CompletableFuture<Void> compactRegion(byte[] regionName, byte[] columnFamily);
/**
* Major compact a table. When the returned CompletableFuture is done, it only means the compact
* request was sent to HBase and may need some time to finish the compact operation.
* @param tableName table to major compact
*/
- default CompletableFuture<Void> majorCompact(TableName tableName) {
- return majorCompact(tableName, Optional.empty());
- }
+ CompletableFuture<Void> majorCompact(TableName tableName);
/**
* Major compact a column family within a table. When the returned CompletableFuture is done, it
@@ -324,16 +325,14 @@ public interface AsyncAdmin {
* @param columnFamily column family within a table. If not present, major compact the table's all
* column families.
*/
- CompletableFuture<Void> majorCompact(TableName tableName, Optional<byte[]> columnFamily);
+ CompletableFuture<Void> majorCompact(TableName tableName, byte[] columnFamily);
/**
* Major compact a region. When the returned CompletableFuture is done, it only means the compact
* request was sent to HBase and may need some time to finish the compact operation.
* @param regionName region to major compact
*/
- default CompletableFuture<Void> majorCompactRegion(byte[] regionName) {
- return majorCompactRegion(regionName, Optional.empty());
- }
+ CompletableFuture<Void> majorCompactRegion(byte[] regionName);
/**
* Major compact a column family within region. When the returned CompletableFuture is done, it
@@ -343,7 +342,7 @@ public interface AsyncAdmin {
* @param columnFamily column family within a region. If not present, major compact the region's
* all column families.
*/
- CompletableFuture<Void> majorCompactRegion(byte[] regionName, Optional<byte[]> columnFamily);
+ CompletableFuture<Void> majorCompactRegion(byte[] regionName, byte[] columnFamily);
/**
* Compact all regions on the region server.
@@ -405,9 +404,7 @@ public interface AsyncAdmin {
* Split an individual region.
* @param regionName region to split
*/
- default CompletableFuture<Void> splitRegion(byte[] regionName) {
- return splitRegion(regionName, Optional.empty());
- }
+ CompletableFuture<Void> splitRegion(byte[] regionName);
/**
* Split a table.
@@ -422,7 +419,7 @@ public interface AsyncAdmin {
* @param splitPoint the explicit position to split on. If not present, it will decide by region
* server.
*/
- CompletableFuture<Void> splitRegion(byte[] regionName, Optional<byte[]> splitPoint);
+ CompletableFuture<Void> splitRegion(byte[] regionName, byte[] splitPoint);
/**
* @param regionName Encoded or full name of region to assign.
@@ -432,7 +429,7 @@ public interface AsyncAdmin {
/**
* Unassign a region from current hosting regionserver. Region will then be assigned to a
* regionserver chosen at random. Region could be reassigned back to the same server. Use
- * {@link #move(byte[], Optional)} if you want to control the region movement.
+ * {@link #move(byte[], ServerName)} if you want to control the region movement.
* @param regionName Encoded or full name of region to unassign. Will clear any existing
* RegionPlan if one found.
* @param forcible If true, force unassign (Will remove region from regions-in-transition too if
@@ -452,13 +449,19 @@ public interface AsyncAdmin {
CompletableFuture<Void> offline(byte[] regionName);
/**
+ * Move the region <code>r</code> to a random server.
+ * @param regionName Encoded or full name of region to move.
+ */
+ CompletableFuture<Void> move(byte[] regionName);
+
+ /**
* Move the region <code>r</code> to <code>dest</code>.
* @param regionName Encoded or full name of region to move.
* @param destServerName The servername of the destination regionserver. If not present, we'll
* assign to a random server. A server name is made of host, port and startcode. Here is
* an example: <code> host187.example.com,60020,1289493121758</code>
*/
- CompletableFuture<Void> move(byte[] regionName, Optional<ServerName> destServerName);
+ CompletableFuture<Void> move(byte[] regionName, ServerName destServerName);
/**
* Apply the new quota settings.
@@ -535,9 +538,7 @@ public interface AsyncAdmin {
* @return a list of replication peers description. The return value will be wrapped by a
* {@link CompletableFuture}.
*/
- default CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers() {
- return listReplicationPeers(Optional.empty());
- }
+ CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers();
/**
* Return a list of replication peers.
@@ -545,8 +546,7 @@ public interface AsyncAdmin {
* @return a list of replication peers description. The return value will be wrapped by a
* {@link CompletableFuture}.
*/
- CompletableFuture<List<ReplicationPeerDescription>>
- listReplicationPeers(Optional<Pattern> pattern);
+ CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers(Pattern pattern);
/**
* Find all table and column families that are replicated from this cluster
@@ -652,16 +652,22 @@ public interface AsyncAdmin {
* @return a list of snapshot descriptors for completed snapshots wrapped by a
* {@link CompletableFuture}
*/
- default CompletableFuture<List<SnapshotDescription>> listSnapshots() {
- return listSnapshots(Optional.empty());
- }
+ CompletableFuture<List<SnapshotDescription>> listSnapshots();
/**
* List all the completed snapshots matching the given pattern.
* @param pattern The compiled regular expression to match against
* @return - returns a List of SnapshotDescription wrapped by a {@link CompletableFuture}
*/
- CompletableFuture<List<SnapshotDescription>> listSnapshots(Optional<Pattern> pattern);
+ CompletableFuture<List<SnapshotDescription>> listSnapshots(Pattern pattern);
+
+ /**
+ * List all the completed snapshots matching the given table name pattern.
+ * @param tableNamePattern The compiled table name regular expression to match against
+ * @return - returns a List of completed SnapshotDescription wrapped by a
+ * {@link CompletableFuture}
+ */
+ CompletableFuture<List<SnapshotDescription>> listTableSnapshots(Pattern tableNamePattern);
/**
* List all the completed snapshots matching the given table name regular expression and snapshot
@@ -681,12 +687,21 @@ public interface AsyncAdmin {
CompletableFuture<Void> deleteSnapshot(String snapshotName);
/**
+ * Delete all existing snapshots.
+ */
+ CompletableFuture<Void> deleteSnapshots();
+
+ /**
* Delete existing snapshots whose names match the pattern passed.
* @param pattern pattern for names of the snapshot to match
*/
- default CompletableFuture<Void> deleteSnapshots(Pattern pattern) {
- return deleteTableSnapshots(null, pattern);
- }
+ CompletableFuture<Void> deleteSnapshots(Pattern pattern);
+
+ /**
+ * Delete all existing snapshots matching the given table name pattern.
+ * @param tableNamePattern The compiled table name regular expression to match against
+ */
+ CompletableFuture<Void> deleteTableSnapshots(Pattern tableNamePattern);
/**
* Delete all existing snapshots matching the given table name regular expression and snapshot
@@ -823,15 +838,6 @@ public interface AsyncAdmin {
}
/**
- * Get a list of {@link RegionLoad} of all regions hosted on a region seerver.
- * @param serverName
- * @return a list of {@link RegionLoad} wrapped by {@link CompletableFuture}
- */
- default CompletableFuture<List<RegionLoad>> getRegionLoads(ServerName serverName) {
- return getRegionLoads(serverName, Optional.empty());
- }
-
- /**
* Shuts down the HBase cluster.
*/
CompletableFuture<Void> shutdown();
@@ -878,13 +884,19 @@ public interface AsyncAdmin {
CompletableFuture<Void> clearCompactionQueues(ServerName serverName, Set<String> queues);
/**
+ * Get a list of {@link RegionLoad} of all regions hosted on a region seerver.
+ * @param serverName
+ * @return a list of {@link RegionLoad} wrapped by {@link CompletableFuture}
+ */
+ CompletableFuture<List<RegionLoad>> getRegionLoads(ServerName serverName);
+
+ /**
* Get a list of {@link RegionLoad} of all regions hosted on a region seerver for a table.
* @param serverName
* @param tableName
* @return a list of {@link RegionLoad} wrapped by {@link CompletableFuture}
*/
- CompletableFuture<List<RegionLoad>> getRegionLoads(ServerName serverName,
- Optional<TableName> tableName);
+ CompletableFuture<List<RegionLoad>> getRegionLoads(ServerName serverName, TableName tableName);
/**
* Check whether master is in maintenance mode
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncHBaseAdmin.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncHBaseAdmin.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncHBaseAdmin.java
index 23dea81..04005eb 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncHBaseAdmin.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncHBaseAdmin.java
@@ -43,7 +43,6 @@ import org.apache.hadoop.hbase.quotas.QuotaFilter;
import org.apache.hadoop.hbase.quotas.QuotaSettings;
import org.apache.hadoop.hbase.replication.ReplicationPeerConfig;
import org.apache.hadoop.hbase.replication.ReplicationPeerDescription;
-import org.apache.hadoop.hbase.util.Pair;
import org.apache.yetus.audience.InterfaceAudience;
import com.google.protobuf.RpcChannel;
@@ -84,13 +83,23 @@ public class AsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<List<TableDescriptor>> listTables(Optional<Pattern> pattern,
+ public CompletableFuture<List<TableDescriptor>> listTables(boolean includeSysTables) {
+ return wrap(rawAdmin.listTables(includeSysTables));
+ }
+
+ @Override
+ public CompletableFuture<List<TableDescriptor>> listTables(Pattern pattern,
boolean includeSysTables) {
return wrap(rawAdmin.listTables(pattern, includeSysTables));
}
@Override
- public CompletableFuture<List<TableName>> listTableNames(Optional<Pattern> pattern,
+ public CompletableFuture<List<TableName>> listTableNames(boolean includeSysTables) {
+ return wrap(rawAdmin.listTableNames(includeSysTables));
+ }
+
+ @Override
+ public CompletableFuture<List<TableName>> listTableNames(Pattern pattern,
boolean includeSysTables) {
return wrap(rawAdmin.listTableNames(pattern, includeSysTables));
}
@@ -101,13 +110,18 @@ public class AsyncHBaseAdmin implements AsyncAdmin {
}
@Override
+ public CompletableFuture<Void> createTable(TableDescriptor desc) {
+ return wrap(rawAdmin.createTable(desc));
+ }
+
+ @Override
public CompletableFuture<Void> createTable(TableDescriptor desc, byte[] startKey, byte[] endKey,
int numRegions) {
return wrap(rawAdmin.createTable(desc, startKey, endKey, numRegions));
}
@Override
- public CompletableFuture<Void> createTable(TableDescriptor desc, Optional<byte[][]> splitKeys) {
+ public CompletableFuture<Void> createTable(TableDescriptor desc, byte[][] splitKeys) {
return wrap(rawAdmin.createTable(desc, splitKeys));
}
@@ -142,6 +156,11 @@ public class AsyncHBaseAdmin implements AsyncAdmin {
}
@Override
+ public CompletableFuture<Boolean> isTableAvailable(TableName tableName) {
+ return wrap(rawAdmin.isTableAvailable(tableName));
+ }
+
+ @Override
public CompletableFuture<Boolean> isTableAvailable(TableName tableName, byte[][] splitKeys) {
return wrap(rawAdmin.isTableAvailable(tableName, splitKeys));
}
@@ -209,23 +228,42 @@ public class AsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<Void> compact(TableName tableName, Optional<byte[]> columnFamily) {
+ public CompletableFuture<Void> compact(TableName tableName) {
+ return wrap(rawAdmin.compact(tableName));
+ }
+
+ @Override
+ public CompletableFuture<Void> compact(TableName tableName, byte[] columnFamily) {
return wrap(rawAdmin.compact(tableName, columnFamily));
}
@Override
- public CompletableFuture<Void> compactRegion(byte[] regionName, Optional<byte[]> columnFamily) {
+ public CompletableFuture<Void> compactRegion(byte[] regionName) {
+ return wrap(rawAdmin.compactRegion(regionName));
+ }
+
+ @Override
+ public CompletableFuture<Void> compactRegion(byte[] regionName, byte[] columnFamily) {
return wrap(rawAdmin.compactRegion(regionName, columnFamily));
}
@Override
- public CompletableFuture<Void> majorCompact(TableName tableName, Optional<byte[]> columnFamily) {
+ public CompletableFuture<Void> majorCompact(TableName tableName) {
+ return wrap(rawAdmin.majorCompact(tableName));
+ }
+
+ @Override
+ public CompletableFuture<Void> majorCompact(TableName tableName, byte[] columnFamily) {
return wrap(rawAdmin.majorCompact(tableName, columnFamily));
}
@Override
- public CompletableFuture<Void>
- majorCompactRegion(byte[] regionName, Optional<byte[]> columnFamily) {
+ public CompletableFuture<Void> majorCompactRegion(byte[] regionName) {
+ return wrap(rawAdmin.majorCompactRegion(regionName));
+ }
+
+ @Override
+ public CompletableFuture<Void> majorCompactRegion(byte[] regionName, byte[] columnFamily) {
return wrap(rawAdmin.majorCompactRegion(regionName, columnFamily));
}
@@ -276,7 +314,12 @@ public class AsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<Void> splitRegion(byte[] regionName, Optional<byte[]> splitPoint) {
+ public CompletableFuture<Void> splitRegion(byte[] regionName) {
+ return wrap(rawAdmin.splitRegion(regionName));
+ }
+
+ @Override
+ public CompletableFuture<Void> splitRegion(byte[] regionName, byte[] splitPoint) {
return wrap(rawAdmin.splitRegion(regionName, splitPoint));
}
@@ -296,7 +339,12 @@ public class AsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<Void> move(byte[] regionName, Optional<ServerName> destServerName) {
+ public CompletableFuture<Void> move(byte[] regionName) {
+ return wrap(rawAdmin.move(regionName));
+ }
+
+ @Override
+ public CompletableFuture<Void> move(byte[] regionName, ServerName destServerName) {
return wrap(rawAdmin.move(regionName, destServerName));
}
@@ -355,8 +403,12 @@ public class AsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers(
- Optional<Pattern> pattern) {
+ public CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers() {
+ return wrap(rawAdmin.listReplicationPeers());
+ }
+
+ @Override
+ public CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers(Pattern pattern) {
return wrap(rawAdmin.listReplicationPeers(pattern));
}
@@ -391,11 +443,21 @@ public class AsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<List<SnapshotDescription>> listSnapshots(Optional<Pattern> pattern) {
+ public CompletableFuture<List<SnapshotDescription>> listSnapshots() {
+ return wrap(rawAdmin.listSnapshots());
+ }
+
+ @Override
+ public CompletableFuture<List<SnapshotDescription>> listSnapshots(Pattern pattern) {
return wrap(rawAdmin.listSnapshots(pattern));
}
@Override
+ public CompletableFuture<List<SnapshotDescription>> listTableSnapshots(Pattern tableNamePattern) {
+ return wrap(rawAdmin.listTableSnapshots(tableNamePattern));
+ }
+
+ @Override
public CompletableFuture<List<SnapshotDescription>> listTableSnapshots(Pattern tableNamePattern,
Pattern snapshotNamePattern) {
return wrap(rawAdmin.listTableSnapshots(tableNamePattern, snapshotNamePattern));
@@ -407,6 +469,21 @@ public class AsyncHBaseAdmin implements AsyncAdmin {
}
@Override
+ public CompletableFuture<Void> deleteSnapshots() {
+ return wrap(rawAdmin.deleteSnapshots());
+ }
+
+ @Override
+ public CompletableFuture<Void> deleteSnapshots(Pattern pattern) {
+ return wrap(rawAdmin.deleteSnapshots(pattern));
+ }
+
+ @Override
+ public CompletableFuture<Void> deleteTableSnapshots(Pattern tableNamePattern) {
+ return wrap(rawAdmin.deleteTableSnapshots(tableNamePattern));
+ }
+
+ @Override
public CompletableFuture<Void> deleteTableSnapshots(Pattern tableNamePattern,
Pattern snapshotNamePattern) {
return wrap(rawAdmin.deleteTableSnapshots(tableNamePattern, snapshotNamePattern));
@@ -513,8 +590,13 @@ public class AsyncHBaseAdmin implements AsyncAdmin {
}
@Override
+ public CompletableFuture<List<RegionLoad>> getRegionLoads(ServerName serverName) {
+ return wrap(rawAdmin.getRegionLoads(serverName));
+ }
+
+ @Override
public CompletableFuture<List<RegionLoad>> getRegionLoads(ServerName serverName,
- Optional<TableName> tableName) {
+ TableName tableName) {
return wrap(rawAdmin.getRegionLoads(serverName, tableName));
}
@@ -625,4 +707,4 @@ public class AsyncHBaseAdmin implements AsyncAdmin {
public CompletableFuture<List<ServerName>> clearDeadServers(List<ServerName> servers) {
return wrap(rawAdmin.clearDeadServers(servers));
}
-}
+}
\ No newline at end of file
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncRpcRetryingCallerFactory.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncRpcRetryingCallerFactory.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncRpcRetryingCallerFactory.java
index b687668..9c45883 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncRpcRetryingCallerFactory.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/AsyncRpcRetryingCallerFactory.java
@@ -427,8 +427,8 @@ class AsyncRpcRetryingCallerFactory {
public AsyncAdminRequestRetryingCaller<T> build() {
return new AsyncAdminRequestRetryingCaller<T>(retryTimer, conn, pauseNs, maxAttempts,
- operationTimeoutNs, rpcTimeoutNs, startLogErrorsCnt, serverName, checkNotNull(callable,
- "action is null"));
+ operationTimeoutNs, rpcTimeoutNs, startLogErrorsCnt, checkNotNull(serverName,
+ "serverName is null"), checkNotNull(callable, "action is null"));
}
public CompletableFuture<T> call() {
@@ -488,8 +488,8 @@ class AsyncRpcRetryingCallerFactory {
public AsyncServerRequestRpcRetryingCaller<T> build() {
return new AsyncServerRequestRpcRetryingCaller<T>(retryTimer, conn, pauseNs, maxAttempts,
- operationTimeoutNs, rpcTimeoutNs, startLogErrorsCnt, serverName, checkNotNull(callable,
- "action is null"));
+ operationTimeoutNs, rpcTimeoutNs, startLogErrorsCnt, checkNotNull(serverName,
+ "serverName is null"), checkNotNull(callable, "action is null"));
}
public CompletableFuture<T> call() {
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HBaseAdmin.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HBaseAdmin.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HBaseAdmin.java
index c090325..556e564 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HBaseAdmin.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HBaseAdmin.java
@@ -1371,14 +1371,14 @@ public class HBaseAdmin implements Admin {
}
@Override
- public void move(final byte [] encodedRegionName, final byte [] destServerName)
- throws IOException {
+ public void move(final byte[] encodedRegionName, final byte[] destServerName) throws IOException {
executeCallable(new MasterCallable<Void>(getConnection(), getRpcControllerFactory()) {
@Override
protected Void rpcCall() throws Exception {
setPriority(encodedRegionName);
MoveRegionRequest request =
- RequestConverter.buildMoveRegionRequest(encodedRegionName, destServerName);
+ RequestConverter.buildMoveRegionRequest(encodedRegionName,
+ destServerName != null ? ServerName.valueOf(Bytes.toString(destServerName)) : null);
master.moveRegion(getRpcController(), request);
return null;
}
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-client/src/main/java/org/apache/hadoop/hbase/client/RawAsyncHBaseAdmin.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/RawAsyncHBaseAdmin.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/RawAsyncHBaseAdmin.java
index 1622497..1d80797 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/RawAsyncHBaseAdmin.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/RawAsyncHBaseAdmin.java
@@ -26,7 +26,6 @@ import java.util.Collection;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashMap;
-import java.util.LinkedList;
import java.util.List;
import java.util.Map;
import java.util.Optional;
@@ -82,9 +81,7 @@ import org.apache.hadoop.hbase.snapshot.SnapshotCreationException;
import org.apache.hadoop.hbase.util.Bytes;
import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
import org.apache.hadoop.hbase.util.ForeignExceptionUtil;
-import org.apache.hadoop.hbase.util.Pair;
import org.apache.yetus.audience.InterfaceAudience;
-
import org.apache.hadoop.hbase.shaded.com.google.protobuf.RpcCallback;
import org.apache.hadoop.hbase.shaded.io.netty.util.Timeout;
import org.apache.hadoop.hbase.shaded.io.netty.util.TimerTask;
@@ -245,6 +242,7 @@ import org.apache.hadoop.hbase.shaded.protobuf.generated.ReplicationProtos.Updat
import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos;
import com.google.common.annotations.VisibleForTesting;
+import com.google.common.base.Preconditions;
import com.google.protobuf.Message;
import com.google.protobuf.RpcChannel;
@@ -385,28 +383,54 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<List<TableDescriptor>> listTables(Optional<Pattern> pattern,
+ public CompletableFuture<List<TableDescriptor>> listTables(boolean includeSysTables) {
+ return getTableDescriptors(RequestConverter.buildGetTableDescriptorsRequest(null,
+ includeSysTables));
+ }
+
+ /**
+ * {@link #listTables(boolean)}
+ */
+ @Override
+ public CompletableFuture<List<TableDescriptor>> listTables(Pattern pattern,
boolean includeSysTables) {
+ Preconditions.checkNotNull(pattern,
+ "pattern is null. If you don't specify a pattern, use listTables(boolean) instead");
+ return getTableDescriptors(RequestConverter.buildGetTableDescriptorsRequest(pattern,
+ includeSysTables));
+ }
+
+ private CompletableFuture<List<TableDescriptor>>
+ getTableDescriptors(GetTableDescriptorsRequest request) {
return this.<List<TableDescriptor>> newMasterCaller()
.action((controller, stub) -> this
.<GetTableDescriptorsRequest, GetTableDescriptorsResponse, List<TableDescriptor>> call(
- controller, stub,
- RequestConverter.buildGetTableDescriptorsRequest(pattern, includeSysTables),
- (s, c, req, done) -> s.getTableDescriptors(c, req, done),
+ controller, stub, request, (s, c, req, done) -> s.getTableDescriptors(c, req, done),
(resp) -> ProtobufUtil.toTableDescriptorList(resp)))
.call();
}
@Override
- public CompletableFuture<List<TableName>> listTableNames(Optional<Pattern> pattern,
- boolean includeSysTables) {
- return this.<List<TableName>> newMasterCaller()
- .action((controller, stub) -> this
- .<GetTableNamesRequest, GetTableNamesResponse, List<TableName>> call(controller, stub,
- RequestConverter.buildGetTableNamesRequest(pattern, includeSysTables),
- (s, c, req, done) -> s.getTableNames(c, req, done),
- (resp) -> ProtobufUtil.toTableNameList(resp.getTableNamesList())))
- .call();
+ public CompletableFuture<List<TableName>> listTableNames(boolean includeSysTables) {
+ return getTableNames(RequestConverter.buildGetTableNamesRequest(null, includeSysTables));
+ }
+
+ @Override
+ public CompletableFuture<List<TableName>>
+ listTableNames(Pattern pattern, boolean includeSysTables) {
+ Preconditions.checkNotNull(pattern,
+ "pattern is null. If you don't specify a pattern, use listTableNames(boolean) instead");
+ return getTableNames(RequestConverter.buildGetTableNamesRequest(pattern, includeSysTables));
+ }
+
+ private CompletableFuture<List<TableName>> getTableNames(GetTableNamesRequest request) {
+ return this
+ .<List<TableName>> newMasterCaller()
+ .action(
+ (controller, stub) -> this
+ .<GetTableNamesRequest, GetTableNamesResponse, List<TableName>> call(controller,
+ stub, request, (s, c, req, done) -> s.getTableNames(c, req, done),
+ (resp) -> ProtobufUtil.toTableNameList(resp.getTableNamesList()))).call();
}
@Override
@@ -433,31 +457,41 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
}
@Override
+ public CompletableFuture<Void> createTable(TableDescriptor desc) {
+ return createTable(desc.getTableName(),
+ RequestConverter.buildCreateTableRequest(desc, null, ng.getNonceGroup(), ng.newNonce()));
+ }
+
+ @Override
public CompletableFuture<Void> createTable(TableDescriptor desc, byte[] startKey, byte[] endKey,
int numRegions) {
try {
- return createTable(desc, Optional.of(getSplitKeys(startKey, endKey, numRegions)));
+ return createTable(desc, getSplitKeys(startKey, endKey, numRegions));
} catch (IllegalArgumentException e) {
return failedFuture(e);
}
}
@Override
- public CompletableFuture<Void> createTable(TableDescriptor desc, Optional<byte[][]> splitKeys) {
- if (desc.getTableName() == null) {
- return failedFuture(new IllegalArgumentException("TableName cannot be null"));
- }
+ public CompletableFuture<Void> createTable(TableDescriptor desc, byte[][] splitKeys) {
+ Preconditions.checkNotNull(splitKeys, "splitKeys is null. If you don't specify splitKeys,"
+ + " use createTable(TableDescriptor) instead");
try {
- splitKeys.ifPresent(keys -> verifySplitKeys(keys));
- return this.<CreateTableRequest, CreateTableResponse> procedureCall(RequestConverter
- .buildCreateTableRequest(desc, splitKeys, ng.getNonceGroup(), ng.newNonce()), (s, c, req,
- done) -> s.createTable(c, req, done), (resp) -> resp.getProcId(),
- new CreateTableProcedureBiConsumer(this, desc.getTableName()));
+ verifySplitKeys(splitKeys);
+ return createTable(desc.getTableName(), RequestConverter.buildCreateTableRequest(desc,
+ splitKeys, ng.getNonceGroup(), ng.newNonce()));
} catch (IllegalArgumentException e) {
return failedFuture(e);
}
}
+ private CompletableFuture<Void> createTable(TableName tableName, CreateTableRequest request) {
+ Preconditions.checkNotNull(tableName, "table name is null");
+ return this.<CreateTableRequest, CreateTableResponse> procedureCall(request,
+ (s, c, req, done) -> s.createTable(c, req, done), (resp) -> resp.getProcId(),
+ new CreateTableProcedureBiConsumer(this, tableName));
+ }
+
@Override
public CompletableFuture<Void> deleteTable(TableName tableName) {
return this.<DeleteTableRequest, DeleteTableResponse> procedureCall(RequestConverter
@@ -526,11 +560,18 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
@Override
public CompletableFuture<Boolean> isTableAvailable(TableName tableName) {
- return isTableAvailable(tableName, null);
+ return isTableAvailable(tableName, Optional.empty());
}
@Override
public CompletableFuture<Boolean> isTableAvailable(TableName tableName, byte[][] splitKeys) {
+ Preconditions.checkNotNull(splitKeys, "splitKeys is null. If you don't specify splitKeys,"
+ + " use isTableAvailable(TableName) instead");
+ return isTableAvailable(tableName, Optional.of(splitKeys));
+ }
+
+ private CompletableFuture<Boolean> isTableAvailable(TableName tableName,
+ Optional<byte[][]> splitKeys) {
CompletableFuture<Boolean> future = new CompletableFuture<>();
isTableEnabled(tableName).whenComplete(
(enabled, error) -> {
@@ -548,54 +589,46 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
future.completeExceptionally(error1);
return;
}
- int notDeployed = 0;
- int regionCount = 0;
- for (HRegionLocation location : locations) {
- RegionInfo info = location.getRegionInfo();
- if (location.getServerName() == null) {
- if (LOG.isDebugEnabled()) {
- LOG.debug("Table " + tableName + " has not deployed region "
- + info.getEncodedName());
- }
- notDeployed++;
- } else if (splitKeys != null
- && !Bytes.equals(info.getStartKey(), HConstants.EMPTY_BYTE_ARRAY)) {
- for (byte[] splitKey : splitKeys) {
- // Just check if the splitkey is available
- if (Bytes.equals(info.getStartKey(), splitKey)) {
- regionCount++;
- break;
- }
- }
- } else {
- // Always empty start row should be counted
- regionCount++;
- }
- }
- if (notDeployed > 0) {
- if (LOG.isDebugEnabled()) {
- LOG.debug("Table " + tableName + " has " + notDeployed + " regions");
- }
- future.complete(false);
- } else if (splitKeys != null && regionCount != splitKeys.length + 1) {
+ List<HRegionLocation> notDeployedRegions =
+ locations.stream().filter(loc -> loc.getServerName() == null)
+ .collect(Collectors.toList());
+ if (notDeployedRegions.size() > 0) {
if (LOG.isDebugEnabled()) {
- LOG.debug("Table " + tableName + " expected to have "
- + (splitKeys.length + 1) + " regions, but only " + regionCount
- + " available");
+ LOG.debug("Table " + tableName + " has " + notDeployedRegions.size()
+ + " regions");
}
future.complete(false);
- } else {
- if (LOG.isDebugEnabled()) {
- LOG.debug("Table " + tableName + " should be available");
- }
- future.complete(true);
+ return;
}
+
+ Optional<Boolean> available =
+ splitKeys.map(keys -> compareRegionsWithSplitKeys(locations, keys));
+ future.complete(available.orElse(true));
});
}
});
return future;
}
+ private boolean compareRegionsWithSplitKeys(List<HRegionLocation> locations, byte[][] splitKeys) {
+ int regionCount = 0;
+ for (HRegionLocation location : locations) {
+ RegionInfo info = location.getRegion();
+ if (Bytes.equals(info.getStartKey(), HConstants.EMPTY_BYTE_ARRAY)) {
+ regionCount++;
+ continue;
+ }
+ for (byte[] splitKey : splitKeys) {
+ // Just check if the splitkey is available
+ if (Bytes.equals(info.getStartKey(), splitKey)) {
+ regionCount++;
+ break;
+ }
+ }
+ }
+ return regionCount == splitKeys.length + 1;
+ }
+
@Override
public CompletableFuture<Void> addColumnFamily(TableName tableName, ColumnFamilyDescriptor columnFamily) {
return this.<AddColumnRequest, AddColumnResponse> procedureCall(
@@ -757,22 +790,50 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<Void> compact(TableName tableName, Optional<byte[]> columnFamily) {
+ public CompletableFuture<Void> compact(TableName tableName) {
+ return compact(tableName, null, false, CompactType.NORMAL);
+ }
+
+ @Override
+ public CompletableFuture<Void> compact(TableName tableName, byte[] columnFamily) {
+ Preconditions.checkNotNull(columnFamily,
+ "columnFamily is null. If you don't specify a columnFamily, use compact(TableName) instead");
return compact(tableName, columnFamily, false, CompactType.NORMAL);
}
@Override
- public CompletableFuture<Void> compactRegion(byte[] regionName, Optional<byte[]> columnFamily) {
+ public CompletableFuture<Void> compactRegion(byte[] regionName) {
+ return compactRegion(regionName, null, false);
+ }
+
+ @Override
+ public CompletableFuture<Void> compactRegion(byte[] regionName, byte[] columnFamily) {
+ Preconditions.checkNotNull(columnFamily, "columnFamily is null."
+ + " If you don't specify a columnFamily, use compactRegion(regionName) instead");
return compactRegion(regionName, columnFamily, false);
}
@Override
- public CompletableFuture<Void> majorCompact(TableName tableName, Optional<byte[]> columnFamily) {
+ public CompletableFuture<Void> majorCompact(TableName tableName) {
+ return compact(tableName, null, true, CompactType.NORMAL);
+ }
+
+ @Override
+ public CompletableFuture<Void> majorCompact(TableName tableName, byte[] columnFamily) {
+ Preconditions.checkNotNull(columnFamily, "columnFamily is null."
+ + " If you don't specify a columnFamily, use majorCompact(TableName) instead");
return compact(tableName, columnFamily, true, CompactType.NORMAL);
}
@Override
- public CompletableFuture<Void> majorCompactRegion(byte[] regionName, Optional<byte[]> columnFamily) {
+ public CompletableFuture<Void> majorCompactRegion(byte[] regionName) {
+ return compactRegion(regionName, null, true);
+ }
+
+ @Override
+ public CompletableFuture<Void> majorCompactRegion(byte[] regionName, byte[] columnFamily) {
+ Preconditions.checkNotNull(columnFamily, "columnFamily is null."
+ + " If you don't specify a columnFamily, use majorCompactRegion(regionName) instead");
return compactRegion(regionName, columnFamily, true);
}
@@ -795,7 +856,7 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
}
List<CompletableFuture<Void>> compactFutures = new ArrayList<>();
if (hRegionInfos != null) {
- hRegionInfos.forEach(region -> compactFutures.add(compact(sn, region, major, Optional.empty())));
+ hRegionInfos.forEach(region -> compactFutures.add(compact(sn, region, major, null)));
}
CompletableFuture
.allOf(compactFutures.toArray(new CompletableFuture<?>[compactFutures.size()]))
@@ -810,7 +871,7 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
return future;
}
- private CompletableFuture<Void> compactRegion(byte[] regionName, Optional<byte[]> columnFamily,
+ private CompletableFuture<Void> compactRegion(byte[] regionName, byte[] columnFamily,
boolean major) {
CompletableFuture<Void> future = new CompletableFuture<>();
getRegionLocation(regionName).whenComplete(
@@ -868,7 +929,7 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
/**
* Compact column family of a table, Asynchronous operation even if CompletableFuture.get()
*/
- private CompletableFuture<Void> compact(final TableName tableName, Optional<byte[]> columnFamily,
+ private CompletableFuture<Void> compact(final TableName tableName, byte[] columnFamily,
final boolean major, CompactType compactType) {
if (CompactType.MOB.equals(compactType)) {
// TODO support MOB compact.
@@ -905,7 +966,7 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
* Compact the region at specific region server.
*/
private CompletableFuture<Void> compact(final ServerName sn, final RegionInfo hri,
- final boolean major, Optional<byte[]> columnFamily) {
+ final boolean major, byte[] columnFamily) {
return this
.<Void> newAdminCaller()
.serverName(sn)
@@ -1078,7 +1139,7 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
if (hri == null || hri.isSplitParent()
|| hri.getReplicaId() != RegionInfo.DEFAULT_REPLICA_ID)
continue;
- splitFutures.add(split(hri, Optional.empty()));
+ splitFutures.add(split(hri, null));
}
}
}
@@ -1114,7 +1175,7 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
result.completeExceptionally(new IllegalArgumentException(
"Region does not found: rowKey=" + Bytes.toStringBinary(splitPoint)));
} else {
- splitRegion(loc.getRegionInfo().getRegionName(), Optional.of(splitPoint))
+ splitRegion(loc.getRegionInfo().getRegionName(), splitPoint)
.whenComplete((ret, err2) -> {
if (err2 != null) {
result.completeExceptionally(err2);
@@ -1129,11 +1190,11 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<Void> splitRegion(byte[] regionName, Optional<byte[]> splitPoint) {
+ public CompletableFuture<Void> splitRegion(byte[] regionName) {
CompletableFuture<Void> future = new CompletableFuture<>();
getRegionLocation(regionName).whenComplete(
(location, err) -> {
- RegionInfo regionInfo = location.getRegionInfo();
+ RegionInfo regionInfo = location.getRegion();
if (regionInfo.getReplicaId() != RegionInfo.DEFAULT_REPLICA_ID) {
future.completeExceptionally(new IllegalArgumentException(
"Can't split replicas directly. "
@@ -1146,7 +1207,7 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
.toStringBinary(regionName)));
return;
}
- split(regionInfo, splitPoint).whenComplete((ret, err2) -> {
+ split(regionInfo, null).whenComplete((ret, err2) -> {
if (err2 != null) {
future.completeExceptionally(err2);
} else {
@@ -1157,20 +1218,50 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
return future;
}
- private CompletableFuture<Void> split(final RegionInfo hri,
- Optional<byte[]> splitPoint) {
- if (hri.getStartKey() != null && splitPoint.isPresent()
- && Bytes.compareTo(hri.getStartKey(), splitPoint.get()) == 0) {
- return failedFuture(new IllegalArgumentException(
- "should not give a splitkey which equals to startkey!"));
- }
+ @Override
+ public CompletableFuture<Void> splitRegion(byte[] regionName, byte[] splitPoint) {
+ Preconditions.checkNotNull(splitPoint,
+ "splitPoint is null. If you don't specify a splitPoint, use splitRegion(byte[]) instead");
+ CompletableFuture<Void> future = new CompletableFuture<>();
+ getRegionLocation(regionName).whenComplete(
+ (location, err) -> {
+ RegionInfo regionInfo = location.getRegion();
+ if (regionInfo.getReplicaId() != RegionInfo.DEFAULT_REPLICA_ID) {
+ future.completeExceptionally(new IllegalArgumentException(
+ "Can't split replicas directly. "
+ + "Replicas are auto-split when their primary is split."));
+ return;
+ }
+ ServerName serverName = location.getServerName();
+ if (serverName == null) {
+ future.completeExceptionally(new NoServerForRegionException(Bytes
+ .toStringBinary(regionName)));
+ return;
+ }
+ if (regionInfo.getStartKey() != null
+ && Bytes.compareTo(regionInfo.getStartKey(), splitPoint) == 0) {
+ future.completeExceptionally(new IllegalArgumentException(
+ "should not give a splitkey which equals to startkey!"));
+ return;
+ }
+ split(regionInfo, splitPoint).whenComplete((ret, err2) -> {
+ if (err2 != null) {
+ future.completeExceptionally(err2);
+ } else {
+ future.complete(ret);
+ }
+ });
+ });
+ return future;
+ }
+ private CompletableFuture<Void> split(final RegionInfo hri, byte[] splitPoint) {
CompletableFuture<Void> future = new CompletableFuture<>();
TableName tableName = hri.getTable();
SplitTableRegionRequest request = null;
try {
request = RequestConverter
- .buildSplitTableRegionRequest(hri, splitPoint.isPresent() ? splitPoint.get() : null,
+ .buildSplitTableRegionRequest(hri, splitPoint,
ng.getNonceGroup(), ng.newNonce());
} catch (DeserializationException e) {
future.completeExceptionally(e);
@@ -1266,7 +1357,7 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<Void> move(byte[] regionName, Optional<ServerName> destServerName) {
+ public CompletableFuture<Void> move(byte[] regionName) {
CompletableFuture<Void> future = new CompletableFuture<>();
getRegionInfo(regionName).whenComplete(
(regionInfo, err) -> {
@@ -1274,12 +1365,9 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
future.completeExceptionally(err);
return;
}
- this.<Void> newMasterCaller()
- .action(
- (controller, stub) -> this.<MoveRegionRequest, MoveRegionResponse, Void> call(
- controller, stub, RequestConverter.buildMoveRegionRequest(
- regionInfo.getEncodedNameAsBytes(), destServerName), (s, c, req, done) -> s
- .moveRegion(c, req, done), resp -> null)).call().whenComplete((ret, err2) -> {
+ moveRegion(
+ RequestConverter.buildMoveRegionRequest(regionInfo.getEncodedNameAsBytes(), null))
+ .whenComplete((ret, err2) -> {
if (err2 != null) {
future.completeExceptionally(err2);
} else {
@@ -1291,6 +1379,37 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
}
@Override
+ public CompletableFuture<Void> move(byte[] regionName, ServerName destServerName) {
+ Preconditions.checkNotNull(destServerName,
+ "destServerName is null. If you don't specify a destServerName, use move(byte[]) instead");
+ CompletableFuture<Void> future = new CompletableFuture<>();
+ getRegionInfo(regionName).whenComplete((regionInfo, err) -> {
+ if (err != null) {
+ future.completeExceptionally(err);
+ return;
+ }
+ moveRegion(
+ RequestConverter.buildMoveRegionRequest(regionInfo.getEncodedNameAsBytes(), destServerName))
+ .whenComplete((ret, err2) -> {
+ if (err2 != null) {
+ future.completeExceptionally(err2);
+ } else {
+ future.complete(ret);
+ }
+ });
+ });
+ return future;
+ }
+
+ private CompletableFuture<Void> moveRegion(MoveRegionRequest request) {
+ return this
+ .<Void> newMasterCaller()
+ .action(
+ (controller, stub) -> this.<MoveRegionRequest, MoveRegionResponse, Void> call(controller,
+ stub, request, (s, c, req, done) -> s.moveRegion(c, req, done), resp -> null)).call();
+ }
+
+ @Override
public CompletableFuture<Void> setQuota(QuotaSettings quota) {
return this
.<Void> newMasterCaller()
@@ -1452,7 +1571,19 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers(Optional<Pattern> pattern) {
+ public CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers() {
+ return listReplicationPeers(RequestConverter.buildListReplicationPeersRequest(null));
+ }
+
+ @Override
+ public CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers(Pattern pattern) {
+ Preconditions.checkNotNull(pattern,
+ "pattern is null. If you don't specify a pattern, use listReplicationPeers() instead");
+ return listReplicationPeers(RequestConverter.buildListReplicationPeersRequest(pattern));
+ }
+
+ private CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers(
+ ListReplicationPeersRequest request) {
return this
.<List<ReplicationPeerDescription>> newMasterCaller()
.action(
@@ -1460,7 +1591,7 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
.<ListReplicationPeersRequest, ListReplicationPeersResponse, List<ReplicationPeerDescription>> call(
controller,
stub,
- RequestConverter.buildListReplicationPeersRequest(pattern),
+ request,
(s, c, req, done) -> s.listReplicationPeers(c, req, done),
(resp) -> resp.getPeerDescList().stream()
.map(ReplicationSerDeHelper::toReplicationPeerDescription)
@@ -1570,7 +1701,7 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
@Override
public CompletableFuture<Void> restoreSnapshot(String snapshotName, boolean takeFailSafeSnapshot) {
CompletableFuture<Void> future = new CompletableFuture<>();
- listSnapshots(Optional.of(Pattern.compile(snapshotName))).whenComplete(
+ listSnapshots(Pattern.compile(snapshotName)).whenComplete(
(snapshotDescriptions, err) -> {
if (err != null) {
future.completeExceptionally(err);
@@ -1715,37 +1846,47 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
}
@Override
- public CompletableFuture<List<SnapshotDescription>> listSnapshots(Optional<Pattern> pattern) {
- CompletableFuture<List<SnapshotDescription>> future = new CompletableFuture<>();
- this.<GetCompletedSnapshotsResponse> newMasterCaller()
- .action(
- (controller, stub) -> this
- .<GetCompletedSnapshotsRequest, GetCompletedSnapshotsResponse, GetCompletedSnapshotsResponse> call(
- controller, stub, GetCompletedSnapshotsRequest.newBuilder().build(), (s, c, req,
- done) -> s.getCompletedSnapshots(c, req, done), resp -> resp))
- .call()
- .whenComplete(
- (resp, err) -> {
- if (err != null) {
- future.completeExceptionally(err);
- return;
- }
- future.complete(resp
- .getSnapshotsList()
- .stream()
- .map(ProtobufUtil::createSnapshotDesc)
- .filter(
- snap -> pattern.isPresent() ? pattern.get().matcher(snap.getName()).matches()
- : true).collect(Collectors.toList()));
- });
- return future;
+ public CompletableFuture<List<SnapshotDescription>> listSnapshots() {
+ return getCompletedSnapshots(null);
+ }
+
+ @Override
+ public CompletableFuture<List<SnapshotDescription>> listSnapshots(Pattern pattern) {
+ Preconditions.checkNotNull(pattern,
+ "pattern is null. If you don't specify a pattern, use listSnapshots() instead");
+ return getCompletedSnapshots(pattern);
+ }
+
+ private CompletableFuture<List<SnapshotDescription>> getCompletedSnapshots(Pattern pattern) {
+ return this.<List<SnapshotDescription>> newMasterCaller().action((controller, stub) -> this
+ .<GetCompletedSnapshotsRequest, GetCompletedSnapshotsResponse, List<SnapshotDescription>>
+ call(controller, stub, GetCompletedSnapshotsRequest.newBuilder().build(),
+ (s, c, req, done) -> s.getCompletedSnapshots(c, req, done),
+ resp -> ProtobufUtil.toSnapshotDescriptionList(resp, pattern)))
+ .call();
+ }
+
+ @Override
+ public CompletableFuture<List<SnapshotDescription>> listTableSnapshots(Pattern tableNamePattern) {
+ Preconditions.checkNotNull(tableNamePattern, "tableNamePattern is null."
+ + " If you don't specify a tableNamePattern, use listSnapshots() instead");
+ return getCompletedSnapshots(tableNamePattern, null);
}
@Override
public CompletableFuture<List<SnapshotDescription>> listTableSnapshots(Pattern tableNamePattern,
Pattern snapshotNamePattern) {
+ Preconditions.checkNotNull(tableNamePattern, "tableNamePattern is null."
+ + " If you don't specify a tableNamePattern, use listSnapshots(Pattern) instead");
+ Preconditions.checkNotNull(snapshotNamePattern, "snapshotNamePattern is null."
+ + " If you don't specify a snapshotNamePattern, use listTableSnapshots(Pattern) instead");
+ return getCompletedSnapshots(tableNamePattern, snapshotNamePattern);
+ }
+
+ private CompletableFuture<List<SnapshotDescription>> getCompletedSnapshots(
+ Pattern tableNamePattern, Pattern snapshotNamePattern) {
CompletableFuture<List<SnapshotDescription>> future = new CompletableFuture<>();
- listTableNames(Optional.ofNullable(tableNamePattern), false).whenComplete(
+ listTableNames(tableNamePattern, false).whenComplete(
(tableNames, err) -> {
if (err != null) {
future.completeExceptionally(err);
@@ -1755,7 +1896,7 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
future.complete(Collections.emptyList());
return;
}
- listSnapshots(Optional.ofNullable(snapshotNamePattern)).whenComplete(
+ getCompletedSnapshots(snapshotNamePattern).whenComplete(
(snapshotDescList, err2) -> {
if (err2 != null) {
future.completeExceptionally(err2);
@@ -1779,31 +1920,59 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
}
@Override
+ public CompletableFuture<Void> deleteSnapshots() {
+ return internalDeleteSnapshots(null, null);
+ }
+
+ @Override
public CompletableFuture<Void> deleteSnapshots(Pattern snapshotNamePattern) {
- return deleteTableSnapshots(null, snapshotNamePattern);
+ Preconditions.checkNotNull(snapshotNamePattern, "snapshotNamePattern is null."
+ + " If you don't specify a snapshotNamePattern, use deleteSnapshots() instead");
+ return internalDeleteSnapshots(null, snapshotNamePattern);
+ }
+
+ @Override
+ public CompletableFuture<Void> deleteTableSnapshots(Pattern tableNamePattern) {
+ Preconditions.checkNotNull(tableNamePattern, "tableNamePattern is null."
+ + " If you don't specify a tableNamePattern, use deleteSnapshots() instead");
+ return internalDeleteSnapshots(tableNamePattern, null);
}
@Override
public CompletableFuture<Void> deleteTableSnapshots(Pattern tableNamePattern,
Pattern snapshotNamePattern) {
+ Preconditions.checkNotNull(tableNamePattern, "tableNamePattern is null."
+ + " If you don't specify a tableNamePattern, use deleteSnapshots(Pattern) instead");
+ Preconditions.checkNotNull(snapshotNamePattern, "snapshotNamePattern is null."
+ + " If you don't specify a snapshotNamePattern, use deleteSnapshots(Pattern) instead");
+ return internalDeleteSnapshots(tableNamePattern, snapshotNamePattern);
+ }
+
+ private CompletableFuture<Void> internalDeleteSnapshots(Pattern tableNamePattern,
+ Pattern snapshotNamePattern) {
+ CompletableFuture<List<SnapshotDescription>> listSnapshotsFuture;
+ if (tableNamePattern == null) {
+ listSnapshotsFuture = getCompletedSnapshots(snapshotNamePattern);
+ } else {
+ listSnapshotsFuture = getCompletedSnapshots(tableNamePattern, snapshotNamePattern);
+ }
CompletableFuture<Void> future = new CompletableFuture<>();
- listTableSnapshots(tableNamePattern, snapshotNamePattern).whenComplete(
- ((snapshotDescriptions, err) -> {
- if (err != null) {
- future.completeExceptionally(err);
- return;
- }
- if (snapshotDescriptions == null || snapshotDescriptions.isEmpty()) {
- future.complete(null);
- return;
- }
- List<CompletableFuture<Void>> deleteSnapshotFutures = new ArrayList<>();
- snapshotDescriptions.forEach(snapDesc -> deleteSnapshotFutures
- .add(internalDeleteSnapshot(snapDesc)));
- CompletableFuture.allOf(
- deleteSnapshotFutures.toArray(new CompletableFuture<?>[deleteSnapshotFutures.size()]))
- .thenAccept(v -> future.complete(v));
- }));
+ listSnapshotsFuture.whenComplete(((snapshotDescriptions, err) -> {
+ if (err != null) {
+ future.completeExceptionally(err);
+ return;
+ }
+ if (snapshotDescriptions == null || snapshotDescriptions.isEmpty()) {
+ future.complete(null);
+ return;
+ }
+ List<CompletableFuture<Void>> deleteSnapshotFutures = new ArrayList<>();
+ snapshotDescriptions.forEach(snapDesc -> deleteSnapshotFutures
+ .add(internalDeleteSnapshot(snapDesc)));
+ CompletableFuture.allOf(
+ deleteSnapshotFutures.toArray(new CompletableFuture<?>[deleteSnapshotFutures.size()]))
+ .thenAccept(v -> future.complete(v));
+ }));
return future;
}
@@ -2485,16 +2654,26 @@ public class RawAsyncHBaseAdmin implements AsyncAdmin {
}
@Override
+ public CompletableFuture<List<RegionLoad>> getRegionLoads(ServerName serverName) {
+ return getRegionLoad(GetRegionLoadRequest.newBuilder().build(), serverName);
+ }
+
+ @Override
public CompletableFuture<List<RegionLoad>> getRegionLoads(ServerName serverName,
- Optional<TableName> tableName) {
- return this
- .<List<RegionLoad>> newAdminCaller()
- .action(
- (controller, stub) -> this
- .<GetRegionLoadRequest, GetRegionLoadResponse, List<RegionLoad>> adminCall(
- controller, stub, RequestConverter.buildGetRegionLoadRequest(tableName), (s, c,
- req, done) -> s.getRegionLoad(controller, req, done),
- ProtobufUtil::getRegionLoadInfo)).serverName(serverName).call();
+ TableName tableName) {
+ Preconditions.checkNotNull(tableName,
+ "tableName is null. If you don't specify a tableName, use getRegionLoads() instead");
+ return getRegionLoad(RequestConverter.buildGetRegionLoadRequest(tableName), serverName);
+ }
+
+ private CompletableFuture<List<RegionLoad>> getRegionLoad(GetRegionLoadRequest request,
+ ServerName serverName) {
+ return this.<List<RegionLoad>> newAdminCaller()
+ .action((controller, stub) -> this
+ .<GetRegionLoadRequest, GetRegionLoadResponse, List<RegionLoad>> adminCall(controller,
+ stub, request, (s, c, req, done) -> s.getRegionLoad(controller, req, done),
+ ProtobufUtil::getRegionLoadInfo))
+ .serverName(serverName).call();
}
@Override
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-client/src/main/java/org/apache/hadoop/hbase/shaded/protobuf/ProtobufUtil.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/shaded/protobuf/ProtobufUtil.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/shaded/protobuf/ProtobufUtil.java
index 9024cdd..7953b8a 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/shaded/protobuf/ProtobufUtil.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/shaded/protobuf/ProtobufUtil.java
@@ -36,6 +36,7 @@ import java.util.Optional;
import java.util.concurrent.Callable;
import java.util.concurrent.TimeUnit;
import java.util.function.Function;
+import java.util.regex.Pattern;
import java.util.stream.Collectors;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.Path;
@@ -158,6 +159,7 @@ import org.apache.hadoop.hbase.shaded.protobuf.generated.LockServiceProtos;
import org.apache.hadoop.hbase.shaded.protobuf.generated.MapReduceProtos;
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos;
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.CreateTableRequest;
+import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.GetCompletedSnapshotsResponse;
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.GetTableDescriptorsResponse;
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.ListNamespaceDescriptorsResponse;
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.MajorCompactionTimestampResponse;
@@ -1733,7 +1735,7 @@ public final class ProtobufUtil {
final RpcController controller, final AdminService.BlockingInterface admin,
final TableName tableName) throws IOException {
GetRegionLoadRequest request =
- RequestConverter.buildGetRegionLoadRequest(Optional.ofNullable(tableName));
+ RequestConverter.buildGetRegionLoadRequest(tableName);
GetRegionLoadResponse response;
try {
response = admin.getRegionLoad(controller, request);
@@ -3376,4 +3378,11 @@ public final class ProtobufUtil {
}
return rib.build();
}
+
+ public static List<SnapshotDescription> toSnapshotDescriptionList(
+ GetCompletedSnapshotsResponse response, Pattern pattern) {
+ return response.getSnapshotsList().stream().map(ProtobufUtil::createSnapshotDesc)
+ .filter(snap -> pattern != null ? pattern.matcher(snap.getName()).matches() : true)
+ .collect(Collectors.toList());
+ }
}
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-client/src/main/java/org/apache/hadoop/hbase/shaded/protobuf/RequestConverter.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/shaded/protobuf/RequestConverter.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/shaded/protobuf/RequestConverter.java
index 72bd324..4ad28f2 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/shaded/protobuf/RequestConverter.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/shaded/protobuf/RequestConverter.java
@@ -19,11 +19,9 @@ package org.apache.hadoop.hbase.shaded.protobuf;
import java.io.IOException;
import java.util.ArrayList;
-import java.util.Arrays;
import java.util.Collection;
import java.util.EnumSet;
import java.util.List;
-import java.util.Optional;
import java.util.Set;
import java.util.regex.Pattern;
@@ -824,132 +822,112 @@ public final class RequestConverter {
* Create a protocol buffer GetRegionLoadRequest for all regions/regions of a table.
* @param tableName the table for which regionLoad should be obtained from RS
* @return a protocol buffer GetRegionLoadRequest
- * @deprecated use {@link #buildGetRegionLoadRequest(Optional)} instead.
*/
- @Deprecated
public static GetRegionLoadRequest buildGetRegionLoadRequest(final TableName tableName) {
- return buildGetRegionLoadRequest(Optional.ofNullable(tableName));
+ GetRegionLoadRequest.Builder builder = GetRegionLoadRequest.newBuilder();
+ if (tableName != null) {
+ builder.setTableName(ProtobufUtil.toProtoTableName(tableName));
+ }
+ return builder.build();
}
/**
- * Create a protocol buffer GetRegionLoadRequest for all regions/regions of a table.
- * @param tableName the table for which regionLoad should be obtained from RS
- * @return a protocol buffer GetRegionLoadRequest
+ * Create a protocol buffer GetOnlineRegionRequest
+ * @return a protocol buffer GetOnlineRegionRequest
*/
- public static GetRegionLoadRequest buildGetRegionLoadRequest(Optional<TableName> tableName) {
- GetRegionLoadRequest.Builder builder = GetRegionLoadRequest.newBuilder();
- tableName.ifPresent(table -> builder.setTableName(ProtobufUtil.toProtoTableName(table)));
+ public static GetOnlineRegionRequest buildGetOnlineRegionRequest() {
+ return GetOnlineRegionRequest.newBuilder().build();
+ }
+
+ /**
+ * Create a protocol buffer FlushRegionRequest for a given region name
+ * @param regionName the name of the region to get info
+ * @return a protocol buffer FlushRegionRequest
+ */
+ public static FlushRegionRequest buildFlushRegionRequest(final byte[] regionName) {
+ return buildFlushRegionRequest(regionName, false);
+ }
+
+ /**
+ * Create a protocol buffer FlushRegionRequest for a given region name
+ * @param regionName the name of the region to get info
+ * @return a protocol buffer FlushRegionRequest
+ */
+ public static FlushRegionRequest buildFlushRegionRequest(final byte[] regionName,
+ boolean writeFlushWALMarker) {
+ FlushRegionRequest.Builder builder = FlushRegionRequest.newBuilder();
+ RegionSpecifier region = buildRegionSpecifier(RegionSpecifierType.REGION_NAME, regionName);
+ builder.setRegion(region);
+ builder.setWriteFlushWalMarker(writeFlushWALMarker);
return builder.build();
}
- /**
- * Create a protocol buffer GetOnlineRegionRequest
- *
- * @return a protocol buffer GetOnlineRegionRequest
- */
- public static GetOnlineRegionRequest buildGetOnlineRegionRequest() {
- return GetOnlineRegionRequest.newBuilder().build();
- }
-
- /**
- * Create a protocol buffer FlushRegionRequest for a given region name
- *
- * @param regionName the name of the region to get info
- * @return a protocol buffer FlushRegionRequest
- */
- public static FlushRegionRequest
- buildFlushRegionRequest(final byte[] regionName) {
- return buildFlushRegionRequest(regionName, false);
- }
-
- /**
- * Create a protocol buffer FlushRegionRequest for a given region name
- *
- * @param regionName the name of the region to get info
- * @return a protocol buffer FlushRegionRequest
- */
- public static FlushRegionRequest
- buildFlushRegionRequest(final byte[] regionName, boolean writeFlushWALMarker) {
- FlushRegionRequest.Builder builder = FlushRegionRequest.newBuilder();
- RegionSpecifier region = buildRegionSpecifier(
- RegionSpecifierType.REGION_NAME, regionName);
- builder.setRegion(region);
- builder.setWriteFlushWalMarker(writeFlushWALMarker);
- return builder.build();
- }
-
- /**
- * Create a protocol buffer OpenRegionRequest to open a list of regions
- *
- * @param server the serverName for the RPC
- * @param regionOpenInfos info of a list of regions to open
- * @param openForReplay
- * @return a protocol buffer OpenRegionRequest
- */
- public static OpenRegionRequest
- buildOpenRegionRequest(ServerName server, final List<Pair<RegionInfo,
- List<ServerName>>> regionOpenInfos, Boolean openForReplay) {
- OpenRegionRequest.Builder builder = OpenRegionRequest.newBuilder();
- for (Pair<RegionInfo, List<ServerName>> regionOpenInfo: regionOpenInfos) {
- builder.addOpenInfo(buildRegionOpenInfo(regionOpenInfo.getFirst(),
- regionOpenInfo.getSecond(), openForReplay));
- }
- if (server != null) {
- builder.setServerStartCode(server.getStartcode());
- }
- // send the master's wall clock time as well, so that the RS can refer to it
- builder.setMasterSystemTime(EnvironmentEdgeManager.currentTime());
- return builder.build();
- }
-
- /**
- * Create a protocol buffer OpenRegionRequest for a given region
- *
- * @param server the serverName for the RPC
- * @param region the region to open
- * @param favoredNodes
- * @param openForReplay
- * @return a protocol buffer OpenRegionRequest
- */
- public static OpenRegionRequest buildOpenRegionRequest(ServerName server,
- final RegionInfo region, List<ServerName> favoredNodes,
- Boolean openForReplay) {
- OpenRegionRequest.Builder builder = OpenRegionRequest.newBuilder();
- builder.addOpenInfo(buildRegionOpenInfo(region, favoredNodes,
- openForReplay));
- if (server != null) {
- builder.setServerStartCode(server.getStartcode());
- }
- builder.setMasterSystemTime(EnvironmentEdgeManager.currentTime());
- return builder.build();
- }
-
- /**
- * Create a protocol buffer UpdateFavoredNodesRequest to update a list of favorednode mappings
- * @param updateRegionInfos
- * @return a protocol buffer UpdateFavoredNodesRequest
- */
- public static UpdateFavoredNodesRequest buildUpdateFavoredNodesRequest(
- final List<Pair<RegionInfo, List<ServerName>>> updateRegionInfos) {
- UpdateFavoredNodesRequest.Builder ubuilder = UpdateFavoredNodesRequest.newBuilder();
- if (updateRegionInfos != null && !updateRegionInfos.isEmpty()) {
- RegionUpdateInfo.Builder builder = RegionUpdateInfo.newBuilder();
- for (Pair<RegionInfo, List<ServerName>> pair : updateRegionInfos) {
- builder.setRegion(ProtobufUtil.toRegionInfo(pair.getFirst()));
- for (ServerName server : pair.getSecond()) {
- builder.addFavoredNodes(ProtobufUtil.toServerName(server));
+ /**
+ * Create a protocol buffer OpenRegionRequest to open a list of regions
+ * @param server the serverName for the RPC
+ * @param regionOpenInfos info of a list of regions to open
+ * @param openForReplay whether open for replay
+ * @return a protocol buffer OpenRegionRequest
+ */
+ public static OpenRegionRequest buildOpenRegionRequest(ServerName server,
+ final List<Pair<RegionInfo, List<ServerName>>> regionOpenInfos, Boolean openForReplay) {
+ OpenRegionRequest.Builder builder = OpenRegionRequest.newBuilder();
+ for (Pair<RegionInfo, List<ServerName>> regionOpenInfo : regionOpenInfos) {
+ builder.addOpenInfo(buildRegionOpenInfo(regionOpenInfo.getFirst(),
+ regionOpenInfo.getSecond(), openForReplay));
+ }
+ if (server != null) {
+ builder.setServerStartCode(server.getStartcode());
+ }
+ // send the master's wall clock time as well, so that the RS can refer to it
+ builder.setMasterSystemTime(EnvironmentEdgeManager.currentTime());
+ return builder.build();
+ }
+
+ /**
+ * Create a protocol buffer OpenRegionRequest for a given region
+ * @param server the serverName for the RPC
+ * @param region the region to open
+ * @param favoredNodes a list of favored nodes
+ * @param openForReplay whether open for replay
+ * @return a protocol buffer OpenRegionRequest
+ */
+ public static OpenRegionRequest buildOpenRegionRequest(ServerName server,
+ final RegionInfo region, List<ServerName> favoredNodes, Boolean openForReplay) {
+ OpenRegionRequest.Builder builder = OpenRegionRequest.newBuilder();
+ builder.addOpenInfo(buildRegionOpenInfo(region, favoredNodes, openForReplay));
+ if (server != null) {
+ builder.setServerStartCode(server.getStartcode());
+ }
+ builder.setMasterSystemTime(EnvironmentEdgeManager.currentTime());
+ return builder.build();
+ }
+
+ /**
+ * Create a protocol buffer UpdateFavoredNodesRequest to update a list of favorednode mappings
+ * @param updateRegionInfos a list of favored node mappings
+ * @return a protocol buffer UpdateFavoredNodesRequest
+ */
+ public static UpdateFavoredNodesRequest buildUpdateFavoredNodesRequest(
+ final List<Pair<RegionInfo, List<ServerName>>> updateRegionInfos) {
+ UpdateFavoredNodesRequest.Builder ubuilder = UpdateFavoredNodesRequest.newBuilder();
+ if (updateRegionInfos != null && !updateRegionInfos.isEmpty()) {
+ RegionUpdateInfo.Builder builder = RegionUpdateInfo.newBuilder();
+ for (Pair<RegionInfo, List<ServerName>> pair : updateRegionInfos) {
+ builder.setRegion(ProtobufUtil.toRegionInfo(pair.getFirst()));
+ for (ServerName server : pair.getSecond()) {
+ builder.addFavoredNodes(ProtobufUtil.toServerName(server));
+ }
+ ubuilder.addUpdateInfo(builder.build());
+ builder.clear();
}
- ubuilder.addUpdateInfo(builder.build());
- builder.clear();
}
- }
- return ubuilder.build();
- }
+ return ubuilder.build();
+ }
/**
- * Create a WarmupRegionRequest for a given region name
- *
- * @param regionInfo Region we are warming up
+ * Create a WarmupRegionRequest for a given region name
+ * @param regionInfo Region we are warming up
*/
public static WarmupRegionRequest buildWarmupRegionRequest(final RegionInfo regionInfo) {
WarmupRegionRequest.Builder builder = WarmupRegionRequest.newBuilder();
@@ -963,72 +941,57 @@ public final class RequestConverter {
* @param major indicator if it is a major compaction
* @param columnFamily
* @return a CompactRegionRequest
- * @deprecated Use {@link #buildCompactRegionRequest(byte[], boolean, Optional)} instead.
*/
- @Deprecated
public static CompactRegionRequest buildCompactRegionRequest(byte[] regionName, boolean major,
byte[] columnFamily) {
- return buildCompactRegionRequest(regionName, major, Optional.ofNullable(columnFamily));
- }
-
- /**
- * Create a CompactRegionRequest for a given region name
- * @param regionName the name of the region to get info
- * @param major indicator if it is a major compaction
- * @param columnFamily
- * @return a CompactRegionRequest
- */
- public static CompactRegionRequest buildCompactRegionRequest(byte[] regionName, boolean major,
- Optional<byte[]> columnFamily) {
CompactRegionRequest.Builder builder = CompactRegionRequest.newBuilder();
RegionSpecifier region = buildRegionSpecifier(RegionSpecifierType.REGION_NAME, regionName);
builder.setRegion(region);
builder.setMajor(major);
- columnFamily.ifPresent(family -> builder.setFamily(UnsafeByteOperations.unsafeWrap(family)));
+ if (columnFamily != null) {
+ builder.setFamily(UnsafeByteOperations.unsafeWrap(columnFamily));
+ }
return builder.build();
}
- /**
- * @see {@link #buildRollWALWriterRequest()}
- */
- private static RollWALWriterRequest ROLL_WAL_WRITER_REQUEST =
- RollWALWriterRequest.newBuilder().build();
-
- /**
- * Create a new RollWALWriterRequest
- *
- * @return a ReplicateWALEntryRequest
- */
- public static RollWALWriterRequest buildRollWALWriterRequest() {
- return ROLL_WAL_WRITER_REQUEST;
- }
-
- /**
- * @see {@link #buildGetServerInfoRequest()}
- */
- private static GetServerInfoRequest GET_SERVER_INFO_REQUEST =
- GetServerInfoRequest.newBuilder().build();
-
- /**
- * Create a new GetServerInfoRequest
- *
- * @return a GetServerInfoRequest
- */
- public static GetServerInfoRequest buildGetServerInfoRequest() {
- return GET_SERVER_INFO_REQUEST;
- }
-
- /**
- * Create a new StopServerRequest
- *
- * @param reason the reason to stop the server
- * @return a StopServerRequest
- */
- public static StopServerRequest buildStopServerRequest(final String reason) {
- StopServerRequest.Builder builder = StopServerRequest.newBuilder();
- builder.setReason(reason);
- return builder.build();
- }
+ /**
+ * @see {@link #buildRollWALWriterRequest()}
+ */
+ private static RollWALWriterRequest ROLL_WAL_WRITER_REQUEST = RollWALWriterRequest.newBuilder()
+ .build();
+
+ /**
+ * Create a new RollWALWriterRequest
+ * @return a ReplicateWALEntryRequest
+ */
+ public static RollWALWriterRequest buildRollWALWriterRequest() {
+ return ROLL_WAL_WRITER_REQUEST;
+ }
+
+ /**
+ * @see {@link #buildGetServerInfoRequest()}
+ */
+ private static GetServerInfoRequest GET_SERVER_INFO_REQUEST = GetServerInfoRequest.newBuilder()
+ .build();
+
+ /**
+ * Create a new GetServerInfoRequest
+ * @return a GetServerInfoRequest
+ */
+ public static GetServerInfoRequest buildGetServerInfoRequest() {
+ return GET_SERVER_INFO_REQUEST;
+ }
+
+ /**
+ * Create a new StopServerRequest
+ * @param reason the reason to stop the server
+ * @return a StopServerRequest
+ */
+ public static StopServerRequest buildStopServerRequest(final String reason) {
+ StopServerRequest.Builder builder = StopServerRequest.newBuilder();
+ builder.setReason(reason);
+ return builder.build();
+ }
//End utilities for Admin
@@ -1136,36 +1099,15 @@ public final class RequestConverter {
* @param encodedRegionName
* @param destServerName
* @return A MoveRegionRequest
- * @throws DeserializationException
- * @deprecated Use {@link #buildMoveRegionRequest(byte[], Optional)} instead.
- */
- @Deprecated
- public static MoveRegionRequest buildMoveRegionRequest(
- final byte [] encodedRegionName, final byte [] destServerName) throws
- DeserializationException {
- MoveRegionRequest.Builder builder = MoveRegionRequest.newBuilder();
- builder.setRegion(
- buildRegionSpecifier(RegionSpecifierType.ENCODED_REGION_NAME,encodedRegionName));
- if (destServerName != null) {
- builder.setDestServerName(
- ProtobufUtil.toServerName(ServerName.valueOf(Bytes.toString(destServerName))));
- }
- return builder.build();
- }
-
- /**
- * Create a protocol buffer MoveRegionRequest
- * @param encodedRegionName
- * @param destServerName
- * @return A MoveRegionRequest
*/
public static MoveRegionRequest buildMoveRegionRequest(byte[] encodedRegionName,
- Optional<ServerName> destServerName) {
+ ServerName destServerName) {
MoveRegionRequest.Builder builder = MoveRegionRequest.newBuilder();
builder.setRegion(buildRegionSpecifier(RegionSpecifierType.ENCODED_REGION_NAME,
encodedRegionName));
- destServerName.ifPresent(serverName -> builder.setDestServerName(ProtobufUtil
- .toServerName(serverName)));
+ if (destServerName != null) {
+ builder.setDestServerName(ProtobufUtil.toServerName(destServerName));
+ }
return builder.build();
}
@@ -1320,21 +1262,13 @@ public final class RequestConverter {
final byte [][] splitKeys,
final long nonceGroup,
final long nonce) {
- return buildCreateTableRequest(tableDescriptor, Optional.ofNullable(splitKeys), nonceGroup, nonce);
- }
-
- /**
- * Creates a protocol buffer CreateTableRequest
- * @param tableDescriptor
- * @param splitKeys
- * @return a CreateTableRequest
- */
- public static CreateTableRequest buildCreateTableRequest(TableDescriptor tableDescriptor,
- Optional<byte[][]> splitKeys, long nonceGroup, long nonce) {
CreateTableRequest.Builder builder = CreateTableRequest.newBuilder();
builder.setTableSchema(ProtobufUtil.toTableSchema(tableDescriptor));
- splitKeys.ifPresent(keys -> Arrays.stream(keys).forEach(
- key -> builder.addSplitKeys(UnsafeByteOperations.unsafeWrap(key))));
+ if (splitKeys != null) {
+ for(byte[] key : splitKeys) {
+ builder.addSplitKeys(UnsafeByteOperations.unsafeWrap(key));
+ }
+ }
builder.setNonceGroup(nonceGroup);
builder.setNonce(nonce);
return builder.build();
@@ -1396,25 +1330,13 @@ public final class RequestConverter {
* @param pattern The compiled regular expression to match against
* @param includeSysTables False to match only against userspace tables
* @return a GetTableDescriptorsRequest
- * @deprecated Use {@link #buildGetTableDescriptorsRequest(Optional, boolean)} instead.
*/
- @Deprecated
public static GetTableDescriptorsRequest buildGetTableDescriptorsRequest(final Pattern pattern,
boolean includeSysTables) {
- return buildGetTableDescriptorsRequest(Optional.ofNullable(pattern), includeSysTables);
- }
-
- /**
- * Creates a protocol buffer GetTableDescriptorsRequest
- *
- * @param pattern The compiled regular expression to match against
- * @param includeSysTables False to match only against userspace tables
- * @return a GetTableDescriptorsRequest
- */
- public static GetTableDescriptorsRequest
- buildGetTableDescriptorsRequest(Optional<Pattern> pattern, boolean includeSysTables) {
GetTableDescriptorsRequest.Builder builder = GetTableDescriptorsRequest.newBuilder();
- pattern.ifPresent(p -> builder.setRegex(p.toString()));
+ if (pattern != null) {
+ builder.setRegex(pattern.toString());
+ }
builder.setIncludeSysTables(includeSysTables);
return builder.build();
}
@@ -1425,25 +1347,13 @@ public final class RequestConverter {
* @param pattern The compiled regular expression to match against
* @param includeSysTables False to match only against userspace tables
* @return a GetTableNamesRequest
- * @deprecated Use {@link #buildGetTableNamesRequest(Optional, boolean)} instead.
*/
- @Deprecated
public static GetTableNamesRequest buildGetTableNamesRequest(final Pattern pattern,
boolean includeSysTables) {
- return buildGetTableNamesRequest(Optional.ofNullable(pattern), includeSysTables);
- }
-
- /**
- * Creates a protocol buffer GetTableNamesRequest
- *
- * @param pattern The compiled regular expression to match against
- * @param includeSysTables False to match only against userspace tables
- * @return a GetTableNamesRequest
- */
- public static GetTableNamesRequest buildGetTableNamesRequest(Optional<Pattern> pattern,
- boolean includeSysTables) {
GetTableNamesRequest.Builder builder = GetTableNamesRequest.newBuilder();
- pattern.ifPresent(p -> builder.setRegex(p.toString()));
+ if (pattern != null) {
+ builder.setRegex(pattern.toString());
+ }
builder.setIncludeSysTables(includeSysTables);
return builder.build();
}
@@ -1745,18 +1655,11 @@ public final class RequestConverter {
return builder.build();
}
- /**
- * @deprecated Use {@link #buildListReplicationPeersRequest(Optional)} instead.
- */
- @Deprecated
public static ListReplicationPeersRequest buildListReplicationPeersRequest(Pattern pattern) {
- return buildListReplicationPeersRequest(Optional.ofNullable(pattern));
- }
-
- public static ListReplicationPeersRequest
- buildListReplicationPeersRequest(Optional<Pattern> pattern) {
ListReplicationPeersRequest.Builder builder = ListReplicationPeersRequest.newBuilder();
- pattern.ifPresent(p -> builder.setRegex(p.toString()));
+ if (pattern != null) {
+ builder.setRegex(pattern.toString());
+ }
return builder.build();
}
@@ -1877,4 +1780,4 @@ public final class RequestConverter {
}
return pbServers;
}
-}
+}
\ No newline at end of file
http://git-wip-us.apache.org/repos/asf/hbase/blob/888f2335/hbase-endpoint/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoadWithOldSecureEndpoint.java
----------------------------------------------------------------------
diff --git a/hbase-endpoint/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoadWithOldSecureEndpoint.java b/hbase-endpoint/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoadWithOldSecureEndpoint.java
index 0d5c993..25953bc 100644
--- a/hbase-endpoint/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoadWithOldSecureEndpoint.java
+++ b/hbase-endpoint/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegionServerBulkLoadWithOldSecureEndpoint.java
@@ -20,7 +20,6 @@ package org.apache.hadoop.hbase.regionserver;
import java.io.IOException;
import java.util.ArrayList;
import java.util.List;
-import java.util.Optional;
import java.util.concurrent.atomic.AtomicLong;
import org.apache.commons.logging.Log;
@@ -138,7 +137,7 @@ public class TestHRegionServerBulkLoadWithOldSecureEndpoint extends TestHRegionS
conn.getAdmin(getLocation().getServerName());
CompactRegionRequest request =
RequestConverter.buildCompactRegionRequest(
- getLocation().getRegionInfo().getRegionName(), true, Optional.empty());
+ getLocation().getRegionInfo().getRegionName(), true, null);
server.compactRegion(null, request);
numCompactions.incrementAndGet();
return null;