HBASE-26304 Reflect out of band locality improvements in metrics and balancer (#3803)
[hbase.git] / hbase-server / src / main / java / org / apache / hadoop / hbase / util / NettyEventLoopGroupConfig.java
blob346f3df51834804bf5d4ff198ef05e47ec20cc79
1 /**
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org.apache.hadoop.hbase.util;
20 import org.apache.hbase.thirdparty.io.netty.channel.Channel;
21 import org.apache.hbase.thirdparty.io.netty.channel.EventLoopGroup;
22 import org.apache.hbase.thirdparty.io.netty.channel.ServerChannel;
23 import org.apache.hbase.thirdparty.io.netty.channel.epoll.EpollEventLoopGroup;
24 import org.apache.hbase.thirdparty.io.netty.channel.epoll.EpollServerSocketChannel;
25 import org.apache.hbase.thirdparty.io.netty.channel.epoll.EpollSocketChannel;
26 import org.apache.hbase.thirdparty.io.netty.channel.nio.NioEventLoopGroup;
27 import org.apache.hbase.thirdparty.io.netty.channel.socket.nio.NioServerSocketChannel;
28 import org.apache.hbase.thirdparty.io.netty.channel.socket.nio.NioSocketChannel;
29 import org.apache.hbase.thirdparty.io.netty.util.concurrent.DefaultThreadFactory;
30 import java.util.concurrent.ThreadFactory;
31 import org.apache.hadoop.conf.Configuration;
32 import org.apache.hadoop.hbase.ipc.NettyRpcClientConfigHelper;
33 import org.apache.hadoop.hbase.wal.NettyAsyncFSWALConfigHelper;
34 import org.apache.yetus.audience.InterfaceAudience;
36 /**
37 * Event loop group related config.
39 @InterfaceAudience.Private
40 public class NettyEventLoopGroupConfig {
41 private final EventLoopGroup group;
43 private final Class<? extends ServerChannel> serverChannelClass;
45 private final Class<? extends Channel> clientChannelClass;
47 private static boolean useEpoll(Configuration conf) {
48 // Config to enable native transport.
49 boolean epollEnabled = conf.getBoolean("hbase.netty.nativetransport", true);
50 // Use the faster native epoll transport mechanism on linux if enabled
51 return epollEnabled && JVM.isLinux() && JVM.isAmd64();
54 public NettyEventLoopGroupConfig(Configuration conf, String threadPoolName) {
55 boolean useEpoll = useEpoll(conf);
56 int workerCount = conf.getInt("hbase.netty.worker.count", 0);
57 ThreadFactory eventLoopThreadFactory =
58 new DefaultThreadFactory(threadPoolName, true, Thread.MAX_PRIORITY);
59 if (useEpoll) {
60 group = new EpollEventLoopGroup(workerCount, eventLoopThreadFactory);
61 serverChannelClass = EpollServerSocketChannel.class;
62 clientChannelClass = EpollSocketChannel.class;
63 } else {
64 group = new NioEventLoopGroup(workerCount, eventLoopThreadFactory);
65 serverChannelClass = NioServerSocketChannel.class;
66 clientChannelClass = NioSocketChannel.class;
70 public EventLoopGroup group() {
71 return group;
74 public Class<? extends ServerChannel> serverChannelClass() {
75 return serverChannelClass;
78 public Class<? extends Channel> clientChannelClass() {
79 return clientChannelClass;
82 public static NettyEventLoopGroupConfig setup(Configuration conf, String threadPoolName) {
83 // Initialize netty event loop group at start as we may use it for rpc server, rpc client & WAL.
84 NettyEventLoopGroupConfig nelgc = new NettyEventLoopGroupConfig(conf, threadPoolName);
85 NettyRpcClientConfigHelper.setEventLoopConfig(conf, nelgc.group(), nelgc.clientChannelClass());
86 NettyAsyncFSWALConfigHelper.setEventLoopConfig(conf, nelgc.group(), nelgc.clientChannelClass());
87 return nelgc;