org.apache.hadoop.hbase.ipc.BlockingRpcConnection Maven / Gradle / Ivy
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hbase.ipc;
import static org.apache.hadoop.hbase.ipc.IPCUtil.buildRequestHeader;
import static org.apache.hadoop.hbase.ipc.IPCUtil.createRemoteException;
import static org.apache.hadoop.hbase.ipc.IPCUtil.getTotalSizeWhenWrittenDelimited;
import static org.apache.hadoop.hbase.ipc.IPCUtil.isFatalConnectionException;
import static org.apache.hadoop.hbase.ipc.IPCUtil.setCancelled;
import static org.apache.hadoop.hbase.ipc.IPCUtil.write;
import io.opentelemetry.context.Scope;
import java.io.BufferedInputStream;
import java.io.BufferedOutputStream;
import java.io.DataInputStream;
import java.io.DataOutputStream;
import java.io.IOException;
import java.io.InputStream;
import java.io.InterruptedIOException;
import java.io.OutputStream;
import java.net.InetSocketAddress;
import java.net.Socket;
import java.net.SocketTimeoutException;
import java.security.PrivilegedExceptionAction;
import java.util.ArrayDeque;
import java.util.Locale;
import java.util.Queue;
import java.util.concurrent.ConcurrentHashMap;
import java.util.concurrent.ConcurrentMap;
import java.util.concurrent.ThreadLocalRandom;
import java.util.concurrent.atomic.AtomicInteger;
import javax.security.sasl.SaslException;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.CellScanner;
import org.apache.hadoop.hbase.DoNotRetryIOException;
import org.apache.hadoop.hbase.exceptions.ConnectionClosingException;
import org.apache.hadoop.hbase.io.ByteArrayOutputStream;
import org.apache.hadoop.hbase.ipc.HBaseRpcController.CancellationCallback;
import org.apache.hadoop.hbase.log.HBaseMarkers;
import org.apache.hadoop.hbase.security.HBaseSaslRpcClient;
import org.apache.hadoop.hbase.security.SaslUtil;
import org.apache.hadoop.hbase.security.SaslUtil.QualityOfProtection;
import org.apache.hadoop.hbase.security.provider.SaslClientAuthenticationProvider;
import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
import org.apache.hadoop.hbase.util.ExceptionUtil;
import org.apache.hadoop.io.IOUtils;
import org.apache.hadoop.ipc.RemoteException;
import org.apache.hadoop.net.NetUtils;
import org.apache.hadoop.security.UserGroupInformation;
import org.apache.hadoop.util.StringUtils;
import org.apache.yetus.audience.InterfaceAudience;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import org.apache.hbase.thirdparty.com.google.protobuf.Message;
import org.apache.hbase.thirdparty.com.google.protobuf.RpcCallback;
import org.apache.hbase.thirdparty.io.netty.buffer.ByteBuf;
import org.apache.hbase.thirdparty.io.netty.buffer.PooledByteBufAllocator;
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
import org.apache.hadoop.hbase.shaded.protobuf.generated.RPCProtos;
import org.apache.hadoop.hbase.shaded.protobuf.generated.RPCProtos.CellBlockMeta;
import org.apache.hadoop.hbase.shaded.protobuf.generated.RPCProtos.ConnectionHeader;
import org.apache.hadoop.hbase.shaded.protobuf.generated.RPCProtos.ExceptionResponse;
import org.apache.hadoop.hbase.shaded.protobuf.generated.RPCProtos.RequestHeader;
import org.apache.hadoop.hbase.shaded.protobuf.generated.RPCProtos.ResponseHeader;
/**
* Thread that reads responses and notifies callers. Each connection owns a socket connected to a
* remote address. Calls are multiplexed through this socket: responses may be delivered out of
* order.
*/
@InterfaceAudience.Private
class BlockingRpcConnection extends RpcConnection implements Runnable {
private static final Logger LOG = LoggerFactory.getLogger(BlockingRpcConnection.class);
private final BlockingRpcClient rpcClient;
private final String threadName;
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value = "IS2_INCONSISTENT_SYNC",
justification = "We are always under lock actually")
private Thread thread;
// Used for ensuring two reader threads don't run over each other. Should only be used
// in reader thread run() method, to avoid deadlocks with synchronization on BlockingRpcConnection
private final Object readerThreadLock = new Object();
// Used to suffix the threadName in a way that we can differentiate them in logs/thread dumps.
private final AtomicInteger attempts = new AtomicInteger();
// connected socket. protected for writing UT.
protected Socket socket = null;
private DataInputStream in;
private DataOutputStream out;
private HBaseSaslRpcClient saslRpcClient;
// currently active calls
private final ConcurrentMap calls = new ConcurrentHashMap<>();
private final CallSender callSender;
private boolean closed = false;
private byte[] connectionHeaderPreamble;
private byte[] connectionHeaderWithLength;
private boolean waitingConnectionHeaderResponse = false;
/**
* If the client wants to interrupt its calls easily (i.e. call Thread#interrupt), it gets into a
* java issue: an interruption during a write closes the socket/channel. A way to avoid this is to
* use a different thread for writing. This way, on interruptions, we either cancel the writes or
* ignore the answer if the write is already done, but we don't stop the write in the middle. This
* adds a thread per region server in the client, so it's kept as an option.
*
* The implementation is simple: the client threads adds their call to the queue, and then wait
* for an answer. The CallSender blocks on the queue, and writes the calls one after the other. On
* interruption, the client cancels its call. The CallSender checks that the call has not been
* canceled before writing it.
*
* When the connection closes, all the calls not yet sent are dismissed. The client thread is
* notified with an appropriate exception, as if the call was already sent but the answer not yet
* received.
*
*/
private class CallSender extends Thread {
private final Queue callsToWrite;
private final int maxQueueSize;
public CallSender(String name, Configuration conf) {
int queueSize = conf.getInt("hbase.ipc.client.write.queueSize", 1000);
callsToWrite = new ArrayDeque<>(queueSize);
this.maxQueueSize = queueSize;
setDaemon(true);
setName(name + " - writer");
}
public void sendCall(final Call call) throws IOException {
if (callsToWrite.size() >= maxQueueSize) {
throw new IOException("Can't add " + call.toShortString()
+ " to the write queue. callsToWrite.size()=" + callsToWrite.size());
}
callsToWrite.offer(call);
BlockingRpcConnection.this.notifyAll();
}
public void remove(Call call) {
callsToWrite.remove(call);
// By removing the call from the expected call list, we make the list smaller, but
// it means as well that we don't know how many calls we cancelled.
calls.remove(call.id);
call.setException(new CallCancelledException(call.toShortString() + ", waitTime="
+ (EnvironmentEdgeManager.currentTime() - call.getStartTime()) + ", rpcTimeout="
+ call.timeout));
}
/**
* Reads the call from the queue, write them on the socket.
*/
@Override
public void run() {
synchronized (BlockingRpcConnection.this) {
while (!closed) {
if (callsToWrite.isEmpty()) {
// We should use another monitor object here for better performance since the read
// thread also uses ConnectionImpl.this. But this makes the locking schema more
// complicated, can do it later as an optimization.
try {
BlockingRpcConnection.this.wait();
} catch (InterruptedException e) {
// Restore interrupt status
Thread.currentThread().interrupt();
}
// check if we need to quit, so continue the main loop instead of fallback.
continue;
}
Call call = callsToWrite.poll();
if (call.isDone()) {
continue;
}
try (Scope scope = call.span.makeCurrent()) {
writeRequest(call);
} catch (IOException e) {
// exception here means the call has not been added to the pendingCalls yet, so we need
// to fail it by our own.
LOG.debug("call write error for {}", call.toShortString());
call.setException(e);
closeConn(e);
}
}
}
}
/**
* Cleans the call not yet sent when we finish.
*/
public void cleanup(IOException e) {
IOException ie =
new ConnectionClosingException("Connection to " + remoteId.getAddress() + " is closing.");
for (Call call : callsToWrite) {
call.setException(ie);
}
callsToWrite.clear();
}
}
BlockingRpcConnection(BlockingRpcClient rpcClient, ConnectionId remoteId) throws IOException {
super(rpcClient.conf, AbstractRpcClient.WHEEL_TIMER, remoteId, rpcClient.clusterId,
rpcClient.userProvider.isHBaseSecurityEnabled(), rpcClient.codec, rpcClient.compressor,
rpcClient.metrics);
this.rpcClient = rpcClient;
this.connectionHeaderPreamble = getConnectionHeaderPreamble();
ConnectionHeader header = getConnectionHeader();
ByteArrayOutputStream baos = new ByteArrayOutputStream(4 + header.getSerializedSize());
DataOutputStream dos = new DataOutputStream(baos);
dos.writeInt(header.getSerializedSize());
header.writeTo(dos);
assert baos.size() == 4 + header.getSerializedSize();
this.connectionHeaderWithLength = baos.getBuffer();
UserGroupInformation ticket = remoteId.ticket.getUGI();
this.threadName = "BRPC Connection (" + this.rpcClient.socketFactory.hashCode() + ") to "
+ remoteId.getAddress().toString()
+ ((ticket == null) ? " from an unknown user" : (" from " + ticket.getUserName()));
if (this.rpcClient.conf.getBoolean(BlockingRpcClient.SPECIFIC_WRITE_THREAD, false)) {
callSender = new CallSender(threadName, this.rpcClient.conf);
callSender.start();
} else {
callSender = null;
}
}
// protected for write UT.
protected void setupConnection() throws IOException {
short ioFailures = 0;
short timeoutFailures = 0;
while (true) {
try {
this.socket = this.rpcClient.socketFactory.createSocket();
this.socket.setTcpNoDelay(this.rpcClient.isTcpNoDelay());
this.socket.setKeepAlive(this.rpcClient.tcpKeepAlive);
if (this.rpcClient.localAddr != null) {
this.socket.bind(this.rpcClient.localAddr);
}
InetSocketAddress remoteAddr = getRemoteInetAddress(rpcClient.metrics);
NetUtils.connect(this.socket, remoteAddr, this.rpcClient.connectTO);
this.socket.setSoTimeout(this.rpcClient.readTO);
return;
} catch (SocketTimeoutException toe) {
/*
* The max number of retries is 45, which amounts to 20s*45 = 15 minutes retries.
*/
if (LOG.isDebugEnabled()) {
LOG.debug(
"Received exception in connection setup.\n" + StringUtils.stringifyException(toe));
}
handleConnectionFailure(timeoutFailures++, this.rpcClient.maxRetries, toe);
} catch (IOException ie) {
if (LOG.isDebugEnabled()) {
LOG.debug(
"Received exception in connection setup.\n" + StringUtils.stringifyException(ie));
}
handleConnectionFailure(ioFailures++, this.rpcClient.maxRetries, ie);
}
}
}
/**
* Handle connection failures If the current number of retries is equal to the max number of
* retries, stop retrying and throw the exception; Otherwise backoff N seconds and try connecting
* again. This Method is only called from inside setupIOstreams(), which is synchronized. Hence
* the sleep is synchronized; the locks will be retained.
* @param curRetries current number of retries
* @param maxRetries max number of retries allowed
* @param ioe failure reason
* @throws IOException if max number of retries is reached
*/
private void handleConnectionFailure(int curRetries, int maxRetries, IOException ioe)
throws IOException {
closeSocket();
// throw the exception if the maximum number of retries is reached
if (curRetries >= maxRetries || ExceptionUtil.isInterrupt(ioe)) {
throw ioe;
}
// otherwise back off and retry
try {
Thread.sleep(this.rpcClient.failureSleep);
} catch (InterruptedException ie) {
ExceptionUtil.rethrowIfInterrupt(ie);
}
if (LOG.isInfoEnabled()) {
LOG.info("Retrying connect to server: " + remoteId.getAddress() + " after sleeping "
+ this.rpcClient.failureSleep + "ms. Already tried " + curRetries + " time(s).");
}
}
/*
* wait till someone signals us to start reading RPC response or it is idle too long, it is marked
* as to be closed, or the client is marked as not running.
* @return true if it is time to read a response; false otherwise.
*/
private synchronized boolean waitForWork() {
// beware of the concurrent access to the calls list: we can add calls, but as well
// remove them.
long waitUntil = EnvironmentEdgeManager.currentTime() + this.rpcClient.minIdleTimeBeforeClose;
for (;;) {
if (thread == null) {
return false;
}
// If closeConn is called while we are in the readResponse method, it's possible that a new
// call to setupIOStreams comes in and creates a new value for "thread" before readResponse
// finishes. Once readResponse finishes, it will come in here and thread will be non-null
// above, but pointing at a new thread. In that case, we should end to avoid a situation
// where two threads are forever competing for the same socket.
if (!isCurrentThreadExpected()) {
LOG.debug("Thread replaced by new connection thread. Ending waitForWork loop.");
return false;
}
if (!calls.isEmpty()) {
return true;
}
if (EnvironmentEdgeManager.currentTime() >= waitUntil) {
closeConn(
new IOException("idle connection closed with " + calls.size() + " pending request(s)"));
return false;
}
try {
wait(Math.min(this.rpcClient.minIdleTimeBeforeClose, 1000));
} catch (InterruptedException e) {
// Restore interrupt status
Thread.currentThread().interrupt();
String msg = "Interrupted while waiting for work";
// If we were interrupted by closeConn, it would have set thread to null.
// We are synchronized here and if we somehow got interrupted without setting thread to
// null, we want to make sure the connection is closed since the read thread would be dead.
// Rather than do a null check here, we check if the current thread is the expected thread.
// This guards against the case where a call to setupIOStreams got the synchronized lock
// first after closeConn, thus changing the thread to a new thread.
if (isCurrentThreadExpected()) {
LOG.debug(msg + ", closing connection");
closeConn(new InterruptedIOException(msg));
} else {
LOG.debug(msg);
}
return false;
}
}
}
@Override
public void run() {
if (LOG.isTraceEnabled()) {
LOG.trace("starting");
}
// We have a synchronization here because it's possible in error scenarios for a new
// thread to be started while readResponse is still reading on the socket. We don't want
// two threads to be reading from the same socket/inputstream.
// The below calls can synchronize on "BlockingRpcConnection.this".
// We should not synchronize on readerThreadLock anywhere else, to avoid deadlocks
synchronized (readerThreadLock) {
if (LOG.isTraceEnabled()) {
LOG.trace("started");
}
while (waitForWork()) {
readResponse();
}
}
if (LOG.isTraceEnabled()) {
LOG.trace("stopped");
}
}
private void disposeSasl() {
if (saslRpcClient != null) {
saslRpcClient.dispose();
saslRpcClient = null;
}
}
private boolean setupSaslConnection(final InputStream in2, final OutputStream out2)
throws IOException {
if (this.metrics != null) {
this.metrics.incrNsLookups();
}
saslRpcClient = new HBaseSaslRpcClient(this.rpcClient.conf, provider, token,
socket.getInetAddress(), securityInfo, this.rpcClient.fallbackAllowed,
this.rpcClient.conf.get("hbase.rpc.protection",
QualityOfProtection.AUTHENTICATION.name().toLowerCase(Locale.ROOT)),
this.rpcClient.conf.getBoolean(CRYPTO_AES_ENABLED_KEY, CRYPTO_AES_ENABLED_DEFAULT));
return saslRpcClient.saslConnect(in2, out2);
}
/**
* If multiple clients with the same principal try to connect to the same server at the same time,
* the server assumes a replay attack is in progress. This is a feature of kerberos. In order to
* work around this, what is done is that the client backs off randomly and tries to initiate the
* connection again. The other problem is to do with ticket expiry. To handle that, a relogin is
* attempted.
*
* The retry logic is governed by the {@link SaslClientAuthenticationProvider#canRetry()} method.
* Some providers have the ability to obtain new credentials and then re-attempt to authenticate
* with HBase services. Other providers will continue to fail if they failed the first time -- for
* those, we want to fail-fast.
*
*/
private void handleSaslConnectionFailure(final int currRetries, final int maxRetries,
final Exception ex, final UserGroupInformation user) throws IOException, InterruptedException {
closeSocket();
user.doAs(new PrivilegedExceptionAction