This is an automated email from the ASF dual-hosted git repository. inigoiri pushed a commit to branch trunk in repository https://gitbox.apache.org/repos/asf/hadoop.git
The following commit(s) were added to refs/heads/trunk by this push: new 80fefd093f7 HDFS-17067 Use BlockingThreadPoolExecutorService for nnProbingThreadPool in ObserverReadProxy (#5803) 80fefd093f7 is described below commit 80fefd093f7117757bc2d2ad8f5ca57538d8632e Author: Xing Lin <linxing...@gmail.com> AuthorDate: Thu Jul 20 10:46:41 2023 -0700 HDFS-17067 Use BlockingThreadPoolExecutorService for nnProbingThreadPool in ObserverReadProxy (#5803) --- .../namenode/ha/ObserverReadProxyProvider.java | 26 ++++++++++------------ 1 file changed, 12 insertions(+), 14 deletions(-) diff --git a/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/server/namenode/ha/ObserverReadProxyProvider.java b/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/server/namenode/ha/ObserverReadProxyProvider.java index 11143d7ef5a..2c58ad1a97b 100644 --- a/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/server/namenode/ha/ObserverReadProxyProvider.java +++ b/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/server/namenode/ha/ObserverReadProxyProvider.java @@ -27,11 +27,8 @@ import java.net.URI; import java.util.concurrent.Callable; import java.util.concurrent.ExecutionException; import java.util.concurrent.RejectedExecutionException; -import java.util.concurrent.ExecutorService; import java.util.concurrent.Future; -import java.util.concurrent.ThreadPoolExecutor; import java.util.concurrent.TimeUnit; -import java.util.concurrent.ArrayBlockingQueue; import java.util.concurrent.TimeoutException; import java.util.List; @@ -54,6 +51,7 @@ import org.apache.hadoop.ipc.RPC; import org.apache.hadoop.ipc.RemoteException; import org.apache.hadoop.ipc.RpcInvocationHandler; import org.apache.hadoop.ipc.StandbyException; +import org.apache.hadoop.util.BlockingThreadPoolExecutorService; import org.apache.hadoop.util.Time; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -187,18 +185,8 @@ public class ObserverReadProxyProvider<T> /** * Threadpool to send the getHAServiceState requests. - * - * One thread running all the time, with up to 4 threads. Idle threads will be killed after - * 1 minute. At most 1024 requests can be submitted before they start to be rejected. - * - * Each hdfs client will have its own ObserverReadProxyProvider. Thus, - * having 1 thread running should be sufficient in most cases. - * We are not expecting to receive a lot of outstanding RPC calls - * from a single hdfs client, thus setting the queue size to 1024. */ - private final ExecutorService nnProbingThreadPool = - new ThreadPoolExecutor(1, 4, 1L, TimeUnit.MINUTES, - new ArrayBlockingQueue<Runnable>(1024)); + private final BlockingThreadPoolExecutorService nnProbingThreadPool; /** * By default ObserverReadProxyProvider uses @@ -262,6 +250,15 @@ public class ObserverReadProxyProvider<T> + "class does not implement {}", uri, ClientProtocol.class.getName()); this.observerReadEnabled = false; } + + /* + * At most 4 threads will be running and each thread will die after 10 + * seconds of no use. Up to 132 tasks (4 active + 128 waiting) can be + * submitted simultaneously. + */ + nnProbingThreadPool = + BlockingThreadPoolExecutorService.newInstance(4, 128, 10L, TimeUnit.SECONDS, + "nn-ha-state-probing"); } public AlignmentContext getAlignmentContext() { @@ -648,6 +645,7 @@ public class ObserverReadProxyProvider<T> } } failoverProxy.close(); + nnProbingThreadPool.shutdown(); } @Override --------------------------------------------------------------------- To unsubscribe, e-mail: common-commits-unsubscr...@hadoop.apache.org For additional commands, e-mail: common-commits-h...@hadoop.apache.org