You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@ratis.apache.org by GitBox <gi...@apache.org> on 2022/07/14 13:08:48 UTC

[GitHub] [ratis] codings-dan opened a new pull request, #682: RATIS-1593. Support CAS mode to setConfiguration

codings-dan opened a new pull request, #682:
URL: https://github.com/apache/ratis/pull/682

   see https://issues.apache.org/jira/browse/RATIS-1593


-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [ratis] codings-dan commented on pull request #682: RATIS-1593. Support CAS mode to setConfiguration

Posted by GitBox <gi...@apache.org>.
codings-dan commented on PR #682:
URL: https://github.com/apache/ratis/pull/682#issuecomment-1188507462

   @szetszwo PTAL, thank you!


-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [ratis] codings-dan commented on a diff in pull request #682: RATIS-1593. Support CAS mode to setConfiguration

Posted by GitBox <gi...@apache.org>.
codings-dan commented on code in PR #682:
URL: https://github.com/apache/ratis/pull/682#discussion_r923269763


##########
ratis-server/src/test/java/org/apache/ratis/server/impl/RaftReconfigurationBaseTest.java:
##########
@@ -216,6 +217,43 @@ private void runTestSetConfigurationInAddMode(CLUSTER cluster) throws Exception
     cluster.close();
   }
 
+  @Test
+  public void testSetConfigurationInCasMode() throws Exception {
+    runWithNewCluster(2, this::runTestSetConfigurationInCasMode);
+  }
+
+  private void runTestSetConfigurationInCasMode(CLUSTER cluster) throws Exception {
+    final RaftServer.Division leader = RaftTestUtil.waitForLeader(cluster);
+    List<RaftPeer> oldPeers = cluster.getPeers();
+
+    PeerChanges change = cluster.addNewPeers(1, true);
+    List<RaftPeer> peers = Arrays.asList(change.allPeersInNewConf);
+
+    try (final RaftClient client = cluster.createClient(leader.getId())) {
+      for (int i = 0; i < 10; i++) {
+        RaftClientReply reply = client.io().send(new SimpleMessage("m" + i));
+        Assert.assertTrue(reply.isSuccess());
+      }
+
+      testFailureCase("Can't set configuration in CAS mode ",
+          () -> client.admin().setConfiguration(SetConfigurationRequest.Arguments.newBuilder()
+              .setServersInNewConf(peers)
+              .setServersInCurConf(peers)
+              .setMode(SetConfigurationRequest.Mode.CAS)
+              .build()), SetConfigurationCasModeException.class);
+
+      RaftClientReply reply = client.admin().setConfiguration(
+          SetConfigurationRequest.Arguments.newBuilder()
+              .setServersInNewConf(peers)
+              .setServersInCurConf(oldPeers)

Review Comment:
   done



-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [ratis] codings-dan commented on pull request #682: RATIS-1593. Support CAS mode to setConfiguration

Posted by GitBox <gi...@apache.org>.
codings-dan commented on PR #682:
URL: https://github.com/apache/ratis/pull/682#issuecomment-1184472714

   @szetszwo Could you help review this pull request, thank you in advance!


-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [ratis] codings-dan commented on a diff in pull request #682: RATIS-1593. Support CAS mode to setConfiguration

Posted by GitBox <gi...@apache.org>.
codings-dan commented on code in PR #682:
URL: https://github.com/apache/ratis/pull/682#discussion_r923269477


##########
ratis-common/src/main/java/org/apache/ratis/protocol/SetConfigurationRequest.java:
##########
@@ -29,25 +29,41 @@ public class SetConfigurationRequest extends RaftClientRequest {
 
   public enum Mode {
     SET_UNCONDITIONALLY,
-    ADD
+    ADD,
+    CAS
   }
 
   public static final class Arguments {
     private final List<RaftPeer> serversInNewConf;
     private final List<RaftPeer> listenersInNewConf;
+    private final List<RaftPeer> serversInCurConf;
+    private final List<RaftPeer> listenersInCurConf;
     private final Mode mode;
 
-    private Arguments(List<RaftPeer> serversInNewConf, List<RaftPeer> listenersInNewConf,Mode mode) {
+    private Arguments(List<RaftPeer> serversInNewConf, List<RaftPeer> listenersInNewConf, Mode mode) {
+      this(serversInNewConf, listenersInNewConf, Collections.emptyList(), Collections.emptyList(), mode);
+    }

Review Comment:
   done



##########
ratis-common/src/main/java/org/apache/ratis/protocol/SetConfigurationRequest.java:
##########
@@ -59,6 +75,14 @@ public List<RaftPeer> getPeersInNewConf(RaftProtos.RaftPeerRole role) {
       }
     }
 
+    public List<RaftPeer> getListenersInCurConf() {
+      return listenersInCurConf;
+    }
+
+    public List<RaftPeer> getServersInCurConf() {
+      return serversInCurConf;
+    }

Review Comment:
   done



-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [ratis] codings-dan commented on a diff in pull request #682: RATIS-1593. Support CAS mode to setConfiguration

Posted by GitBox <gi...@apache.org>.
codings-dan commented on code in PR #682:
URL: https://github.com/apache/ratis/pull/682#discussion_r923272753


##########
ratis-server/src/main/java/org/apache/ratis/server/impl/RaftServerImpl.java:
##########
@@ -1108,6 +1109,17 @@ public CompletableFuture<RaftClientReply> setConfigurationAsync(SetConfiguration
       if (arguments.getMode() == SetConfigurationRequest.Mode.ADD) {
         serversInNewConf = add(RaftPeerRole.FOLLOWER, current, arguments);
         listenersInNewConf = add(RaftPeerRole.LISTENER, current, arguments);
+      } else if (arguments.getMode() == SetConfigurationRequest.Mode.CAS) {
+        List<RaftPeer> serversInCurConf = arguments.getServersInCurConf();
+        List<RaftPeer> listenersInCurConf = arguments.getListenersInCurConf();
+        if (serversInCurConf.equals(current.getAllPeers(RaftPeerRole.FOLLOWER))

Review Comment:
   Good catch! I have added the method and changed the related code.



-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [ratis] codings-dan commented on a diff in pull request #682: RATIS-1593. Support CAS mode to setConfiguration

Posted by GitBox <gi...@apache.org>.
codings-dan commented on code in PR #682:
URL: https://github.com/apache/ratis/pull/682#discussion_r923268823


##########
ratis-proto/src/main/proto/Raft.proto:
##########
@@ -409,13 +409,20 @@ message SetConfigurationRequestProto {
   enum Mode {
     SET_UNCONDITIONALLY = 0;
     ADD = 1;
+    CAS = 2;

Review Comment:
   done



-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [ratis] szetszwo commented on a diff in pull request #682: RATIS-1593. Support CAS mode to setConfiguration

Posted by GitBox <gi...@apache.org>.
szetszwo commented on code in PR #682:
URL: https://github.com/apache/ratis/pull/682#discussion_r922706289


##########
ratis-common/src/main/java/org/apache/ratis/protocol/exceptions/SetConfigurationCasModeException.java:
##########
@@ -0,0 +1,31 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.ratis.protocol.exceptions;
+
+import java.io.IOException;
+
+public class SetConfigurationCasModeException extends IOException {

Review Comment:
   Let's call it SetConfigurationException so that it can be used for the other cases and it should extend RaftException.



##########
ratis-common/src/main/java/org/apache/ratis/protocol/SetConfigurationRequest.java:
##########
@@ -29,25 +29,41 @@ public class SetConfigurationRequest extends RaftClientRequest {
 
   public enum Mode {
     SET_UNCONDITIONALLY,
-    ADD
+    ADD,
+    CAS
   }
 
   public static final class Arguments {
     private final List<RaftPeer> serversInNewConf;
     private final List<RaftPeer> listenersInNewConf;
+    private final List<RaftPeer> serversInCurConf;
+    private final List<RaftPeer> listenersInCurConf;
     private final Mode mode;
 
-    private Arguments(List<RaftPeer> serversInNewConf, List<RaftPeer> listenersInNewConf,Mode mode) {
+    private Arguments(List<RaftPeer> serversInNewConf, List<RaftPeer> listenersInNewConf, Mode mode) {
+      this(serversInNewConf, listenersInNewConf, Collections.emptyList(), Collections.emptyList(), mode);
+    }

Review Comment:
   This private constructor can be removed since it is not used anywhere.



##########
ratis-proto/src/main/proto/Raft.proto:
##########
@@ -409,13 +409,20 @@ message SetConfigurationRequestProto {
   enum Mode {
     SET_UNCONDITIONALLY = 0;
     ADD = 1;
+    CAS = 2;

Review Comment:
   Let's call it COMPARE_AND_SET.



##########
ratis-server/src/main/java/org/apache/ratis/server/impl/RaftServerImpl.java:
##########
@@ -1108,6 +1109,17 @@ public CompletableFuture<RaftClientReply> setConfigurationAsync(SetConfiguration
       if (arguments.getMode() == SetConfigurationRequest.Mode.ADD) {
         serversInNewConf = add(RaftPeerRole.FOLLOWER, current, arguments);
         listenersInNewConf = add(RaftPeerRole.LISTENER, current, arguments);
+      } else if (arguments.getMode() == SetConfigurationRequest.Mode.CAS) {
+        List<RaftPeer> serversInCurConf = arguments.getServersInCurConf();
+        List<RaftPeer> listenersInCurConf = arguments.getListenersInCurConf();
+        if (serversInCurConf.equals(current.getAllPeers(RaftPeerRole.FOLLOWER))

Review Comment:
   When two lists are in different orders, they become unequal.  We need an equalsIgnoreOrder method.



##########
ratis-proto/src/main/proto/Raft.proto:
##########
@@ -409,13 +409,20 @@ message SetConfigurationRequestProto {
   enum Mode {
     SET_UNCONDITIONALLY = 0;
     ADD = 1;
+    CAS = 2;
   }
   RaftRpcRequestProto rpcRequest = 1;
-  repeated RaftPeerProto peers = 2;
-  repeated RaftPeerProto listeners = 3;
+  SetConfigurationSevers servers = 2;

Review Comment:
   This is an incompatible change.  Let's simply add the new lists and use "currentPeers" for the current conf.
   ```
   message SetConfigurationRequestProto {
     enum Mode {
       SET_UNCONDITIONALLY = 0;
       ADD = 1;
       COMPARE_AND_SET = 2;
     }
     RaftRpcRequestProto rpcRequest = 1;
     repeated RaftPeerProto peers = 2;
     repeated RaftPeerProto listeners = 3;
     optional Mode mode = 4;
     repeated RaftPeerProto currentPeers = 5;
     repeated RaftPeerProto currentListeners = 6;
   }
   ```



##########
ratis-common/src/main/java/org/apache/ratis/protocol/SetConfigurationRequest.java:
##########
@@ -59,6 +75,14 @@ public List<RaftPeer> getPeersInNewConf(RaftProtos.RaftPeerRole role) {
       }
     }
 
+    public List<RaftPeer> getListenersInCurConf() {
+      return listenersInCurConf;
+    }
+
+    public List<RaftPeer> getServersInCurConf() {
+      return serversInCurConf;
+    }

Review Comment:
   Use "Current" instead of "Cur".



##########
ratis-server/src/test/java/org/apache/ratis/server/impl/RaftReconfigurationBaseTest.java:
##########
@@ -216,6 +217,43 @@ private void runTestSetConfigurationInAddMode(CLUSTER cluster) throws Exception
     cluster.close();
   }
 
+  @Test
+  public void testSetConfigurationInCasMode() throws Exception {
+    runWithNewCluster(2, this::runTestSetConfigurationInCasMode);
+  }
+
+  private void runTestSetConfigurationInCasMode(CLUSTER cluster) throws Exception {
+    final RaftServer.Division leader = RaftTestUtil.waitForLeader(cluster);
+    List<RaftPeer> oldPeers = cluster.getPeers();
+
+    PeerChanges change = cluster.addNewPeers(1, true);
+    List<RaftPeer> peers = Arrays.asList(change.allPeersInNewConf);
+
+    try (final RaftClient client = cluster.createClient(leader.getId())) {
+      for (int i = 0; i < 10; i++) {
+        RaftClientReply reply = client.io().send(new SimpleMessage("m" + i));
+        Assert.assertTrue(reply.isSuccess());
+      }
+
+      testFailureCase("Can't set configuration in CAS mode ",
+          () -> client.admin().setConfiguration(SetConfigurationRequest.Arguments.newBuilder()
+              .setServersInNewConf(peers)
+              .setServersInCurConf(peers)
+              .setMode(SetConfigurationRequest.Mode.CAS)
+              .build()), SetConfigurationCasModeException.class);
+
+      RaftClientReply reply = client.admin().setConfiguration(
+          SetConfigurationRequest.Arguments.newBuilder()
+              .setServersInNewConf(peers)
+              .setServersInCurConf(oldPeers)

Review Comment:
   Change oldPeers to a different order.
   ```
         Collections.shuffle(oldPeers);
   ```



##########
ratis-common/src/main/java/org/apache/ratis/protocol/SetConfigurationRequest.java:
##########
@@ -29,25 +29,41 @@ public class SetConfigurationRequest extends RaftClientRequest {
 
   public enum Mode {
     SET_UNCONDITIONALLY,
-    ADD
+    ADD,
+    CAS
   }
 
   public static final class Arguments {
     private final List<RaftPeer> serversInNewConf;
     private final List<RaftPeer> listenersInNewConf;
+    private final List<RaftPeer> serversInCurConf;
+    private final List<RaftPeer> listenersInCurConf;
     private final Mode mode;
 
-    private Arguments(List<RaftPeer> serversInNewConf, List<RaftPeer> listenersInNewConf,Mode mode) {
+    private Arguments(List<RaftPeer> serversInNewConf, List<RaftPeer> listenersInNewConf, Mode mode) {
+      this(serversInNewConf, listenersInNewConf, Collections.emptyList(), Collections.emptyList(), mode);
+    }
+
+    private Arguments(List<RaftPeer> serversInNewConf, List<RaftPeer> listenersInNewConf,
+        List<RaftPeer> serversInCurConf, List<RaftPeer> listenersInCurConf, Mode mode) {

Review Comment:
   Let's move the mode in the middle so that it harder to pass incorrect parameters.



-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [ratis] szetszwo merged pull request #682: RATIS-1593. Support CAS mode to setConfiguration

Posted by GitBox <gi...@apache.org>.
szetszwo merged PR #682:
URL: https://github.com/apache/ratis/pull/682


-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [ratis] codings-dan commented on a diff in pull request #682: RATIS-1593. Support CAS mode to setConfiguration

Posted by GitBox <gi...@apache.org>.
codings-dan commented on code in PR #682:
URL: https://github.com/apache/ratis/pull/682#discussion_r923269152


##########
ratis-proto/src/main/proto/Raft.proto:
##########
@@ -409,13 +409,20 @@ message SetConfigurationRequestProto {
   enum Mode {
     SET_UNCONDITIONALLY = 0;
     ADD = 1;
+    CAS = 2;
   }
   RaftRpcRequestProto rpcRequest = 1;
-  repeated RaftPeerProto peers = 2;
-  repeated RaftPeerProto listeners = 3;
+  SetConfigurationSevers servers = 2;

Review Comment:
   Good catch, I have fixed it



##########
ratis-common/src/main/java/org/apache/ratis/protocol/exceptions/SetConfigurationCasModeException.java:
##########
@@ -0,0 +1,31 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.ratis.protocol.exceptions;
+
+import java.io.IOException;
+
+public class SetConfigurationCasModeException extends IOException {

Review Comment:
   done



-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [ratis] codings-dan commented on a diff in pull request #682: RATIS-1593. Support CAS mode to setConfiguration

Posted by GitBox <gi...@apache.org>.
codings-dan commented on code in PR #682:
URL: https://github.com/apache/ratis/pull/682#discussion_r923271893


##########
ratis-common/src/main/java/org/apache/ratis/protocol/SetConfigurationRequest.java:
##########
@@ -29,25 +29,41 @@ public class SetConfigurationRequest extends RaftClientRequest {
 
   public enum Mode {
     SET_UNCONDITIONALLY,
-    ADD
+    ADD,
+    CAS
   }
 
   public static final class Arguments {
     private final List<RaftPeer> serversInNewConf;
     private final List<RaftPeer> listenersInNewConf;
+    private final List<RaftPeer> serversInCurConf;
+    private final List<RaftPeer> listenersInCurConf;
     private final Mode mode;
 
-    private Arguments(List<RaftPeer> serversInNewConf, List<RaftPeer> listenersInNewConf,Mode mode) {
+    private Arguments(List<RaftPeer> serversInNewConf, List<RaftPeer> listenersInNewConf, Mode mode) {
+      this(serversInNewConf, listenersInNewConf, Collections.emptyList(), Collections.emptyList(), mode);
+    }
+
+    private Arguments(List<RaftPeer> serversInNewConf, List<RaftPeer> listenersInNewConf,
+        List<RaftPeer> serversInCurConf, List<RaftPeer> listenersInCurConf, Mode mode) {

Review Comment:
   done



-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org