Skip to content

Commit

Permalink
RATIS-1656. Leftover usage of ForkJoinPool.commonPool() in RaftServer…
Browse files Browse the repository at this point in the history
…Impl (apache#702)
  • Loading branch information
adoroszlai authored and JoeCqupt committed Aug 7, 2022
1 parent 6790b48 commit 179ffde
Show file tree
Hide file tree
Showing 3 changed files with 11 additions and 3 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -1413,7 +1413,8 @@ leaderId, getMemberId(), currentTerm, followerCommit, state.getNextIndex(), NOT_
}
}
return JavaUtils.allOf(futures).whenCompleteAsync(
(r, t) -> followerState.ifPresent(fs -> fs.updateLastRpcTime(FollowerState.UpdateType.APPEND_COMPLETE))
(r, t) -> followerState.ifPresent(fs -> fs.updateLastRpcTime(FollowerState.UpdateType.APPEND_COMPLETE)),
serverExecutor
).thenApply(v -> {
final AppendEntriesReplyProto reply;
synchronized(this) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -115,7 +115,9 @@ public synchronized void close() {
return;
}
isClosed = true;
map.entrySet().parallelStream().forEach(entry -> close(entry.getKey(), entry.getValue()));
ConcurrentUtils.parallelForEachAsync(map.entrySet(),
entry -> close(entry.getKey(), entry.getValue()),
executor);
}

private void close(RaftGroupId groupId, CompletableFuture<RaftServerImpl> future) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -51,6 +51,8 @@
import java.nio.file.Path;
import java.util.List;
import java.util.concurrent.CompletableFuture;
import java.util.concurrent.Executor;
import java.util.concurrent.Executors;
import java.util.concurrent.atomic.AtomicInteger;
import java.util.concurrent.atomic.AtomicReference;
import java.util.function.Supplier;
Expand Down Expand Up @@ -85,6 +87,9 @@ public abstract class InstallSnapshotNotificationTests<CLUSTER extends MiniRaftC
private static final AtomicInteger numNotifyInstallSnapshotFinished = new AtomicInteger();

private static class StateMachine4InstallSnapshotNotificationTests extends SimpleStateMachine4Testing {

private final Executor stateMachineExecutor = Executors.newSingleThreadExecutor();

@Override
public CompletableFuture<TermIndex> notifyInstallSnapshotFromLeader(
RaftProtos.RoleInfoProto roleInfoProto,
Expand Down Expand Up @@ -120,7 +125,7 @@ public CompletableFuture<TermIndex> notifyInstallSnapshotFromLeader(
return leaderSnapshotInfo.getTermIndex();
};

return CompletableFuture.supplyAsync(supplier);
return CompletableFuture.supplyAsync(supplier, stateMachineExecutor);
}

@Override
Expand Down

0 comments on commit 179ffde

Please sign in to comment.