Apache9 commented on code in PR #5837:
URL: https://github.com/apache/hbase/pull/5837#discussion_r1635971770
##########
hbase-client/src/main/java/org/apache/hadoop/hbase/zookeeper/ReadOnlyZKClient.java:
##########
@@ -274,6 +309,28 @@ protected void doExec(ZooKeeper zk) {
return future;
}
+ public CompletableFuture<Stat> existsWithTimeout(String path, int endTime) {
Review Comment:
endTime should be long instead of int...
##########
hbase-client/src/main/java/org/apache/hadoop/hbase/zookeeper/ReadOnlyZKClient.java:
##########
@@ -274,6 +309,28 @@ protected void doExec(ZooKeeper zk) {
return future;
}
+ public CompletableFuture<Stat> existsWithTimeout(String path, int endTime) {
+ CompletableFuture<Stat> future = exists(path);
+ TimerTask timerTask = new TimerTask() {
+ @Override
+ public void run(Timeout timeout) throws Exception {
+ if (EnvironmentEdgeManager.currentTime() > endTime) {
Review Comment:
Better use System.nanoTime instead of currentTime. And for Betty's timer, I
do not think we need to check for endTime again, we should queue a task into
the queue to finish the future without timeout exception.
##########
hbase-client/src/main/java/org/apache/hadoop/hbase/client/ZKConnectionRegistry.java:
##########
@@ -69,39 +73,85 @@ class ZKConnectionRegistry implements ConnectionRegistry {
private final ReadOnlyZKClient zk;
private final ZNodePaths znodePaths;
+ private final Configuration conf;
+
+ public static final String EXPECTED_TIMEOUT = "expected.timeout";
+ public static final int DEFAULT_EXPECTED_TIMEOUT = 200000;
+ public static final String MAX_ATTEMPTS = "max.attempts";
+ public static final int DEFAULT_MAX_ATTEMPTS = 5;
+ public static final String PAUSE_NS = "pause.ns";
+ public static final long DEFAULT_PAUSE_NS = 100000;
+
// User not used, but for rpc based registry we need it
ZKConnectionRegistry(Configuration conf, User ignored) {
this.znodePaths = new ZNodePaths(conf);
this.zk = new ReadOnlyZKClient(conf);
+ this.conf = conf;
if (NEEDS_LOG_WARN) {
synchronized (WARN_LOCK) {
if (NEEDS_LOG_WARN) {
LOG.warn(
"ZKConnectionRegistry is deprecated. See
https://hbase.apache.org/book.html#client.rpcconnectionregistry");
NEEDS_LOG_WARN = false;
+
}
}
}
}
+ public ZNodePaths getZNodePaths() {
+ return znodePaths;
+ }
+
private interface Converter<T> {
T convert(byte[] data) throws Exception;
}
private <T> CompletableFuture<T> getAndConvert(String path, Converter<T>
converter) {
CompletableFuture<T> future = new CompletableFuture<>();
- addListener(zk.get(path), (data, error) -> {
- if (error != null) {
- future.completeExceptionally(error);
- return;
- }
- try {
- future.complete(converter.convert(data));
- } catch (Exception e) {
- future.completeExceptionally(e);
- }
- });
+ TimerTask pollingTask = new TimerTask() {
Review Comment:
I do not think we need to do this here?
We have timeout in ReadOnlyZKClient, here we just need to schedule retries?
##########
hbase-client/src/main/java/org/apache/hadoop/hbase/zookeeper/ReadOnlyZKClient.java:
##########
@@ -76,8 +83,14 @@ public final class ReadOnlyZKClient implements Closeable {
private final int keepAliveTimeMs;
+ public static final HashedWheelTimer RETRY_TIMER = new HashedWheelTimer(
Review Comment:
Just reuse the one in AsyncConnectionImpl? Could pass it in when creating
ReadOnlyZKClient.
--
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
To unsubscribe, e-mail: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]