rpuch commented on code in PR #1348:
URL: https://github.com/apache/ignite-3/pull/1348#discussion_r1026432756
##########
modules/raft-client/src/main/java/org/apache/ignite/raft/client/service/RaftGroupService.java:
##########
@@ -150,25 +150,26 @@ public interface RaftGroupService {
* @param peers Peers.
* @return A future.
*/
- CompletableFuture<Void> changePeers(List<Peer> peers);
+ CompletableFuture<Void> changePeers(Collection<Peer> peers);
/**
- * Changes peers of the replication group.
+ * Changes peers and learners of a replication group.
*
* <p>Asynchronous variant of the previous method.
- * When the future completed, it just means, that changePeers process
successfully started.
+ * When the future completed, it just means, that {@code changePeers}
process has successfully started.
*
- * <p>The results of rebalance itself will be processed by the listener of
raft reconfiguration event
+ * <p>The results of rebalance itself will be processed by the listener of
Raft reconfiguration event
* (from raft/server module).
*
* <p>This operation is executed on a group leader.
*
- * @param peers Peers.
+ * @param peers New peers.
+ * @param learners New learners.
* @param term Current known leader term.
* If real raft group term will be different - changePeers
will be skipped.
* @return A future.
*/
- CompletableFuture<Void> changePeersAsync(List<Peer> peers, long term);
+ CompletableFuture<Void> changePeersAsync(Collection<Peer> peers,
Collection<Peer> learners, long term);
Review Comment:
Shouldn't we change method name? As it now not only changes peers, but also
learners.
##########
modules/table/src/main/java/org/apache/ignite/internal/table/distributed/raft/RebalanceRaftGroupEventsListener.java:
##########
@@ -123,8 +124,8 @@ public class RebalanceRaftGroupEventsListener implements
RaftGroupEventsListener
/** Executor for scheduling rebalance retries. */
private final ScheduledExecutorService rebalanceScheduler;
- /** Function that performs a reconfiguration of a raft group of a
partition. */
- private final BiFunction<List<Peer>, Long, CompletableFuture<Void>>
movePartitionFn;
+ /** Class that performs a reconfiguration of a raft group of a partition.
*/
Review Comment:
It's an instance, not a class. How about removing two first words altogether?
##########
modules/raft/src/test/java/org/apache/ignite/raft/jraft/rpc/impl/cli/ChangePeersAsyncRequestProcessorTest.java:
##########
@@ -22,22 +22,26 @@
import java.util.List;
import org.apache.ignite.raft.jraft.Closure;
-import org.apache.ignite.raft.jraft.JRaftUtils;
import org.apache.ignite.raft.jraft.Node;
import org.apache.ignite.raft.jraft.Status;
+import org.apache.ignite.raft.jraft.conf.Configuration;
import org.apache.ignite.raft.jraft.entity.PeerId;
import org.apache.ignite.raft.jraft.rpc.CliRequests.ChangePeersAsyncRequest;
import org.apache.ignite.raft.jraft.rpc.CliRequests.ChangePeersAsyncResponse;
import org.mockito.ArgumentCaptor;
import org.mockito.Mockito;
public class ChangePeersAsyncRequestProcessorTest extends
AbstractCliRequestProcessorTest<ChangePeersAsyncRequest>{
+ private static final List<String> FOLLOWERS = List.of("follower1",
"follower2");
+ private static final List<String> LEARNERS = List.of("learner1",
"learner2", "learner3");
+
@Override
public ChangePeersAsyncRequest createRequest(String groupId, PeerId
peerId) {
return msgFactory.changePeersAsyncRequest()
.groupId(groupId)
.leaderId(peerId.toString())
- .newPeersList(List.of("localhost:8084", "localhost:8085"))
+ .newPeersList(FOLLOWERS)
Review Comment:
Isn't this JRaft code? Should we change the lines that do not require to be
changed?
##########
modules/core/src/testFixtures/java/org/apache/ignite/internal/testframework/matchers/CompletableFutureExceptionMatcher.java:
##########
@@ -119,4 +143,11 @@ public static CompletableFutureExceptionMatcher
willThrow(Class<? extends Except
public static CompletableFutureExceptionMatcher willThrow(Class<? extends
Exception> cls, int timeout, TimeUnit timeUnit) {
return willThrow(is(instanceOf(cls)), timeout, timeUnit);
}
+
+ /**
+ * Creates a matcher that matches a future that completes with an
exception that has a given {@code cause} in the exception stacktrace.
+ */
+ public static CompletableFutureExceptionMatcher willThrowWithCause(Class<?
extends Exception> cause) {
Review Comment:
Should the method name (and javadoc) mention that the matcher also inspects
suppressed exceptions? Like `willThrowWithCauseOrSuppressed()` or
`willThrowWithRelatedException()`?
##########
modules/table/src/main/java/org/apache/ignite/internal/table/distributed/TableManager.java:
##########
@@ -1890,15 +1886,15 @@ public boolean onUpdate(@NotNull WatchEvent evt) {
RaftGroupService partGrpSvc =
internalTable.partitionRaftGroupService(partId);
- IgniteBiTuple<Peer, Long> leaderWithTerm =
partGrpSvc.refreshAndGetLeaderWithTerm().join();
+ LeaderWithTerm leaderWithTerm =
partGrpSvc.refreshAndGetLeaderWithTerm().join();
// run update of raft configuration if this node is a
leader
- if
(localMember.name().equals(leaderWithTerm.get1().consistentId())) {
+ if
(localMember.name().equals(leaderWithTerm.leader().consistentId())) {
LOG.info("Current node={} is the leader of partition
raft group={}. "
+ "Initiate rebalance process for
partition={}, table={}",
localMember.address(), replicaGrpId, partId,
tbl.name());
- partGrpSvc.changePeersAsync(newNodes,
leaderWithTerm.get2()).join();
+ partGrpSvc.changePeersAsync(newNodes, List.of(),
leaderWithTerm.term()).join();
Review Comment:
Is it ok that we always clear the list of learners here?
--
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
To unsubscribe, e-mail: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]