Repository: flume Updated Branches: refs/heads/trunk 1dfcb4b0e -> 47507bc40
FLUME-2243. Avro Source should name threads instead of using generic names. (Ashish Paliwal via Hari Shreedharan) Project: http://git-wip-us.apache.org/repos/asf/flume/repo Commit: http://git-wip-us.apache.org/repos/asf/flume/commit/47507bc4 Tree: http://git-wip-us.apache.org/repos/asf/flume/tree/47507bc4 Diff: http://git-wip-us.apache.org/repos/asf/flume/diff/47507bc4 Branch: refs/heads/trunk Commit: 47507bc40202c52e7156106642ffc0ab849fb96e Parents: 1dfcb4b Author: Hari Shreedharan <hshreedha...@apache.org> Authored: Tue Mar 11 13:05:51 2014 -0700 Committer: Hari Shreedharan <hshreedha...@apache.org> Committed: Tue Mar 11 13:07:18 2014 -0700 ---------------------------------------------------------------------- .../java/org/apache/flume/source/AvroSource.java | 19 ++++++++++++++++--- 1 file changed, 16 insertions(+), 3 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/flume/blob/47507bc4/flume-ng-core/src/main/java/org/apache/flume/source/AvroSource.java ---------------------------------------------------------------------- diff --git a/flume-ng-core/src/main/java/org/apache/flume/source/AvroSource.java b/flume-ng-core/src/main/java/org/apache/flume/source/AvroSource.java index c1ee3a9..3eef687 100644 --- a/flume-ng-core/src/main/java/org/apache/flume/source/AvroSource.java +++ b/flume-ng-core/src/main/java/org/apache/flume/source/AvroSource.java @@ -35,7 +35,10 @@ import java.util.concurrent.TimeUnit; import javax.net.ssl.KeyManagerFactory; import javax.net.ssl.SSLContext; import javax.net.ssl.SSLEngine; + +import com.google.common.util.concurrent.ThreadFactoryBuilder; import org.apache.avro.ipc.NettyServer; +import org.apache.avro.ipc.NettyTransceiver; import org.apache.avro.ipc.Responder; import org.apache.avro.ipc.Server; import org.apache.avro.ipc.specific.SpecificResponder; @@ -246,11 +249,21 @@ public class AvroSource extends AbstractSource implements EventDrivenSource, NioServerSocketChannelFactory socketChannelFactory; if (maxThreads <= 0) { socketChannelFactory = new NioServerSocketChannelFactory - (Executors .newCachedThreadPool(), Executors.newCachedThreadPool()); + (Executors.newCachedThreadPool(new ThreadFactoryBuilder(). + setNameFormat("Avro " + NettyTransceiver.class.getSimpleName() + + " Boss-%d").build()), + Executors.newCachedThreadPool(new ThreadFactoryBuilder(). + setNameFormat("Avro " + NettyTransceiver.class.getSimpleName() + + " I/O Worker-%d").build())); } else { socketChannelFactory = new NioServerSocketChannelFactory( - Executors.newCachedThreadPool(), - Executors.newFixedThreadPool(maxThreads)); + Executors.newCachedThreadPool(new ThreadFactoryBuilder(). + setNameFormat( + "Avro " + NettyTransceiver.class.getSimpleName() + + " Boss-%d").build()), + Executors.newFixedThreadPool(maxThreads, new ThreadFactoryBuilder(). + setNameFormat("Avro " + NettyTransceiver.class.getSimpleName() + + " I/O Worker-%d").build())); } return socketChannelFactory; }