|
@@ -279,6 +279,15 @@ public abstract class Server {
|
|
Call call = CurCall.get();
|
|
Call call = CurCall.get();
|
|
return call != null ? call.callId : RpcConstants.INVALID_CALL_ID;
|
|
return call != null ? call.callId : RpcConstants.INVALID_CALL_ID;
|
|
}
|
|
}
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * @return The current active RPC call's retry count. -1 indicates the retry
|
|
|
|
+ * cache is not supported in the client side.
|
|
|
|
+ */
|
|
|
|
+ public static int getCallRetryCount() {
|
|
|
|
+ Call call = CurCall.get();
|
|
|
|
+ return call != null ? call.retryCount : RpcConstants.INVALID_RETRY_COUNT;
|
|
|
|
+ }
|
|
|
|
|
|
/** Returns the remote side ip address when invoked inside an RPC
|
|
/** Returns the remote side ip address when invoked inside an RPC
|
|
* Returns null incase of an error.
|
|
* Returns null incase of an error.
|
|
@@ -453,6 +462,7 @@ public abstract class Server {
|
|
/** A call queued for handling. */
|
|
/** A call queued for handling. */
|
|
private static class Call {
|
|
private static class Call {
|
|
private final int callId; // the client's call id
|
|
private final int callId; // the client's call id
|
|
|
|
+ private final int retryCount; // the retry count of the call
|
|
private final Writable rpcRequest; // Serialized Rpc request from client
|
|
private final Writable rpcRequest; // Serialized Rpc request from client
|
|
private final Connection connection; // connection to client
|
|
private final Connection connection; // connection to client
|
|
private long timestamp; // time received when response is null
|
|
private long timestamp; // time received when response is null
|
|
@@ -461,14 +471,16 @@ public abstract class Server {
|
|
private final RPC.RpcKind rpcKind;
|
|
private final RPC.RpcKind rpcKind;
|
|
private final byte[] clientId;
|
|
private final byte[] clientId;
|
|
|
|
|
|
- private Call(int id, Writable param, Connection connection) {
|
|
|
|
- this(id, param, connection, RPC.RpcKind.RPC_BUILTIN,
|
|
|
|
|
|
+ private Call(int id, int retryCount, Writable param,
|
|
|
|
+ Connection connection) {
|
|
|
|
+ this(id, retryCount, param, connection, RPC.RpcKind.RPC_BUILTIN,
|
|
RpcConstants.DUMMY_CLIENT_ID);
|
|
RpcConstants.DUMMY_CLIENT_ID);
|
|
}
|
|
}
|
|
|
|
|
|
- private Call(int id, Writable param, Connection connection,
|
|
|
|
|
|
+ private Call(int id, int retryCount, Writable param, Connection connection,
|
|
RPC.RpcKind kind, byte[] clientId) {
|
|
RPC.RpcKind kind, byte[] clientId) {
|
|
this.callId = id;
|
|
this.callId = id;
|
|
|
|
+ this.retryCount = retryCount;
|
|
this.rpcRequest = param;
|
|
this.rpcRequest = param;
|
|
this.connection = connection;
|
|
this.connection = connection;
|
|
this.timestamp = Time.now();
|
|
this.timestamp = Time.now();
|
|
@@ -479,7 +491,8 @@ public abstract class Server {
|
|
|
|
|
|
@Override
|
|
@Override
|
|
public String toString() {
|
|
public String toString() {
|
|
- return rpcRequest + " from " + connection + " Call#" + callId;
|
|
|
|
|
|
+ return rpcRequest + " from " + connection + " Call#" + callId + " Retry#"
|
|
|
|
+ + retryCount;
|
|
}
|
|
}
|
|
|
|
|
|
public void setResponse(ByteBuffer response) {
|
|
public void setResponse(ByteBuffer response) {
|
|
@@ -1160,11 +1173,12 @@ public abstract class Server {
|
|
private static final int AUTHORIZATION_FAILED_CALLID = -1;
|
|
private static final int AUTHORIZATION_FAILED_CALLID = -1;
|
|
private static final int CONNECTION_CONTEXT_CALL_ID = -3;
|
|
private static final int CONNECTION_CONTEXT_CALL_ID = -3;
|
|
|
|
|
|
- private final Call authFailedCall =
|
|
|
|
- new Call(AUTHORIZATION_FAILED_CALLID, null, this);
|
|
|
|
|
|
+ private final Call authFailedCall = new Call(AUTHORIZATION_FAILED_CALLID,
|
|
|
|
+ RpcConstants.INVALID_RETRY_COUNT, null, this);
|
|
private ByteArrayOutputStream authFailedResponse = new ByteArrayOutputStream();
|
|
private ByteArrayOutputStream authFailedResponse = new ByteArrayOutputStream();
|
|
|
|
|
|
- private final Call saslCall = new Call(AuthProtocol.SASL.callId, null, this);
|
|
|
|
|
|
+ private final Call saslCall = new Call(AuthProtocol.SASL.callId,
|
|
|
|
+ RpcConstants.INVALID_RETRY_COUNT, null, this);
|
|
private final ByteArrayOutputStream saslResponse = new ByteArrayOutputStream();
|
|
private final ByteArrayOutputStream saslResponse = new ByteArrayOutputStream();
|
|
|
|
|
|
private boolean sentNegotiate = false;
|
|
private boolean sentNegotiate = false;
|
|
@@ -1592,20 +1606,23 @@ public abstract class Server {
|
|
|
|
|
|
if (clientVersion >= 9) {
|
|
if (clientVersion >= 9) {
|
|
// Versions >>9 understand the normal response
|
|
// Versions >>9 understand the normal response
|
|
- Call fakeCall = new Call(-1, null, this);
|
|
|
|
|
|
+ Call fakeCall = new Call(-1, RpcConstants.INVALID_RETRY_COUNT, null,
|
|
|
|
+ this);
|
|
setupResponse(buffer, fakeCall,
|
|
setupResponse(buffer, fakeCall,
|
|
RpcStatusProto.FATAL, RpcErrorCodeProto.FATAL_VERSION_MISMATCH,
|
|
RpcStatusProto.FATAL, RpcErrorCodeProto.FATAL_VERSION_MISMATCH,
|
|
null, VersionMismatch.class.getName(), errMsg);
|
|
null, VersionMismatch.class.getName(), errMsg);
|
|
responder.doRespond(fakeCall);
|
|
responder.doRespond(fakeCall);
|
|
} else if (clientVersion >= 3) {
|
|
} else if (clientVersion >= 3) {
|
|
- Call fakeCall = new Call(-1, null, this);
|
|
|
|
|
|
+ Call fakeCall = new Call(-1, RpcConstants.INVALID_RETRY_COUNT, null,
|
|
|
|
+ this);
|
|
// Versions 3 to 8 use older response
|
|
// Versions 3 to 8 use older response
|
|
setupResponseOldVersionFatal(buffer, fakeCall,
|
|
setupResponseOldVersionFatal(buffer, fakeCall,
|
|
null, VersionMismatch.class.getName(), errMsg);
|
|
null, VersionMismatch.class.getName(), errMsg);
|
|
|
|
|
|
responder.doRespond(fakeCall);
|
|
responder.doRespond(fakeCall);
|
|
} else if (clientVersion == 2) { // Hadoop 0.18.3
|
|
} else if (clientVersion == 2) { // Hadoop 0.18.3
|
|
- Call fakeCall = new Call(0, null, this);
|
|
|
|
|
|
+ Call fakeCall = new Call(0, RpcConstants.INVALID_RETRY_COUNT, null,
|
|
|
|
+ this);
|
|
DataOutputStream out = new DataOutputStream(buffer);
|
|
DataOutputStream out = new DataOutputStream(buffer);
|
|
out.writeInt(0); // call ID
|
|
out.writeInt(0); // call ID
|
|
out.writeBoolean(true); // error
|
|
out.writeBoolean(true); // error
|
|
@@ -1618,7 +1635,7 @@ public abstract class Server {
|
|
}
|
|
}
|
|
|
|
|
|
private void setupHttpRequestOnIpcPortResponse() throws IOException {
|
|
private void setupHttpRequestOnIpcPortResponse() throws IOException {
|
|
- Call fakeCall = new Call(0, null, this);
|
|
|
|
|
|
+ Call fakeCall = new Call(0, RpcConstants.INVALID_RETRY_COUNT, null, this);
|
|
fakeCall.setResponse(ByteBuffer.wrap(
|
|
fakeCall.setResponse(ByteBuffer.wrap(
|
|
RECEIVED_HTTP_REQ_RESPONSE.getBytes()));
|
|
RECEIVED_HTTP_REQ_RESPONSE.getBytes()));
|
|
responder.doRespond(fakeCall);
|
|
responder.doRespond(fakeCall);
|
|
@@ -1750,12 +1767,14 @@ public abstract class Server {
|
|
private void processOneRpc(byte[] buf)
|
|
private void processOneRpc(byte[] buf)
|
|
throws IOException, WrappedRpcServerException, InterruptedException {
|
|
throws IOException, WrappedRpcServerException, InterruptedException {
|
|
int callId = -1;
|
|
int callId = -1;
|
|
|
|
+ int retry = RpcConstants.INVALID_RETRY_COUNT;
|
|
try {
|
|
try {
|
|
final DataInputStream dis =
|
|
final DataInputStream dis =
|
|
new DataInputStream(new ByteArrayInputStream(buf));
|
|
new DataInputStream(new ByteArrayInputStream(buf));
|
|
final RpcRequestHeaderProto header =
|
|
final RpcRequestHeaderProto header =
|
|
decodeProtobufFromStream(RpcRequestHeaderProto.newBuilder(), dis);
|
|
decodeProtobufFromStream(RpcRequestHeaderProto.newBuilder(), dis);
|
|
callId = header.getCallId();
|
|
callId = header.getCallId();
|
|
|
|
+ retry = header.getRetryCount();
|
|
if (LOG.isDebugEnabled()) {
|
|
if (LOG.isDebugEnabled()) {
|
|
LOG.debug(" got #" + callId);
|
|
LOG.debug(" got #" + callId);
|
|
}
|
|
}
|
|
@@ -1772,7 +1791,7 @@ public abstract class Server {
|
|
}
|
|
}
|
|
} catch (WrappedRpcServerException wrse) { // inform client of error
|
|
} catch (WrappedRpcServerException wrse) { // inform client of error
|
|
Throwable ioe = wrse.getCause();
|
|
Throwable ioe = wrse.getCause();
|
|
- final Call call = new Call(callId, null, this);
|
|
|
|
|
|
+ final Call call = new Call(callId, retry, null, this);
|
|
setupResponse(authFailedResponse, call,
|
|
setupResponse(authFailedResponse, call,
|
|
RpcStatusProto.FATAL, wrse.getRpcErrorCodeProto(), null,
|
|
RpcStatusProto.FATAL, wrse.getRpcErrorCodeProto(), null,
|
|
ioe.getClass().getName(), ioe.getMessage());
|
|
ioe.getClass().getName(), ioe.getMessage());
|
|
@@ -1846,9 +1865,9 @@ public abstract class Server {
|
|
RpcErrorCodeProto.FATAL_DESERIALIZING_REQUEST, err);
|
|
RpcErrorCodeProto.FATAL_DESERIALIZING_REQUEST, err);
|
|
}
|
|
}
|
|
|
|
|
|
- Call call = new Call(header.getCallId(), rpcRequest, this,
|
|
|
|
- ProtoUtil.convert(header.getRpcKind()), header.getClientId()
|
|
|
|
- .toByteArray());
|
|
|
|
|
|
+ Call call = new Call(header.getCallId(), header.getRetryCount(),
|
|
|
|
+ rpcRequest, this, ProtoUtil.convert(header.getRpcKind()), header
|
|
|
|
+ .getClientId().toByteArray());
|
|
callQueue.put(call); // queue the call; maybe blocked here
|
|
callQueue.put(call); // queue the call; maybe blocked here
|
|
incRpcCount(); // Increment the rpc count
|
|
incRpcCount(); // Increment the rpc count
|
|
}
|
|
}
|