You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@tajo.apache.org by hy...@apache.org on 2013/12/30 17:37:59 UTC

git commit: TAJO-467: Too many open FD when master failed. (hyoungjunkim via hyunsik)

Updated Branches:
  refs/heads/master c7990c724 -> 33606c802


TAJO-467: Too many open FD when master failed. (hyoungjunkim via hyunsik)


Project: http://git-wip-us.apache.org/repos/asf/incubator-tajo/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-tajo/commit/33606c80
Tree: http://git-wip-us.apache.org/repos/asf/incubator-tajo/tree/33606c80
Diff: http://git-wip-us.apache.org/repos/asf/incubator-tajo/diff/33606c80

Branch: refs/heads/master
Commit: 33606c80268e59693412aa138f27c927b770d43a
Parents: c7990c7
Author: Hyunsik Choi <hy...@apache.org>
Authored: Tue Dec 31 01:32:59 2013 +0900
Committer: Hyunsik Choi <hy...@apache.org>
Committed: Tue Dec 31 01:32:59 2013 +0900

----------------------------------------------------------------------
 CHANGES.txt                                     |  2 +
 .../org/apache/tajo/rpc/NettyClientBase.java    | 61 ++++++++++++--------
 2 files changed, 38 insertions(+), 25 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-tajo/blob/33606c80/CHANGES.txt
----------------------------------------------------------------------
diff --git a/CHANGES.txt b/CHANGES.txt
index fc92fc0..0b1765f 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -182,6 +182,8 @@ Release 0.8.0 - unreleased
 
   BUG FIXES
 
+    TAJO-467: Too many open FD when master failed. (hyoungjunkim via hyunsik)
+
     TAJO-455: Throw PlanningException when Creating table with Partition
     exception COLUMN. (DaeMyung Kang via hyunsik)
 

http://git-wip-us.apache.org/repos/asf/incubator-tajo/blob/33606c80/tajo-rpc/src/main/java/org/apache/tajo/rpc/NettyClientBase.java
----------------------------------------------------------------------
diff --git a/tajo-rpc/src/main/java/org/apache/tajo/rpc/NettyClientBase.java b/tajo-rpc/src/main/java/org/apache/tajo/rpc/NettyClientBase.java
index a03396e..b41bfdd 100644
--- a/tajo-rpc/src/main/java/org/apache/tajo/rpc/NettyClientBase.java
+++ b/tajo-rpc/src/main/java/org/apache/tajo/rpc/NettyClientBase.java
@@ -28,6 +28,7 @@ import org.jboss.netty.channel.socket.ClientSocketChannelFactory;
 import org.jboss.netty.channel.socket.nio.NioClientSocketChannelFactory;
 
 import java.io.Closeable;
+import java.io.IOException;
 import java.net.InetSocketAddress;
 import java.util.concurrent.Executors;
 
@@ -46,29 +47,34 @@ public abstract class NettyClientBase implements Closeable {
   public abstract <T> T getStub();
   public abstract RpcConnectionPool.RpcConnectionKey getKey();
 
-  public void init(InetSocketAddress addr, ChannelPipelineFactory pipeFactory) {
+  public void init(InetSocketAddress addr, ChannelPipelineFactory pipeFactory) throws IOException {
     this.addr = addr;
 
-    this.factory =
-        new NioClientSocketChannelFactory(Executors.newCachedThreadPool(),
-            Executors.newCachedThreadPool());
-
-    this.bootstrap = new ClientBootstrap(factory);
-    this.bootstrap.setPipelineFactory(pipeFactory);
-    // TODO - should be configurable
-    this.bootstrap.setOption("connectTimeoutMillis", 10000);
-    this.bootstrap.setOption("connectResponseTimeoutMillis", 10000);
-    this.bootstrap.setOption("receiveBufferSize", 1048576*2);
-    this.bootstrap.setOption("tcpNoDelay", false);
-    this.bootstrap.setOption("keepAlive", true);
-
-    this.channelFuture = bootstrap.connect(addr);
-    this.channelFuture.awaitUninterruptibly();
-    if (!channelFuture.isSuccess()) {
-      channelFuture.getCause().printStackTrace();
-      throw new RuntimeException(channelFuture.getCause());
+    try {
+      this.factory =
+          new NioClientSocketChannelFactory(Executors.newCachedThreadPool(),
+              Executors.newCachedThreadPool());
+
+      this.bootstrap = new ClientBootstrap(factory);
+      this.bootstrap.setPipelineFactory(pipeFactory);
+      // TODO - should be configurable
+      this.bootstrap.setOption("connectTimeoutMillis", 10000);
+      this.bootstrap.setOption("connectResponseTimeoutMillis", 10000);
+      this.bootstrap.setOption("receiveBufferSize", 1048576*2);
+      this.bootstrap.setOption("tcpNoDelay", false);
+      this.bootstrap.setOption("keepAlive", true);
+
+      this.channelFuture = bootstrap.connect(addr);
+      this.channelFuture.awaitUninterruptibly();
+      if (!channelFuture.isSuccess()) {
+        channelFuture.getCause().printStackTrace();
+        throw new RuntimeException(channelFuture.getCause());
+      }
+      this.channel = channelFuture.getChannel();
+    } catch (Throwable t) {
+      close();
+      throw new IOException(t.getCause());
     }
-    this.channel = channelFuture.getChannel();
   }
 
   public boolean isConnected() {
@@ -85,11 +91,16 @@ public abstract class NettyClientBase implements Closeable {
 
   @Override
   public void close() {
-    this.channel.close().awaitUninterruptibly();
-    this.bootstrap.releaseExternalResources();
-    if(LOG.isDebugEnabled()) {
-      LOG.debug("Proxy is disconnected from " +
-          addr.getAddress().getHostAddress() + ":" + addr.getPort());
+    if(this.channel != null) {
+      this.channel.close().awaitUninterruptibly();
+    }
+
+    if(this.bootstrap != null) {
+      this.bootstrap.releaseExternalResources();
+      if(LOG.isDebugEnabled()) {
+        LOG.debug("Proxy is disconnected from " +
+            addr.getAddress().getHostAddress() + ":" + addr.getPort());
+      }
     }
   }
 }