|
@@ -72,6 +72,8 @@ import org.apache.hadoop.io.DataOutputBuffer;
|
|
import org.apache.hadoop.io.Writable;
|
|
import org.apache.hadoop.io.Writable;
|
|
import org.apache.hadoop.io.WritableUtils;
|
|
import org.apache.hadoop.io.WritableUtils;
|
|
import static org.apache.hadoop.ipc.RpcConstants.*;
|
|
import static org.apache.hadoop.ipc.RpcConstants.*;
|
|
|
|
+
|
|
|
|
+import org.apache.hadoop.ipc.ProtobufRpcEngine.RpcResponseMessageWrapper;
|
|
import org.apache.hadoop.ipc.ProtobufRpcEngine.RpcResponseWrapper;
|
|
import org.apache.hadoop.ipc.ProtobufRpcEngine.RpcResponseWrapper;
|
|
import org.apache.hadoop.ipc.RPC.RpcInvoker;
|
|
import org.apache.hadoop.ipc.RPC.RpcInvoker;
|
|
import org.apache.hadoop.ipc.RPC.VersionMismatch;
|
|
import org.apache.hadoop.ipc.RPC.VersionMismatch;
|
|
@@ -1271,7 +1273,27 @@ public abstract class Server {
|
|
}
|
|
}
|
|
|
|
|
|
private void saslReadAndProcess(DataInputStream dis) throws
|
|
private void saslReadAndProcess(DataInputStream dis) throws
|
|
- WrappedRpcServerException, IOException, InterruptedException {
|
|
|
|
|
|
+ WrappedRpcServerException, IOException, InterruptedException {
|
|
|
|
+ final RpcSaslProto saslMessage =
|
|
|
|
+ decodeProtobufFromStream(RpcSaslProto.newBuilder(), dis);
|
|
|
|
+ switch (saslMessage.getState()) {
|
|
|
|
+ case WRAP: {
|
|
|
|
+ if (!saslContextEstablished || !useWrap) {
|
|
|
|
+ throw new WrappedRpcServerException(
|
|
|
|
+ RpcErrorCodeProto.FATAL_INVALID_RPC_HEADER,
|
|
|
|
+ new SaslException("Server is not wrapping data"));
|
|
|
|
+ }
|
|
|
|
+ // loops over decoded data and calls processOneRpc
|
|
|
|
+ unwrapPacketAndProcessRpcs(saslMessage.getToken().toByteArray());
|
|
|
|
+ break;
|
|
|
|
+ }
|
|
|
|
+ default:
|
|
|
|
+ saslProcess(saslMessage);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ private void saslProcess(RpcSaslProto saslMessage)
|
|
|
|
+ throws WrappedRpcServerException, IOException, InterruptedException {
|
|
if (saslContextEstablished) {
|
|
if (saslContextEstablished) {
|
|
throw new WrappedRpcServerException(
|
|
throw new WrappedRpcServerException(
|
|
RpcErrorCodeProto.FATAL_INVALID_RPC_HEADER,
|
|
RpcErrorCodeProto.FATAL_INVALID_RPC_HEADER,
|
|
@@ -1280,7 +1302,7 @@ public abstract class Server {
|
|
RpcSaslProto saslResponse = null;
|
|
RpcSaslProto saslResponse = null;
|
|
try {
|
|
try {
|
|
try {
|
|
try {
|
|
- saslResponse = processSaslMessage(dis);
|
|
|
|
|
|
+ saslResponse = processSaslMessage(saslMessage);
|
|
} catch (IOException e) {
|
|
} catch (IOException e) {
|
|
IOException sendToClient = e;
|
|
IOException sendToClient = e;
|
|
Throwable cause = e;
|
|
Throwable cause = e;
|
|
@@ -1325,14 +1347,14 @@ public abstract class Server {
|
|
// do NOT enable wrapping until the last auth response is sent
|
|
// do NOT enable wrapping until the last auth response is sent
|
|
if (saslContextEstablished) {
|
|
if (saslContextEstablished) {
|
|
String qop = (String) saslServer.getNegotiatedProperty(Sasl.QOP);
|
|
String qop = (String) saslServer.getNegotiatedProperty(Sasl.QOP);
|
|
|
|
+ // SASL wrapping is only used if the connection has a QOP, and
|
|
|
|
+ // the value is not auth. ex. auth-int & auth-priv
|
|
useWrap = (qop != null && !"auth".equalsIgnoreCase(qop));
|
|
useWrap = (qop != null && !"auth".equalsIgnoreCase(qop));
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|
|
- private RpcSaslProto processSaslMessage(DataInputStream dis)
|
|
|
|
|
|
+ private RpcSaslProto processSaslMessage(RpcSaslProto saslMessage)
|
|
throws IOException, InterruptedException {
|
|
throws IOException, InterruptedException {
|
|
- final RpcSaslProto saslMessage =
|
|
|
|
- decodeProtobufFromStream(RpcSaslProto.newBuilder(), dis);
|
|
|
|
RpcSaslProto saslResponse = null;
|
|
RpcSaslProto saslResponse = null;
|
|
final SaslState state = saslMessage.getState(); // required
|
|
final SaslState state = saslMessage.getState(); // required
|
|
switch (state) {
|
|
switch (state) {
|
|
@@ -1527,7 +1549,7 @@ public abstract class Server {
|
|
dataLengthBuffer.clear();
|
|
dataLengthBuffer.clear();
|
|
data.flip();
|
|
data.flip();
|
|
boolean isHeaderRead = connectionContextRead;
|
|
boolean isHeaderRead = connectionContextRead;
|
|
- processRpcRequestPacket(data.array());
|
|
|
|
|
|
+ processOneRpc(data.array());
|
|
data = null;
|
|
data = null;
|
|
if (!isHeaderRead) {
|
|
if (!isHeaderRead) {
|
|
continue;
|
|
continue;
|
|
@@ -1690,29 +1712,19 @@ public abstract class Server {
|
|
}
|
|
}
|
|
|
|
|
|
/**
|
|
/**
|
|
- * Process a RPC Request - if SASL wrapping is enabled, unwrap the
|
|
|
|
- * requests and process each one, else directly process the request
|
|
|
|
- * @param buf - single request or SASL wrapped requests
|
|
|
|
- * @throws IOException - connection failed to authenticate or authorize,
|
|
|
|
- * or the request could not be decoded into a Call
|
|
|
|
|
|
+ * Process a wrapped RPC Request - unwrap the SASL packet and process
|
|
|
|
+ * each embedded RPC request
|
|
|
|
+ * @param buf - SASL wrapped request of one or more RPCs
|
|
|
|
+ * @throws IOException - SASL packet cannot be unwrapped
|
|
* @throws InterruptedException
|
|
* @throws InterruptedException
|
|
*/
|
|
*/
|
|
- private void processRpcRequestPacket(byte[] buf)
|
|
|
|
- throws WrappedRpcServerException, IOException, InterruptedException {
|
|
|
|
- if (saslContextEstablished && useWrap) {
|
|
|
|
- if (LOG.isDebugEnabled())
|
|
|
|
- LOG.debug("Have read input token of size " + buf.length
|
|
|
|
- + " for processing by saslServer.unwrap()");
|
|
|
|
- final byte[] plaintextData = saslServer.unwrap(buf, 0, buf.length);
|
|
|
|
- // loops over decoded data and calls processOneRpc
|
|
|
|
- unwrapPacketAndProcessRpcs(plaintextData);
|
|
|
|
- } else {
|
|
|
|
- processOneRpc(buf);
|
|
|
|
- }
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
private void unwrapPacketAndProcessRpcs(byte[] inBuf)
|
|
private void unwrapPacketAndProcessRpcs(byte[] inBuf)
|
|
throws WrappedRpcServerException, IOException, InterruptedException {
|
|
throws WrappedRpcServerException, IOException, InterruptedException {
|
|
|
|
+ if (LOG.isDebugEnabled()) {
|
|
|
|
+ LOG.debug("Have read input token of size " + inBuf.length
|
|
|
|
+ + " for processing by saslServer.unwrap()");
|
|
|
|
+ }
|
|
|
|
+ inBuf = saslServer.unwrap(inBuf, 0, inBuf.length);
|
|
ReadableByteChannel ch = Channels.newChannel(new ByteArrayInputStream(
|
|
ReadableByteChannel ch = Channels.newChannel(new ByteArrayInputStream(
|
|
inBuf));
|
|
inBuf));
|
|
// Read all RPCs contained in the inBuf, even partial ones
|
|
// Read all RPCs contained in the inBuf, even partial ones
|
|
@@ -2375,9 +2387,21 @@ public abstract class Server {
|
|
LOG.debug("Adding saslServer wrapped token of size " + token.length
|
|
LOG.debug("Adding saslServer wrapped token of size " + token.length
|
|
+ " as call response.");
|
|
+ " as call response.");
|
|
response.reset();
|
|
response.reset();
|
|
- DataOutputStream saslOut = new DataOutputStream(response);
|
|
|
|
- saslOut.writeInt(token.length);
|
|
|
|
- saslOut.write(token, 0, token.length);
|
|
|
|
|
|
+ // rebuild with sasl header and payload
|
|
|
|
+ RpcResponseHeaderProto saslHeader = RpcResponseHeaderProto.newBuilder()
|
|
|
|
+ .setCallId(AuthProtocol.SASL.callId)
|
|
|
|
+ .setStatus(RpcStatusProto.SUCCESS)
|
|
|
|
+ .build();
|
|
|
|
+ RpcSaslProto saslMessage = RpcSaslProto.newBuilder()
|
|
|
|
+ .setState(SaslState.WRAP)
|
|
|
|
+ .setToken(ByteString.copyFrom(token, 0, token.length))
|
|
|
|
+ .build();
|
|
|
|
+ RpcResponseMessageWrapper saslResponse =
|
|
|
|
+ new RpcResponseMessageWrapper(saslHeader, saslMessage);
|
|
|
|
+
|
|
|
|
+ DataOutputStream out = new DataOutputStream(response);
|
|
|
|
+ out.writeInt(saslResponse.getLength());
|
|
|
|
+ saslResponse.write(out);
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|