Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension


Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
4 changes: 1 addition & 3 deletions hadoop-client-modules/hadoop-client-api/pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -141,9 +141,7 @@
<excludes>
<exclude>org/apache/hadoop/*</exclude>
<exclude>org/apache/hadoop/**/*</exclude>
<!-- Our non-shaded htrace and logging libraries -->
<exclude>org/apache/htrace/*</exclude>
<exclude>org/apache/htrace/**/*</exclude>
<!-- Our non-shaded logging libraries -->
<exclude>org/slf4j/*</exclude>
<exclude>org/slf4j/**/*</exclude>
<exclude>org/apache/commons/logging/*</exclude>
Expand Down
2 changes: 0 additions & 2 deletions hadoop-client-modules/hadoop-client-check-invariants/pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -80,8 +80,6 @@
but enforcer still sees it.
-->
<exclude>org.apache.hadoop:hadoop-annotations</exclude>
<!-- We leave HTrace as an unshaded dependnecy on purpose so that tracing within a JVM will work -->
<exclude>org.apache.htrace:htrace-core4</exclude>
<!-- Leave slf4j unshaded so downstream users can configure logging. -->
<exclude>org.slf4j:slf4j-api</exclude>
<!-- Leave commons-logging unshaded so downstream users can configure logging. -->
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -84,8 +84,6 @@
but enforcer still sees it.
-->
<exclude>org.apache.hadoop:hadoop-annotations</exclude>
<!-- We leave HTrace as an unshaded dependnecy on purpose so that tracing within a JVM will work -->
<exclude>org.apache.htrace:htrace-core4</exclude>
<!-- Leave slf4j unshaded so downstream users can configure logging. -->
<exclude>org.slf4j:slf4j-api</exclude>
<!-- Leave commons-logging unshaded so downstream users can configure logging. -->
Expand Down
5 changes: 1 addition & 4 deletions hadoop-client-modules/hadoop-client-minicluster/pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -678,7 +678,6 @@
<exclude>org.apache.hadoop:hadoop-client-api</exclude>
<exclude>org.apache.hadoop:hadoop-client-runtime</exclude>
<!-- Fine to expose our purposefully not-shaded deps as dependencies -->
<exclude>org.apache.htrace:htrace-core4</exclude>
<exclude>org.slf4j:slf4j-api</exclude>
<exclude>commons-logging:commons-logging</exclude>
<exclude>junit:junit</exclude>
Expand Down Expand Up @@ -871,9 +870,7 @@
<excludes>
<exclude>org/apache/hadoop/*</exclude>
<exclude>org/apache/hadoop/**/*</exclude>
<!-- Our non-shaded htrace and logging libraries -->
<exclude>org/apache/htrace/*</exclude>
<exclude>org/apache/htrace/**/*</exclude>
<!-- Our non-shaded logging libraries -->
<exclude>org/slf4j/*</exclude>
<exclude>org/slf4j/**/*</exclude>
<exclude>org/apache/commons/logging/*</exclude>
Expand Down
12 changes: 1 addition & 11 deletions hadoop-client-modules/hadoop-client-runtime/pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -81,15 +81,9 @@
</dependency>
<!-- Since hadoop-client is listed as optional, we have to list transitive
dependencies that we still want to show up.
* HTrace
* Slf4j API
* commons-logging
-->
<dependency>
<groupId>org.apache.htrace</groupId>
<artifactId>htrace-core4</artifactId>
<scope>runtime</scope>
</dependency>
<dependency>
<groupId>org.slf4j</groupId>
<artifactId>slf4j-api</artifactId>
Expand Down Expand Up @@ -152,8 +146,6 @@
<excludes>
<!-- We need a filter that matches just those things that aer included in the api jar -->
<exclude>org.apache.hadoop:hadoop-client-api</exclude>
<!-- Leave HTrace as an unshaded dependency on purpose, since a static class member is used to trace within a given JVM instance -->
<exclude>org.apache.htrace:htrace-core4</exclude>
<!-- Leave slf4j unshaded so downstream users can configure logging. -->
<exclude>org.slf4j:slf4j-api</exclude>
<!-- Leave commons-logging unshaded so downstream users can configure logging. -->
Expand Down Expand Up @@ -258,9 +250,7 @@
<excludes>
<exclude>org/apache/hadoop/*</exclude>
<exclude>org/apache/hadoop/**/*</exclude>
<!-- Our non-shaded htrace and logging libraries -->
<exclude>org/apache/htrace/*</exclude>
<exclude>org/apache/htrace/**/*</exclude>
<!-- Our non-shaded logging libraries -->
<exclude>org/slf4j/*</exclude>
<exclude>org/slf4j/**/*</exclude>
<exclude>org/apache/commons/logging/*</exclude>
Expand Down
4 changes: 0 additions & 4 deletions hadoop-common-project/hadoop-common/pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -286,10 +286,6 @@
<scope>test</scope>
</dependency>

<dependency>
<groupId>org.apache.htrace</groupId>
<artifactId>htrace-core4</artifactId>
</dependency>
<dependency>
<groupId>org.apache.zookeeper</groupId>
<artifactId>zookeeper</artifactId>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,7 +21,7 @@
import org.apache.hadoop.classification.InterfaceAudience;
import org.apache.hadoop.classification.InterfaceStability;
import org.apache.hadoop.util.DataChecksum;
import org.apache.htrace.core.TraceScope;
import org.apache.hadoop.tracing.TraceScope;

import java.io.IOException;
import java.io.OutputStream;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -67,7 +67,7 @@

import org.apache.hadoop.thirdparty.com.google.common.base.Preconditions;
import org.apache.hadoop.thirdparty.com.google.common.annotations.VisibleForTesting;
import org.apache.htrace.core.Tracer;
import org.apache.hadoop.tracing.Tracer;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -83,8 +83,8 @@
import org.apache.hadoop.util.ReflectionUtils;
import org.apache.hadoop.util.ShutdownHookManager;
import org.apache.hadoop.util.StringUtils;
import org.apache.htrace.core.Tracer;
import org.apache.htrace.core.TraceScope;
import org.apache.hadoop.tracing.Tracer;
import org.apache.hadoop.tracing.TraceScope;

import org.apache.hadoop.thirdparty.com.google.common.base.Preconditions;
import org.apache.hadoop.thirdparty.com.google.common.annotations.VisibleForTesting;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -35,8 +35,8 @@
import org.apache.hadoop.util.StringUtils;
import org.apache.hadoop.util.Tool;
import org.apache.hadoop.util.ToolRunner;
import org.apache.htrace.core.TraceScope;
import org.apache.htrace.core.Tracer;
import org.apache.hadoop.tracing.TraceScope;
import org.apache.hadoop.tracing.Tracer;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,12 +17,11 @@
*/
package org.apache.hadoop.fs;

import org.apache.hadoop.thirdparty.com.google.common.annotations.VisibleForTesting;
import org.apache.hadoop.classification.InterfaceAudience;
import org.apache.hadoop.classification.InterfaceStability;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.tracing.TraceUtils;
import org.apache.htrace.core.Tracer;
import org.apache.hadoop.tracing.Tracer;

/**
* Holds the HTrace Tracer used for FileSystem operations.
Expand All @@ -47,18 +46,6 @@ public static synchronized Tracer get(Configuration conf) {
return instance;
}

@VisibleForTesting
public static synchronized void clear() {
if (instance == null) {
return;
}
try {
instance.close();
} finally {
instance = null;
}
}

private FsTracer() {
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -27,8 +27,8 @@
import org.apache.hadoop.classification.InterfaceStability;
import org.apache.hadoop.util.DurationInfo;

import org.apache.htrace.core.TraceScope;
import org.apache.htrace.core.Tracer;
import org.apache.hadoop.tracing.TraceScope;
import org.apache.hadoop.tracing.Tracer;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -54,8 +54,8 @@
import org.apache.hadoop.util.StringUtils;
import org.apache.hadoop.util.Time;
import org.apache.hadoop.util.concurrent.AsyncGet;
import org.apache.htrace.core.Span;
import org.apache.htrace.core.Tracer;
import org.apache.hadoop.tracing.Span;
import org.apache.hadoop.tracing.Tracer;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -37,8 +37,8 @@
import org.apache.hadoop.security.token.TokenIdentifier;
import org.apache.hadoop.util.Time;
import org.apache.hadoop.util.concurrent.AsyncGet;
import org.apache.htrace.core.TraceScope;
import org.apache.htrace.core.Tracer;
import org.apache.hadoop.tracing.TraceScope;
import org.apache.hadoop.tracing.Tracer;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -38,8 +38,8 @@
import org.apache.hadoop.thirdparty.protobuf.TextFormat;
import org.apache.hadoop.util.Time;
import org.apache.hadoop.util.concurrent.AsyncGet;
import org.apache.htrace.core.TraceScope;
import org.apache.htrace.core.Tracer;
import org.apache.hadoop.tracing.Tracer;
import org.apache.hadoop.tracing.TraceScope;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -100,6 +100,7 @@
import org.apache.hadoop.ipc.protobuf.RpcHeaderProtos.RpcSaslProto;
import org.apache.hadoop.ipc.protobuf.RpcHeaderProtos.RpcSaslProto.SaslAuth;
import org.apache.hadoop.ipc.protobuf.RpcHeaderProtos.RpcSaslProto.SaslState;
import org.apache.hadoop.ipc.protobuf.RpcHeaderProtos.RPCTraceInfoProto;
import org.apache.hadoop.net.NetUtils;
import org.apache.hadoop.security.AccessControlException;
import org.apache.hadoop.security.SaslPropertiesResolver;
Expand All @@ -119,10 +120,11 @@
import org.apache.hadoop.util.ProtoUtil;
import org.apache.hadoop.util.StringUtils;
import org.apache.hadoop.util.Time;
import org.apache.htrace.core.SpanId;
import org.apache.htrace.core.TraceScope;
import org.apache.htrace.core.Tracer;

import org.apache.hadoop.tracing.Span;
import org.apache.hadoop.tracing.SpanContext;
import org.apache.hadoop.tracing.TraceScope;
import org.apache.hadoop.tracing.Tracer;
import org.apache.hadoop.tracing.TraceUtils;
import com.fasterxml.jackson.databind.ObjectMapper;
import org.apache.hadoop.thirdparty.com.google.common.annotations.VisibleForTesting;
import org.apache.hadoop.thirdparty.protobuf.ByteString;
Expand Down Expand Up @@ -789,7 +791,7 @@ public static class Call implements Schedulable,
private AtomicInteger responseWaitCount = new AtomicInteger(1);
final RPC.RpcKind rpcKind;
final byte[] clientId;
private final TraceScope traceScope; // the HTrace scope on the server side
private final Span span; // the trace span on the server side
private final CallerContext callerContext; // the call context
private boolean deferredResponse = false;
private int priorityLevel;
Expand All @@ -804,7 +806,7 @@ public static class Call implements Schedulable,

Call(Call call) {
this(call.callId, call.retryCount, call.rpcKind, call.clientId,
call.traceScope, call.callerContext);
call.span, call.callerContext);
}

Call(int id, int retryCount, RPC.RpcKind kind, byte[] clientId) {
Expand All @@ -818,14 +820,14 @@ public Call(int id, int retryCount, Void ignore1, Void ignore2,
}

Call(int id, int retryCount, RPC.RpcKind kind, byte[] clientId,
TraceScope traceScope, CallerContext callerContext) {
Span span, CallerContext callerContext) {
this.callId = id;
this.retryCount = retryCount;
this.timestampNanos = Time.monotonicNowNanos();
this.responseTimestampNanos = timestampNanos;
this.rpcKind = kind;
this.clientId = clientId;
this.traceScope = traceScope;
this.span = span;
this.callerContext = callerContext;
this.clientStateId = Long.MIN_VALUE;
this.isCallCoordinated = false;
Expand Down Expand Up @@ -994,8 +996,8 @@ private class RpcCall extends Call {

RpcCall(Connection connection, int id, int retryCount,
Writable param, RPC.RpcKind kind, byte[] clientId,
TraceScope traceScope, CallerContext context) {
super(id, retryCount, kind, clientId, traceScope, context);
Span span, CallerContext context) {
super(id, retryCount, kind, clientId, span, context);
this.connection = connection;
this.rpcRequest = param;
}
Expand Down Expand Up @@ -2675,19 +2677,24 @@ private void processRpcRequest(RpcRequestHeaderProto header,
throw new FatalRpcServerException(
RpcErrorCodeProto.FATAL_DESERIALIZING_REQUEST, err);
}
TraceScope traceScope = null;

Span span = null;
if (header.hasTraceInfo()) {
if (tracer != null) {
// If the incoming RPC included tracing info, always continue the
// trace
SpanId parentSpanId = new SpanId(
header.getTraceInfo().getTraceId(),
header.getTraceInfo().getParentId());
traceScope = tracer.newScope(
RpcClientUtil.toTraceName(rpcRequest.toString()),
parentSpanId);
traceScope.detach();
RPCTraceInfoProto traceInfoProto = header.getTraceInfo();
if (traceInfoProto.hasSpanContext()) {
if (tracer == null) {
setTracer(Tracer.curThreadTracer());
}
if (tracer != null) {
// If the incoming RPC included tracing info, always continue the
// trace
SpanContext spanCtx = TraceUtils.byteStringToSpanContext(
traceInfoProto.getSpanContext());
if (spanCtx != null) {
span = tracer.newSpan(
RpcClientUtil.toTraceName(rpcRequest.toString()), spanCtx);
}
}
}
}

Expand All @@ -2703,7 +2710,7 @@ private void processRpcRequest(RpcRequestHeaderProto header,
RpcCall call = new RpcCall(this, header.getCallId(),
header.getRetryCount(), rpcRequest,
ProtoUtil.convert(header.getRpcKind()),
header.getClientId().toByteArray(), traceScope, callerContext);
header.getClientId().toByteArray(), span, callerContext);

// Save the priority level assignment by the scheduler
call.setPriorityLevel(callQueue.getPriorityLevel(call));
Expand Down Expand Up @@ -2957,10 +2964,9 @@ public void run() {
LOG.debug(Thread.currentThread().getName() + ": " + call + " for RpcKind " + call.rpcKind);
}
CurCall.set(call);
if (call.traceScope != null) {
call.traceScope.reattach();
traceScope = call.traceScope;
traceScope.getSpan().addTimelineAnnotation("called");
if (call.span != null) {
traceScope = tracer.activateSpan(call.span);
call.span.addTimelineAnnotation("called");
}
// always update the current call context
CallerContext.setCurrent(call.callerContext);
Expand All @@ -2975,14 +2981,14 @@ public void run() {
if (running) { // unexpected -- log it
LOG.info(Thread.currentThread().getName() + " unexpectedly interrupted", e);
if (traceScope != null) {
traceScope.getSpan().addTimelineAnnotation("unexpectedly interrupted: " +
traceScope.addTimelineAnnotation("unexpectedly interrupted: " +
StringUtils.stringifyException(e));
}
}
} catch (Exception e) {
LOG.info(Thread.currentThread().getName() + " caught an exception", e);
if (traceScope != null) {
traceScope.getSpan().addTimelineAnnotation("Exception: " +
traceScope.addTimelineAnnotation("Exception: " +
StringUtils.stringifyException(e));
}
} finally {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -39,8 +39,8 @@
import org.apache.hadoop.classification.InterfaceAudience;
import org.apache.hadoop.classification.InterfaceStability;
import org.apache.hadoop.conf.*;
import org.apache.htrace.core.TraceScope;
import org.apache.htrace.core.Tracer;
import org.apache.hadoop.tracing.TraceScope;
import org.apache.hadoop.tracing.Tracer;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -35,8 +35,8 @@
import java.util.concurrent.atomic.AtomicLong;
import java.util.concurrent.atomic.AtomicReference;

import org.apache.htrace.core.TraceScope;
import org.apache.htrace.core.Tracer;
import org.apache.hadoop.tracing.TraceScope;
import org.apache.hadoop.tracing.Tracer;
import org.apache.hadoop.thirdparty.com.google.common.annotations.VisibleForTesting;
import org.apache.hadoop.thirdparty.com.google.common.base.Ticker;
import org.apache.hadoop.thirdparty.com.google.common.cache.CacheBuilder;
Expand Down
Loading