未验证 提交 a2f8810c 编写于 作者: Y yuz10 提交者: GitHub

[ISSUE #2988] fix fail to send trace of last message before shutting down producer

上级 6b519ef1
...@@ -16,7 +16,6 @@ ...@@ -16,7 +16,6 @@
*/ */
package org.apache.rocketmq.client.trace; package org.apache.rocketmq.client.trace;
import java.io.IOException;
import java.util.ArrayList; import java.util.ArrayList;
import java.util.HashMap; import java.util.HashMap;
import java.util.HashSet; import java.util.HashSet;
...@@ -30,6 +29,7 @@ import java.util.concurrent.TimeUnit; ...@@ -30,6 +29,7 @@ import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicBoolean; import java.util.concurrent.atomic.AtomicBoolean;
import java.util.concurrent.atomic.AtomicInteger; import java.util.concurrent.atomic.AtomicInteger;
import java.util.concurrent.atomic.AtomicLong; import java.util.concurrent.atomic.AtomicLong;
import org.apache.commons.lang3.StringUtils; import org.apache.commons.lang3.StringUtils;
import org.apache.rocketmq.client.AccessChannel; import org.apache.rocketmq.client.AccessChannel;
import org.apache.rocketmq.client.common.ThreadLocalIndex; import org.apache.rocketmq.client.common.ThreadLocalIndex;
...@@ -64,7 +64,7 @@ public class AsyncTraceDispatcher implements TraceDispatcher { ...@@ -64,7 +64,7 @@ public class AsyncTraceDispatcher implements TraceDispatcher {
// The last discard number of log // The last discard number of log
private AtomicLong discardCount; private AtomicLong discardCount;
private Thread worker; private Thread worker;
private ArrayBlockingQueue<TraceContext> traceContextQueue; private final ArrayBlockingQueue<TraceContext> traceContextQueue;
private ArrayBlockingQueue<Runnable> appenderQueue; private ArrayBlockingQueue<Runnable> appenderQueue;
private volatile Thread shutDownHook; private volatile Thread shutDownHook;
private volatile boolean stopped = false; private volatile boolean stopped = false;
...@@ -78,7 +78,7 @@ public class AsyncTraceDispatcher implements TraceDispatcher { ...@@ -78,7 +78,7 @@ public class AsyncTraceDispatcher implements TraceDispatcher {
private String group; private String group;
private Type type; private Type type;
public AsyncTraceDispatcher(String group, Type type,String traceTopicName, RPCHook rpcHook) { public AsyncTraceDispatcher(String group, Type type, String traceTopicName, RPCHook rpcHook) {
// queueSize is greater than or equal to the n power of 2 of value // queueSize is greater than or equal to the n power of 2 of value
this.queueSize = 2048; this.queueSize = 2048;
this.batchSize = 100; this.batchSize = 100;
...@@ -180,10 +180,15 @@ public class AsyncTraceDispatcher implements TraceDispatcher { ...@@ -180,10 +180,15 @@ public class AsyncTraceDispatcher implements TraceDispatcher {
} }
@Override @Override
public void flush() throws IOException { public void flush() {
// The maximum waiting time for refresh,avoid being written all the time, resulting in failure to return. // The maximum waiting time for refresh,avoid being written all the time, resulting in failure to return.
long end = System.currentTimeMillis() + 500; long end = System.currentTimeMillis() + 500;
while (traceContextQueue.size() > 0 || appenderQueue.size() > 0 && System.currentTimeMillis() <= end) { while (System.currentTimeMillis() <= end) {
synchronized (traceContextQueue) {
if (traceContextQueue.size() == 0 && appenderQueue.size() == 0) {
break;
}
}
try { try {
Thread.sleep(1); Thread.sleep(1);
} catch (InterruptedException e) { } catch (InterruptedException e) {
...@@ -196,6 +201,7 @@ public class AsyncTraceDispatcher implements TraceDispatcher { ...@@ -196,6 +201,7 @@ public class AsyncTraceDispatcher implements TraceDispatcher {
@Override @Override
public void shutdown() { public void shutdown() {
this.stopped = true; this.stopped = true;
flush();
this.traceExecutor.shutdown(); this.traceExecutor.shutdown();
if (isStarted.get()) { if (isStarted.get()) {
traceProducer.shutdown(); traceProducer.shutdown();
...@@ -212,11 +218,7 @@ public class AsyncTraceDispatcher implements TraceDispatcher { ...@@ -212,11 +218,7 @@ public class AsyncTraceDispatcher implements TraceDispatcher {
public void run() { public void run() {
synchronized (this) { synchronized (this) {
if (!this.hasShutdown) { if (!this.hasShutdown) {
try {
flush(); flush();
} catch (IOException e) {
log.error("system MQTrace hook shutdown failed ,maybe loss some trace data");
}
} }
} }
} }
...@@ -242,10 +244,11 @@ public class AsyncTraceDispatcher implements TraceDispatcher { ...@@ -242,10 +244,11 @@ public class AsyncTraceDispatcher implements TraceDispatcher {
public void run() { public void run() {
while (!stopped) { while (!stopped) {
List<TraceContext> contexts = new ArrayList<TraceContext>(batchSize); List<TraceContext> contexts = new ArrayList<TraceContext>(batchSize);
synchronized (traceContextQueue) {
for (int i = 0; i < batchSize; i++) { for (int i = 0; i < batchSize; i++) {
TraceContext context = null; TraceContext context = null;
try { try {
//get trace data element from blocking Queue — traceContextQueue //get trace data element from blocking Queue - traceContextQueue
context = traceContextQueue.poll(5, TimeUnit.MILLISECONDS); context = traceContextQueue.poll(5, TimeUnit.MILLISECONDS);
} catch (InterruptedException e) { } catch (InterruptedException e) {
} }
...@@ -262,6 +265,7 @@ public class AsyncTraceDispatcher implements TraceDispatcher { ...@@ -262,6 +265,7 @@ public class AsyncTraceDispatcher implements TraceDispatcher {
this.stopped = true; this.stopped = true;
} }
} }
}
} }
} }
......
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册