@InterfaceAudience.Private public abstract class AbstractRpcClient<T extends RpcConnection> extends Object implements RpcClient
Locking schema of the current IPC implementation
AbstractRpcClient to protect the fetching or creating
connection.Call to make sure that we can only finish the call once.HBaseRpcController as Call. And see the comment of
HBaseRpcController.notifyOnCancel(RpcCallback, HBaseRpcController.CancellationCallback)
of how to deal with cancel.Call and HBaseRpcController's lock should be held
at last. So the callbacks in Call and HBaseRpcController should be execute
outside the lock in Call and HBaseRpcController which means the implementations
of the callbacks are free to hold any lock.| Modifier and Type | Class and Description |
|---|---|
private static class |
AbstractRpcClient.AbstractRpcChannel |
static class |
AbstractRpcClient.BlockingRpcChannelImplementation
Blocking rpc channel that goes via hbase rpc.
|
static class |
AbstractRpcClient.RpcChannelImplementation
Async rpc channel that goes via hbase rpc.
|
| Modifier and Type | Field and Description |
|---|---|
private AtomicInteger |
callIdCnt |
protected CellBlockBuilder |
cellBlockBuilder |
private ScheduledFuture<?> |
cleanupIdleConnectionTask |
protected String |
clusterId |
protected Codec |
codec |
protected org.apache.hadoop.io.compress.CompressionCodec |
compressor |
private static org.apache.hbase.thirdparty.com.google.common.cache.LoadingCache<InetSocketAddress,AtomicInteger> |
concurrentCounterCache |
protected org.apache.hadoop.conf.Configuration |
conf |
protected PoolMap<ConnectionId,T> |
connections |
protected int |
connectTO |
protected FailedServers |
failedServers |
protected long |
failureSleep |
protected boolean |
fallbackAllowed |
private static ScheduledExecutorService |
IDLE_CONN_SWEEPER |
protected SocketAddress |
localAddr |
static org.slf4j.Logger |
LOG |
private int |
maxConcurrentCallsPerServer |
protected int |
maxRetries |
protected MetricsConnection |
metrics |
protected int |
minIdleTimeBeforeClose |
protected int |
readTO |
protected boolean |
running |
protected boolean |
tcpKeepAlive |
protected boolean |
tcpNoDelay |
protected UserProvider |
userProvider |
protected static org.apache.hbase.thirdparty.io.netty.util.HashedWheelTimer |
WHEEL_TIMER |
protected int |
writeTO |
DEFAULT_CODEC_CLASS, DEFAULT_SOCKET_TIMEOUT_CONNECT, DEFAULT_SOCKET_TIMEOUT_READ, DEFAULT_SOCKET_TIMEOUT_WRITE, FAILED_SERVER_EXPIRY_DEFAULT, FAILED_SERVER_EXPIRY_KEY, IDLE_TIME, IPC_CLIENT_FALLBACK_TO_SIMPLE_AUTH_ALLOWED_DEFAULT, IPC_CLIENT_FALLBACK_TO_SIMPLE_AUTH_ALLOWED_KEY, PING_CALL_ID, SOCKET_TIMEOUT_CONNECT, SOCKET_TIMEOUT_READ, SOCKET_TIMEOUT_WRITE, SPECIFIC_WRITE_THREAD| Constructor and Description |
|---|
AbstractRpcClient(org.apache.hadoop.conf.Configuration conf,
String clusterId,
SocketAddress localAddr,
MetricsConnection metrics)
Construct an IPC client for the cluster
clusterId |
| Modifier and Type | Method and Description |
|---|---|
private org.apache.hbase.thirdparty.com.google.protobuf.Message |
callBlockingMethod(org.apache.hbase.thirdparty.com.google.protobuf.Descriptors.MethodDescriptor md,
HBaseRpcController hrc,
org.apache.hbase.thirdparty.com.google.protobuf.Message param,
org.apache.hbase.thirdparty.com.google.protobuf.Message returnType,
User ticket,
InetSocketAddress isa)
Make a blocking call.
|
(package private) Call |
callMethod(org.apache.hbase.thirdparty.com.google.protobuf.Descriptors.MethodDescriptor md,
HBaseRpcController hrc,
org.apache.hbase.thirdparty.com.google.protobuf.Message param,
org.apache.hbase.thirdparty.com.google.protobuf.Message returnType,
User ticket,
InetSocketAddress addr,
org.apache.hbase.thirdparty.com.google.protobuf.RpcCallback<org.apache.hbase.thirdparty.com.google.protobuf.Message> callback) |
void |
cancelConnections(ServerName sn)
Interrupt the connections to the given ip:port server.
|
private void |
cleanupIdleConnections() |
void |
close()
Stop all threads related to this client.
|
protected abstract void |
closeInternal() |
(package private) static HBaseRpcController |
configureHBaseRpcController(org.apache.hbase.thirdparty.com.google.protobuf.RpcController controller,
int channelOperationTimeout)
Configure an hbase rpccontroller
|
(package private) InetSocketAddress |
createAddr(ServerName sn) |
org.apache.hbase.thirdparty.com.google.protobuf.BlockingRpcChannel |
createBlockingRpcChannel(ServerName sn,
User ticket,
int rpcTimeout)
Creates a "channel" that can be used by a blocking protobuf service.
|
protected abstract T |
createConnection(ConnectionId remoteId)
Not connected.
|
org.apache.hbase.thirdparty.com.google.protobuf.RpcChannel |
createRpcChannel(ServerName sn,
User user,
int rpcTimeout)
Creates a "channel" that can be used by a protobuf service.
|
(package private) Codec |
getCodec()
Encapsulate the ugly casting and RuntimeException conversion in private method.
|
private static org.apache.hadoop.io.compress.CompressionCodec |
getCompressor(org.apache.hadoop.conf.Configuration conf)
Encapsulate the ugly casting and RuntimeException conversion in private method.
|
private T |
getConnection(ConnectionId remoteId)
Get a connection from the pool, or create a new one and add it to the pool.
|
static String |
getDefaultCodec(org.apache.hadoop.conf.Configuration c) |
private static int |
getPoolSize(org.apache.hadoop.conf.Configuration config)
Return the pool size specified in the configuration, which is applicable only if the pool type
is
PoolMap.PoolType.RoundRobin. |
private static PoolMap.PoolType |
getPoolType(org.apache.hadoop.conf.Configuration config)
Return the pool type specified in the configuration, which must be set to either
PoolMap.PoolType.RoundRobin or
PoolMap.PoolType.ThreadLocal, otherwise default to the
former. |
boolean |
hasCellBlockSupport() |
(package private) boolean |
isTcpNoDelay() |
private int |
nextCallId() |
private void |
onCallFinished(Call call,
HBaseRpcController hrc,
InetSocketAddress addr,
org.apache.hbase.thirdparty.com.google.protobuf.RpcCallback<org.apache.hbase.thirdparty.com.google.protobuf.Message> callback) |
public static final org.slf4j.Logger LOG
protected static final org.apache.hbase.thirdparty.io.netty.util.HashedWheelTimer WHEEL_TIMER
private static final ScheduledExecutorService IDLE_CONN_SWEEPER
protected boolean running
protected final org.apache.hadoop.conf.Configuration conf
protected final SocketAddress localAddr
protected final MetricsConnection metrics
protected final UserProvider userProvider
protected final CellBlockBuilder cellBlockBuilder
protected final int minIdleTimeBeforeClose
protected final int maxRetries
protected final long failureSleep
protected final boolean tcpNoDelay
protected final boolean tcpKeepAlive
protected final org.apache.hadoop.io.compress.CompressionCodec compressor
protected final boolean fallbackAllowed
protected final FailedServers failedServers
protected final int connectTO
protected final int readTO
protected final int writeTO
protected final PoolMap<ConnectionId,T extends RpcConnection> connections
private final AtomicInteger callIdCnt
private final ScheduledFuture<?> cleanupIdleConnectionTask
private int maxConcurrentCallsPerServer
private static final org.apache.hbase.thirdparty.com.google.common.cache.LoadingCache<InetSocketAddress,AtomicInteger> concurrentCounterCache
public AbstractRpcClient(org.apache.hadoop.conf.Configuration conf, String clusterId, SocketAddress localAddr, MetricsConnection metrics)
clusterIdconf - configurationclusterId - the cluster idlocalAddr - client socket bind address.metrics - the connection metricsprivate void cleanupIdleConnections()
public static String getDefaultCodec(org.apache.hadoop.conf.Configuration c)
Codec getCodec()
public boolean hasCellBlockSupport()
hasCellBlockSupport in interface RpcClientCodec and so
supports cell blocks.boolean isTcpNoDelay()
private static org.apache.hadoop.io.compress.CompressionCodec getCompressor(org.apache.hadoop.conf.Configuration conf)
conf - configurationprivate static PoolMap.PoolType getPoolType(org.apache.hadoop.conf.Configuration config)
PoolMap.PoolType.RoundRobin or
PoolMap.PoolType.ThreadLocal, otherwise default to the
former. For applications with many user threads, use a small round-robin pool. For applications
with few user threads, you may want to try using a thread-local pool. In any case, the number
of RpcClient instances should not exceed the operating
system's hard limit on the number of connections.config - configurationPoolMap.PoolType.RoundRobin or
PoolMap.PoolType.ThreadLocalprivate static int getPoolSize(org.apache.hadoop.conf.Configuration config)
PoolMap.PoolType.RoundRobin.config - configurationprivate int nextCallId()
private org.apache.hbase.thirdparty.com.google.protobuf.Message callBlockingMethod(org.apache.hbase.thirdparty.com.google.protobuf.Descriptors.MethodDescriptor md, HBaseRpcController hrc, org.apache.hbase.thirdparty.com.google.protobuf.Message param, org.apache.hbase.thirdparty.com.google.protobuf.Message returnType, User ticket, InetSocketAddress isa) throws org.apache.hbase.thirdparty.com.google.protobuf.ServiceException
ticket - Be careful which ticket you pass. A new user will mean a new Connection.
UserProvider.getCurrent() makes a new instance of User each time so will be a
new Connection each time.org.apache.hbase.thirdparty.com.google.protobuf.ServiceExceptionprivate T getConnection(ConnectionId remoteId) throws IOException
IOExceptionprotected abstract T createConnection(ConnectionId remoteId) throws IOException
IOExceptionprivate void onCallFinished(Call call, HBaseRpcController hrc, InetSocketAddress addr, org.apache.hbase.thirdparty.com.google.protobuf.RpcCallback<org.apache.hbase.thirdparty.com.google.protobuf.Message> callback)
Call callMethod(org.apache.hbase.thirdparty.com.google.protobuf.Descriptors.MethodDescriptor md, HBaseRpcController hrc, org.apache.hbase.thirdparty.com.google.protobuf.Message param, org.apache.hbase.thirdparty.com.google.protobuf.Message returnType, User ticket, InetSocketAddress addr, org.apache.hbase.thirdparty.com.google.protobuf.RpcCallback<org.apache.hbase.thirdparty.com.google.protobuf.Message> callback)
InetSocketAddress createAddr(ServerName sn) throws UnknownHostException
UnknownHostExceptionpublic void cancelConnections(ServerName sn)
cancelConnections in interface RpcClientsn - server location to cancel connections ofstatic HBaseRpcController configureHBaseRpcController(org.apache.hbase.thirdparty.com.google.protobuf.RpcController controller, int channelOperationTimeout)
controller - to configurechannelOperationTimeout - timeout for operationprotected abstract void closeInternal()
public void close()
RpcClientpublic org.apache.hbase.thirdparty.com.google.protobuf.BlockingRpcChannel createBlockingRpcChannel(ServerName sn, User ticket, int rpcTimeout) throws UnknownHostException
RpcClientcreateBlockingRpcChannel in interface RpcClientsn - server name describing location of serverticket - which is to use the connectionrpcTimeout - default rpc operation timeoutUnknownHostExceptionpublic org.apache.hbase.thirdparty.com.google.protobuf.RpcChannel createRpcChannel(ServerName sn, User user, int rpcTimeout) throws UnknownHostException
RpcClientcreateRpcChannel in interface RpcClientsn - server name describing location of serveruser - which is to use the connectionrpcTimeout - default rpc operation timeoutUnknownHostExceptionCopyright © 2007–2020 The Apache Software Foundation. All rights reserved.