This is an automated email from the ASF dual-hosted git repository.

yuyuankang pushed a commit to branch cluster_performance
in repository https://gitbox.apache.org/repos/asf/incubator-iotdb.git


The following commit(s) were added to refs/heads/cluster_performance by this 
push:
     new 1c69644  vote counter
1c69644 is described below

commit 1c6964418fa8fd0d99266486371860a4d30b8456
Author: Ring-k <[email protected]>
AuthorDate: Wed Sep 2 10:12:28 2020 +0800

    vote counter
---
 .../org/apache/iotdb/cluster/server/Timer.java     |  7 ++++
 .../iotdb/cluster/server/member/RaftMember.java    | 39 ++++++++++++----------
 2 files changed, 29 insertions(+), 17 deletions(-)

diff --git a/cluster/src/main/java/org/apache/iotdb/cluster/server/Timer.java 
b/cluster/src/main/java/org/apache/iotdb/cluster/server/Timer.java
index cea6b4a..cde55f2 100644
--- a/cluster/src/main/java/org/apache/iotdb/cluster/server/Timer.java
+++ b/cluster/src/main/java/org/apache/iotdb/cluster/server/Timer.java
@@ -29,6 +29,8 @@ public class Timer {
   public static AtomicLong raftMemberWaitForPrevLogCounter = new AtomicLong(0);
   public static AtomicLong raftMemberSendLogAyncMS = new AtomicLong(0);
   public static AtomicLong raftMemberSendLogAyncCounter = new AtomicLong(0);
+  public static AtomicLong raftMemberVoteCounterMS = new AtomicLong(0);
+  public static AtomicLong raftMemberVoteCounterCounter = new AtomicLong(0);
 
   private static final String dataGroupMemberProcessPlanLocallyMSString = 
"Data group member - process plan locally : ";
   private static final String dataGroupMemberWaitLeaderMSString = "Data group 
member - wait leader: ";
@@ -42,6 +44,7 @@ public class Timer {
   private static final String dataGroupMemberForwardPlanString = "Data group 
member - forward plan: ";
   private static final String raftMemberWaitForPrevLogString = "Raft member - 
wait for prev log: ";
   private static final String raftMemberSendLogAyncString = "Raft member - 
send log aync: ";
+  private static final String raftMemberVoteCounterString = "Raft member - 
vote counter: ";
 
 
 
@@ -102,6 +105,10 @@ public class Timer {
         + raftMemberSendLogAyncMS.get()/1000000L + ", "
         + raftMemberSendLogAyncCounter + ", "
         + (double) raftMemberSendLogAyncMS.get()/1000000L / 
raftMemberSendLogAyncCounter.get() + "\n";
+    result += raftMemberVoteCounterString
+        + raftMemberVoteCounterMS.get()/1000000L + ", "
+        + raftMemberVoteCounterCounter + ", "
+        + (double) raftMemberVoteCounterMS.get()/1000000L / 
raftMemberVoteCounterCounter.get() + "\n";
     return result;
   }
 }
diff --git 
a/cluster/src/main/java/org/apache/iotdb/cluster/server/member/RaftMember.java 
b/cluster/src/main/java/org/apache/iotdb/cluster/server/member/RaftMember.java
index 7339a51..7266e3d 100644
--- 
a/cluster/src/main/java/org/apache/iotdb/cluster/server/member/RaftMember.java
+++ 
b/cluster/src/main/java/org/apache/iotdb/cluster/server/member/RaftMember.java
@@ -654,28 +654,33 @@ public abstract class RaftMember {
       request.setHeader(getHeader());
     }
 
-    synchronized (voteCounter) {
-      // synchronized: avoid concurrent modification
-      try {
-        for (Node node : allNodes) {
-          appendLogThreadPool.submit(() -> sendLogToFollower(log, voteCounter, 
node,
-              leaderShipStale, newLeaderTerm, request));
-          if (character != NodeCharacter.LEADER) {
-            return AppendLogResult.LEADERSHIP_STALE;
-          }
+    // synchronized: avoid concurrent modification
+    try {
+      for (Node node : allNodes) {
+        appendLogThreadPool.submit(() -> sendLogToFollower(log, voteCounter, 
node,
+            leaderShipStale, newLeaderTerm, request));
+        if (character != NodeCharacter.LEADER) {
+          return AppendLogResult.LEADERSHIP_STALE;
         }
-      } catch (ConcurrentModificationException e) {
-        // retry if allNodes has changed
-        return AppendLogResult.TIME_OUT;
       }
+    } catch (ConcurrentModificationException e) {
+      // retry if allNodes has changed
+      return AppendLogResult.TIME_OUT;
+    }
 
-      try {
-        voteCounter.wait(RaftServer.getWriteOperationTimeoutMS());
-      } catch (InterruptedException e) {
-        Thread.currentThread().interrupt();
-        logger.warn("Unexpected interruption when sending a log", e);
+    long start = System.nanoTime();
+    synchronized (voteCounter) {
+      if (voteCounter.get() > 0) {
+        try {
+          voteCounter.wait(RaftServer.getWriteOperationTimeoutMS());
+        } catch (InterruptedException e) {
+          Thread.currentThread().interrupt();
+          logger.warn("Unexpected interruption when sending a log", e);
+        }
       }
     }
+    Timer.raftMemberVoteCounterMS.addAndGet(System.nanoTime() - start);
+    Timer.raftMemberVoteCounterCounter.incrementAndGet();
 
     // some node has a larger term than the local node, this node is no longer 
a valid leader
     if (leaderShipStale.get()) {

Reply via email to