HDDS-1152. Add trace information for the client side of the datanode writes
Contributed by Elek, Marton.
This commit is contained in:
parent
5c1f946071
commit
3aa0a57ea0
@ -37,6 +37,9 @@
|
|||||||
import org.apache.hadoop.ozone.OzoneConsts;
|
import org.apache.hadoop.ozone.OzoneConsts;
|
||||||
import org.apache.hadoop.security.UserGroupInformation;
|
import org.apache.hadoop.security.UserGroupInformation;
|
||||||
import org.apache.hadoop.util.Time;
|
import org.apache.hadoop.util.Time;
|
||||||
|
|
||||||
|
import io.opentracing.Scope;
|
||||||
|
import io.opentracing.util.GlobalTracer;
|
||||||
import org.apache.ratis.thirdparty.io.grpc.ManagedChannel;
|
import org.apache.ratis.thirdparty.io.grpc.ManagedChannel;
|
||||||
import org.apache.ratis.thirdparty.io.grpc.Status;
|
import org.apache.ratis.thirdparty.io.grpc.Status;
|
||||||
import org.apache.ratis.thirdparty.io.grpc.netty.GrpcSslContexts;
|
import org.apache.ratis.thirdparty.io.grpc.netty.GrpcSslContexts;
|
||||||
@ -288,9 +291,11 @@ private XceiverClientReply sendCommandWithRetry(
|
|||||||
public XceiverClientReply sendCommandAsync(
|
public XceiverClientReply sendCommandAsync(
|
||||||
ContainerCommandRequestProto request)
|
ContainerCommandRequestProto request)
|
||||||
throws IOException, ExecutionException, InterruptedException {
|
throws IOException, ExecutionException, InterruptedException {
|
||||||
|
try (Scope scope = GlobalTracer.get()
|
||||||
|
.buildSpan("XceiverClientGrpc." + request.getCmdType().name())
|
||||||
|
.startActive(true)) {
|
||||||
XceiverClientReply asyncReply =
|
XceiverClientReply asyncReply =
|
||||||
sendCommandAsync(request, pipeline.getFirstNode());
|
sendCommandAsync(request, pipeline.getFirstNode());
|
||||||
|
|
||||||
// TODO : for now make this API sync in nature as async requests are
|
// TODO : for now make this API sync in nature as async requests are
|
||||||
// served out of order over XceiverClientGrpc. This needs to be fixed
|
// served out of order over XceiverClientGrpc. This needs to be fixed
|
||||||
// if this API is to be used for I/O path. Currently, this is not
|
// if this API is to be used for I/O path. Currently, this is not
|
||||||
@ -300,6 +305,7 @@ public XceiverClientReply sendCommandAsync(
|
|||||||
}
|
}
|
||||||
return asyncReply;
|
return asyncReply;
|
||||||
}
|
}
|
||||||
|
}
|
||||||
|
|
||||||
private XceiverClientReply sendCommandAsync(
|
private XceiverClientReply sendCommandAsync(
|
||||||
ContainerCommandRequestProto request, DatanodeDetails dn)
|
ContainerCommandRequestProto request, DatanodeDetails dn)
|
||||||
|
@ -22,6 +22,9 @@
|
|||||||
import org.apache.hadoop.hdds.HddsUtils;
|
import org.apache.hadoop.hdds.HddsUtils;
|
||||||
import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos;
|
import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos;
|
||||||
import org.apache.hadoop.hdds.security.x509.SecurityConfig;
|
import org.apache.hadoop.hdds.security.x509.SecurityConfig;
|
||||||
|
|
||||||
|
import io.opentracing.Scope;
|
||||||
|
import io.opentracing.util.GlobalTracer;
|
||||||
import org.apache.ratis.grpc.GrpcTlsConfig;
|
import org.apache.ratis.grpc.GrpcTlsConfig;
|
||||||
import org.apache.ratis.proto.RaftProtos;
|
import org.apache.ratis.proto.RaftProtos;
|
||||||
import org.apache.ratis.protocol.RaftRetryFailureException;
|
import org.apache.ratis.protocol.RaftRetryFailureException;
|
||||||
@ -191,6 +194,9 @@ private RaftClient getClient() {
|
|||||||
|
|
||||||
private CompletableFuture<RaftClientReply> sendRequestAsync(
|
private CompletableFuture<RaftClientReply> sendRequestAsync(
|
||||||
ContainerCommandRequestProto request) {
|
ContainerCommandRequestProto request) {
|
||||||
|
try (Scope scope = GlobalTracer.get()
|
||||||
|
.buildSpan("XceiverClientRatis." + request.getCmdType().name())
|
||||||
|
.startActive(true)) {
|
||||||
ContainerCommandRequestProto finalPayload =
|
ContainerCommandRequestProto finalPayload =
|
||||||
ContainerCommandRequestProto.newBuilder(request)
|
ContainerCommandRequestProto.newBuilder(request)
|
||||||
.setTraceID(TracingUtil.exportCurrentSpan())
|
.setTraceID(TracingUtil.exportCurrentSpan())
|
||||||
@ -198,9 +204,11 @@ private CompletableFuture<RaftClientReply> sendRequestAsync(
|
|||||||
boolean isReadOnlyRequest = HddsUtils.isReadOnly(finalPayload);
|
boolean isReadOnlyRequest = HddsUtils.isReadOnly(finalPayload);
|
||||||
ByteString byteString = finalPayload.toByteString();
|
ByteString byteString = finalPayload.toByteString();
|
||||||
LOG.debug("sendCommandAsync {} {}", isReadOnlyRequest, finalPayload);
|
LOG.debug("sendCommandAsync {} {}", isReadOnlyRequest, finalPayload);
|
||||||
return isReadOnlyRequest ? getClient().sendReadOnlyAsync(() -> byteString) :
|
return isReadOnlyRequest ?
|
||||||
|
getClient().sendReadOnlyAsync(() -> byteString) :
|
||||||
getClient().sendAsync(() -> byteString);
|
getClient().sendAsync(() -> byteString);
|
||||||
}
|
}
|
||||||
|
}
|
||||||
|
|
||||||
// gets the minimum log index replicated to all servers
|
// gets the minimum log index replicated to all servers
|
||||||
@Override
|
@Override
|
||||||
|
@ -1,3 +1,4 @@
|
|||||||
|
|
||||||
/**
|
/**
|
||||||
* Licensed to the Apache Software Foundation (ASF) under one or more
|
* Licensed to the Apache Software Foundation (ASF) under one or more
|
||||||
* contributor license agreements. See the NOTICE file distributed with this
|
* contributor license agreements. See the NOTICE file distributed with this
|
||||||
@ -614,9 +615,13 @@ public void run() {
|
|||||||
.update(keyCreationDuration);
|
.update(keyCreationDuration);
|
||||||
keyCreationTime.getAndAdd(keyCreationDuration);
|
keyCreationTime.getAndAdd(keyCreationDuration);
|
||||||
long keyWriteStart = System.nanoTime();
|
long keyWriteStart = System.nanoTime();
|
||||||
|
try (Scope writeScope = GlobalTracer.get()
|
||||||
|
.buildSpan("writeKeyData")
|
||||||
|
.startActive(true)) {
|
||||||
os.write(keyValue);
|
os.write(keyValue);
|
||||||
os.write(randomValue);
|
os.write(randomValue);
|
||||||
os.close();
|
os.close();
|
||||||
|
}
|
||||||
|
|
||||||
long keyWriteDuration = System.nanoTime() - keyWriteStart;
|
long keyWriteDuration = System.nanoTime() - keyWriteStart;
|
||||||
|
|
||||||
|
Loading…
Reference in New Issue
Block a user