|
@@ -64,6 +64,7 @@ import org.apache.hadoop.io.BytesWritable;
|
|
|
import org.apache.hadoop.io.IntWritable;
|
|
|
import org.apache.hadoop.io.Writable;
|
|
|
import org.apache.hadoop.io.WritableUtils;
|
|
|
+import org.apache.hadoop.ipc.metrics.RpcDetailedMetrics;
|
|
|
import org.apache.hadoop.ipc.metrics.RpcMetrics;
|
|
|
import org.apache.hadoop.security.AccessControlException;
|
|
|
import org.apache.hadoop.security.SaslRpcServer;
|
|
@@ -172,7 +173,8 @@ public abstract class Server {
|
|
|
// connections to nuke
|
|
|
//during a cleanup
|
|
|
|
|
|
- protected RpcMetrics rpcMetrics;
|
|
|
+ protected RpcMetrics rpcMetrics;
|
|
|
+ protected RpcDetailedMetrics rpcDetailedMetrics;
|
|
|
|
|
|
private Configuration conf;
|
|
|
private SecretManager<TokenIdentifier> secretManager;
|
|
@@ -1268,8 +1270,9 @@ public abstract class Server {
|
|
|
// its own message ordering.
|
|
|
setupResponse(buf, call, (error == null) ? Status.SUCCESS
|
|
|
: Status.ERROR, value, errorClass, error);
|
|
|
- // Discard the large buf and reset it back to
|
|
|
- // smaller size to freeup heap
|
|
|
+
|
|
|
+ // Discard the large buf and reset it back to smaller size
|
|
|
+ // to free up heap
|
|
|
if (buf.size() > maxRespSize) {
|
|
|
LOG.warn("Large response size " + buf.size() + " for call "
|
|
|
+ call.toString());
|
|
@@ -1336,6 +1339,8 @@ public abstract class Server {
|
|
|
this.port = listener.getAddress().getPort();
|
|
|
this.rpcMetrics = new RpcMetrics(serverName,
|
|
|
Integer.toString(this.port), this);
|
|
|
+ this.rpcDetailedMetrics = new RpcDetailedMetrics(serverName,
|
|
|
+ Integer.toString(this.port));
|
|
|
this.tcpNoDelay = conf.getBoolean("ipc.server.tcpnodelay", false);
|
|
|
|
|
|
|
|
@@ -1450,6 +1455,9 @@ public abstract class Server {
|
|
|
if (this.rpcMetrics != null) {
|
|
|
this.rpcMetrics.shutdown();
|
|
|
}
|
|
|
+ if (this.rpcDetailedMetrics != null) {
|
|
|
+ this.rpcDetailedMetrics.shutdown();
|
|
|
+ }
|
|
|
}
|
|
|
|
|
|
/** Wait for the server to be stopped.
|
|
@@ -1540,11 +1548,15 @@ public abstract class Server {
|
|
|
*
|
|
|
* @see WritableByteChannel#write(ByteBuffer)
|
|
|
*/
|
|
|
- private static int channelWrite(WritableByteChannel channel,
|
|
|
- ByteBuffer buffer) throws IOException {
|
|
|
+ private int channelWrite(WritableByteChannel channel,
|
|
|
+ ByteBuffer buffer) throws IOException {
|
|
|
|
|
|
- return (buffer.remaining() <= NIO_BUFFER_LIMIT) ?
|
|
|
- channel.write(buffer) : channelIO(null, channel, buffer);
|
|
|
+ int count = (buffer.remaining() <= NIO_BUFFER_LIMIT) ?
|
|
|
+ channel.write(buffer) : channelIO(null, channel, buffer);
|
|
|
+ if (count > 0) {
|
|
|
+ rpcMetrics.sentBytes.inc(count);
|
|
|
+ }
|
|
|
+ return count;
|
|
|
}
|
|
|
|
|
|
|
|
@@ -1556,11 +1568,15 @@ public abstract class Server {
|
|
|
*
|
|
|
* @see ReadableByteChannel#read(ByteBuffer)
|
|
|
*/
|
|
|
- private static int channelRead(ReadableByteChannel channel,
|
|
|
- ByteBuffer buffer) throws IOException {
|
|
|
+ private int channelRead(ReadableByteChannel channel,
|
|
|
+ ByteBuffer buffer) throws IOException {
|
|
|
|
|
|
- return (buffer.remaining() <= NIO_BUFFER_LIMIT) ?
|
|
|
- channel.read(buffer) : channelIO(channel, null, buffer);
|
|
|
+ int count = (buffer.remaining() <= NIO_BUFFER_LIMIT) ?
|
|
|
+ channel.read(buffer) : channelIO(channel, null, buffer);
|
|
|
+ if (count > 0) {
|
|
|
+ rpcMetrics.receivedBytes.inc(count);
|
|
|
+ }
|
|
|
+ return count;
|
|
|
}
|
|
|
|
|
|
/**
|