sanpwc commented on code in PR #6344: URL: https://github.com/apache/ignite-3/pull/6344#discussion_r2257586157
########## modules/raft/src/main/java/org/apache/ignite/raft/jraft/core/NodeImpl.java: ########## @@ -1841,6 +1851,90 @@ public void handleReadIndexRequest(final ReadIndexRequest request, } } + @Override + public void handleGetLeaderAndTermRequest(GetLeaderRequest request, RpcResponseClosure<GetLeaderResponse> done) { Review Comment: There are two modes `ReadOnlyOption` in raft: read index based and lease based. Do you properly utilize second option? See isLeaderLeaseValid for more details. ########## modules/raft/src/main/java/org/apache/ignite/raft/jraft/rpc/impl/cli/GetLeaderRequestProcessor.java: ########## @@ -16,23 +16,21 @@ */ package org.apache.ignite.raft.jraft.rpc.impl.cli; -import java.util.ArrayList; -import java.util.List; +import static java.util.concurrent.CompletableFuture.runAsync; import java.util.concurrent.Executor; import org.apache.ignite.raft.jraft.RaftMessagesFactory; -import org.apache.ignite.raft.jraft.Node; import org.apache.ignite.raft.jraft.Status; -import org.apache.ignite.raft.jraft.entity.PeerId; -import org.apache.ignite.raft.jraft.error.RaftError; import org.apache.ignite.raft.jraft.rpc.CliRequests.GetLeaderRequest; import org.apache.ignite.raft.jraft.rpc.Message; -import org.apache.ignite.raft.jraft.rpc.RaftRpcFactory; +import org.apache.ignite.raft.jraft.rpc.RaftServerService; import org.apache.ignite.raft.jraft.rpc.RpcRequestClosure; +import org.apache.ignite.raft.jraft.rpc.RpcResponseClosureAdapter; +import org.apache.ignite.raft.jraft.rpc.impl.core.NodeRequestProcessor; /** * Process get leader request. */ -public class GetLeaderRequestProcessor extends BaseCliRequestProcessor<GetLeaderRequest> { Review Comment: Why? Semantically GetLeaderRequestProcessor is CliRequestProcessor. ########## modules/raft/src/main/java/org/apache/ignite/raft/jraft/rpc/RaftClientService.java: ########## @@ -103,4 +103,16 @@ Future<Message> timeoutNow(final PeerId peerId, final RpcRequests.TimeoutNowRequ */ Future<Message> readIndex(final PeerId peerId, final RpcRequests.ReadIndexRequest request, final int timeoutMs, final RpcResponseClosure<RpcRequests.ReadIndexResponse> done); + + /** + * Send a get-leader-and-term request and handle the response with done. + * + * @param peerId destination peer ID + * @param request request data + * @param timeoutMs timeout millis + * @param done callback + * @return a future result + */ + Future<Message> getLeaderAndTerm(final PeerId peerId, final CliRequests.GetLeaderRequest request, final int timeoutMs, Review Comment: Should we instead send a redirect-proposal to client? I mean that instead of sending GetLeaderRequest to leader if original client-based one was processed on follower `case STATE_FOLLOWER:` we way response with redirect proposal in order to force client to re-send GetLeaderRequest to proposed leader. I though that it's common way to solve the leader-miss issue. ########## modules/raft/src/main/java/org/apache/ignite/raft/jraft/core/NodeImpl.java: ########## @@ -1841,6 +1851,90 @@ public void handleReadIndexRequest(final ReadIndexRequest request, } } + @Override + public void handleGetLeaderAndTermRequest(GetLeaderRequest request, RpcResponseClosure<GetLeaderResponse> done) { + final long startMs = Utils.monotonicMs(); + this.readLock.lock(); + try { + switch (this.state) { + case STATE_LEADER: + getLeaderFromLeader(done); + break; + case STATE_FOLLOWER: + getLeaderFromFollower(request, done); + break; + case STATE_TRANSFERRING: + done.run(new Status(RaftError.EBUSY, "Is transferring leadership.")); + break; + default: + done.run(new Status(RaftError.UNKNOWN, "Invalid state for getLeaderAndTerm: %s.", this.state)); + break; + } + } + finally { + this.readLock.unlock(); + this.metrics.recordLatency("handle-get-leader", Utils.monotonicMs() - startMs); + } + } + + private void getLeaderFromFollower(GetLeaderRequest request, RpcResponseClosure<GetLeaderResponse> closure) { + PeerId leaderId = this.leaderId; + + if (leaderId == null || leaderId.isEmpty()) { + closure.run(new Status(RaftError.UNKNOWN, "No leader at term %d.", this.currTerm)); + return; + } + // send request to leader. + final GetLeaderRequest newRequest = raftOptions.getRaftMessagesFactory() + .getLeaderRequest() + .groupId(request.groupId()) + .peerId(leaderId.toString()) + .build(); + + this.rpcClientService.getLeaderAndTerm(leaderId, newRequest, -1, closure); + } + + private void getLeaderFromLeader(RpcResponseClosure<GetLeaderResponse> closure) { + PeerId leaderId = this.leaderId; + + if (leaderId == null || leaderId.isEmpty()) { + closure.run(new Status(RaftError.UNKNOWN, "No leader at term %d.", this.currTerm)); + return; + } + + GetLeaderResponseBuilder respBuilder = raftOptions.getRaftMessagesFactory().getLeaderResponse() + .leaderId(leaderId.toString()) + .currentTerm(this.getCurrentTerm()); + + final int quorum = getQuorum(); + if (quorum <= 1) { + // Only one peer, fast path. + closure.setResponse(respBuilder.build()); + closure.run(Status.OK()); + return; + } + + final List<PeerId> peers = this.conf.getConf().getPeers(); Review Comment: Should we do it under the lock in order to prevent concurrent configuration changes? -- 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: notifications-unsubscr...@ignite.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org