You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@bookkeeper.apache.org by GitBox <gi...@apache.org> on 2018/08/21 22:12:31 UTC

[GitHub] sijie closed pull request #1612: Allow to configure num of Netty IO threads in client and bookie

sijie closed pull request #1612: Allow to configure num of Netty IO threads in client and bookie
URL: https://github.com/apache/bookkeeper/pull/1612
 
 
   

This is a PR merged from a forked repository.
As GitHub hides the original diff on merge, it is displayed below for
the sake of provenance:

As this is a foreign pull request (from a fork), the diff is supplied
below (as it won't show otherwise due to GitHub magic):

diff --git a/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/BookKeeper.java b/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/BookKeeper.java
index 4fc7728bdc..bac935a4c8 100644
--- a/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/BookKeeper.java
+++ b/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/BookKeeper.java
@@ -433,7 +433,7 @@ public BookKeeper(ClientConfiguration conf, ZooKeeper zk, EventLoopGroup eventLo
 
         // initialize event loop group
         if (null == eventLoopGroup) {
-            this.eventLoopGroup = getDefaultEventLoopGroup();
+            this.eventLoopGroup = getDefaultEventLoopGroup(conf);
             this.ownEventLoopGroup = true;
         } else {
             this.eventLoopGroup = eventLoopGroup;
@@ -1297,6 +1297,7 @@ public void deleteLedger(long lId) throws InterruptedException, BKException {
      */
     public void asyncIsClosed(long lId, final IsClosedCallback cb, final Object ctx){
         ledgerManager.readLedgerMetadata(lId, new GenericCallback<LedgerMetadata>(){
+            @Override
             public void operationComplete(int rc, LedgerMetadata lm){
                 if (rc == BKException.Code.OK) {
                     cb.isClosedComplete(rc, lm.isClosed(), ctx);
@@ -1326,6 +1327,7 @@ public boolean isClosed(long lId)
         final Result result = new Result();
 
         final IsClosedCallback cb = new IsClosedCallback(){
+            @Override
             public void isClosedComplete(int rc, boolean isClosed, Object ctx){
                     result.isClosed = isClosed;
                     result.rc = rc;
@@ -1403,9 +1405,9 @@ public void close() throws BKException, InterruptedException {
         this.metadataDriver.close();
     }
 
-    static EventLoopGroup getDefaultEventLoopGroup() {
+    static EventLoopGroup getDefaultEventLoopGroup(ClientConfiguration conf) {
         ThreadFactory threadFactory = new DefaultThreadFactory("bookkeeper-io");
-        final int numThreads = Runtime.getRuntime().availableProcessors() * 2;
+        final int numThreads = conf.getNumIOThreads();
 
         if (SystemUtils.IS_OS_LINUX) {
             try {
diff --git a/bookkeeper-server/src/main/java/org/apache/bookkeeper/conf/ClientConfiguration.java b/bookkeeper-server/src/main/java/org/apache/bookkeeper/conf/ClientConfiguration.java
index 340b40b1f1..c902db823b 100644
--- a/bookkeeper-server/src/main/java/org/apache/bookkeeper/conf/ClientConfiguration.java
+++ b/bookkeeper-server/src/main/java/org/apache/bookkeeper/conf/ClientConfiguration.java
@@ -143,8 +143,9 @@
     protected static final String GET_BOOKIE_INFO_TIMEOUT_SECS = "getBookieInfoTimeoutSecs";
     protected static final String START_TLS_TIMEOUT_SECS = "startTLSTimeoutSecs";
 
-    // Number Woker Threads
+    // Number of Threads
     protected static final String NUM_WORKER_THREADS = "numWorkerThreads";
+    protected static final String NUM_IO_THREADS = "numIOThreads";
 
     // Ensemble Placement Policy
     protected static final String ENSEMBLE_PLACEMENT_POLICY = "ensemblePlacementPolicy";
@@ -850,6 +851,37 @@ public ClientConfiguration setNumWorkerThreads(int numThreads) {
         return this;
     }
 
+    /**
+     * Get the number of IO threads. This is the number of
+     * threads used by Netty to handle TCP connections.
+     *
+     * @return the number of IO threads
+     */
+    public int getNumIOThreads() {
+        return getInt(NUM_IO_THREADS, 2 * Runtime.getRuntime().availableProcessors());
+    }
+
+    /**
+     * Set the number of IO threads.
+     *
+     * <p>
+     * This is the number of threads used by Netty to handle TCP connections.
+     * </p>
+     *
+     * <p>
+     * NOTE: setting the number of IO threads after BookKeeper object is constructed
+     * will not take any effect on the number of threads in the pool.
+     * </p>
+     *
+     * @see #getNumIOThreads()
+     * @param numThreads number of IO threads used for bookkeeper
+     * @return client configuration
+     */
+    public ClientConfiguration setNumIOThreads(int numThreads) {
+        setProperty(NUM_IO_THREADS, numThreads);
+        return this;
+    }
+
     /**
      * Get the period of time after which a speculative entry read should be triggered.
      * A speculative entry read is sent to the next replica bookie before
diff --git a/bookkeeper-server/src/main/java/org/apache/bookkeeper/conf/ServerConfiguration.java b/bookkeeper-server/src/main/java/org/apache/bookkeeper/conf/ServerConfiguration.java
index 7e14f2e2ab..db40e0f502 100644
--- a/bookkeeper-server/src/main/java/org/apache/bookkeeper/conf/ServerConfiguration.java
+++ b/bookkeeper-server/src/main/java/org/apache/bookkeeper/conf/ServerConfiguration.java
@@ -106,6 +106,7 @@
     protected static final String SERVER_SOCK_LINGER = "serverTcpLinger";
     protected static final String SERVER_WRITEBUFFER_LOW_WATER_MARK = "serverWriteBufferLowWaterMark";
     protected static final String SERVER_WRITEBUFFER_HIGH_WATER_MARK = "serverWriteBufferHighWaterMark";
+    protected static final String SERVER_NUM_IO_THREADS = "serverNumIOThreads";
 
     // Zookeeper Parameters
     protected static final String ZK_RETRY_BACKOFF_START_MS = "zkRetryBackoffStartMs";
@@ -1115,6 +1116,32 @@ public ServerConfiguration setServerTcpNoDelay(boolean noDelay) {
         return this;
     }
 
+    /**
+     * Get the number of IO threads. This is the number of
+     * threads used by Netty to handle TCP connections.
+     *
+     * @return the number of IO threads
+     */
+    public int getServerNumIOThreads() {
+        return getInt(SERVER_NUM_IO_THREADS, 2 * Runtime.getRuntime().availableProcessors());
+    }
+
+    /**
+     * Set the number of IO threads.
+     *
+     * <p>
+     * This is the number of threads used by Netty to handle TCP connections.
+     * </p>
+     *
+     * @see #getNumIOThreads()
+     * @param numThreads number of IO threads used for bookkeeper
+     * @return client configuration
+     */
+    public ServerConfiguration setServerNumIOThreads(int numThreads) {
+        setProperty(SERVER_NUM_IO_THREADS, Integer.toString(numThreads));
+        return this;
+    }
+
     /**
      * Timeout to drain the socket on close.
      *
diff --git a/bookkeeper-server/src/main/java/org/apache/bookkeeper/proto/BookieNettyServer.java b/bookkeeper-server/src/main/java/org/apache/bookkeeper/proto/BookieNettyServer.java
index d687a5c98a..1cbb345a25 100644
--- a/bookkeeper-server/src/main/java/org/apache/bookkeeper/proto/BookieNettyServer.java
+++ b/bookkeeper-server/src/main/java/org/apache/bookkeeper/proto/BookieNettyServer.java
@@ -112,7 +112,7 @@
 
         if (!conf.isDisableServerSocketBind()) {
             ThreadFactory threadFactory = new DefaultThreadFactory("bookie-io");
-            final int numThreads = Runtime.getRuntime().availableProcessors() * 2;
+            final int numThreads = conf.getServerNumIOThreads();
 
             EventLoopGroup eventLoopGroup;
             if (SystemUtils.IS_OS_LINUX) {


 

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services