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

rpuch pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/ignite-3.git


The following commit(s) were added to refs/heads/main by this push:
     new 5569ac836c IGNITE-22902 Implement command for changing 
metastorageNodes in CMG (#4345)
5569ac836c is described below

commit 5569ac836cf22e3fecc31fd3dbc08f6cd27eaa9c
Author: Roman Puchkovskiy <[email protected]>
AuthorDate: Fri Sep 6 11:09:22 2024 +0400

    IGNITE-22902 Implement command for changing metastorageNodes in CMG (#4345)
---
 .../management/ClusterManagementGroupManager.java  |  8 +++-
 .../network/messages/CmgMessageGroup.java          |  6 +++
 .../management/raft/CmgRaftGroupListener.java      | 25 ++++++++++
 .../commands/ChangeMetastorageNodesCommand.java    | 34 +++++++++++++
 .../management/raft/CmgRaftGroupListenerTest.java  | 56 ++++++++++++++++++----
 modules/system-disaster-recovery/README.md         |  2 +
 6 files changed, 121 insertions(+), 10 deletions(-)

diff --git 
a/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/ClusterManagementGroupManager.java
 
b/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/ClusterManagementGroupManager.java
index e937c5b951..b204d767b1 100644
--- 
a/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/ClusterManagementGroupManager.java
+++ 
b/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/ClusterManagementGroupManager.java
@@ -781,8 +781,12 @@ public class ClusterManagementGroupManager extends 
AbstractEventProducer<Cluster
                     .startRaftGroupNodeAndWaitNodeReadyFuture(
                             raftNodeId(serverPeer),
                             raftConfiguration,
-                            new CmgRaftGroupListener(clusterStateStorageMgr, 
logicalTopology, validationManager,
-                                    this::onLogicalTopologyChanged),
+                            new CmgRaftGroupListener(
+                                    clusterStateStorageMgr,
+                                    logicalTopology,
+                                    validationManager,
+                                    this::onLogicalTopologyChanged
+                            ),
                             this::onElectedAsLeader,
                             raftGroupOptionsConfigurer
                     )
diff --git 
a/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/network/messages/CmgMessageGroup.java
 
b/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/network/messages/CmgMessageGroup.java
index 8636640f74..c420b7c547 100644
--- 
a/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/network/messages/CmgMessageGroup.java
+++ 
b/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/network/messages/CmgMessageGroup.java
@@ -19,6 +19,7 @@ package 
org.apache.ignite.internal.cluster.management.network.messages;
 
 import org.apache.ignite.internal.cluster.management.ClusterState;
 import org.apache.ignite.internal.cluster.management.ClusterTag;
+import 
org.apache.ignite.internal.cluster.management.raft.commands.ChangeMetastorageNodesCommand;
 import 
org.apache.ignite.internal.cluster.management.raft.commands.ClusterNodeMessage;
 import 
org.apache.ignite.internal.cluster.management.raft.commands.InitCmgStateCommand;
 import 
org.apache.ignite.internal.cluster.management.raft.commands.JoinReadyCommand;
@@ -104,6 +105,11 @@ public class CmgMessageGroup {
          */
         int READ_VALIDATED_NODES = 46;
 
+        /**
+         * Message type for {@link ChangeMetastorageNodesCommand}.
+         */
+        int CHANGE_METASTORAGE_NODES = 47;
+
         /**
          * Message type for {@link ClusterNodeMessage}.
          */
diff --git 
a/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/raft/CmgRaftGroupListener.java
 
b/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/raft/CmgRaftGroupListener.java
index 31b1b6f887..61a51aec6d 100644
--- 
a/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/raft/CmgRaftGroupListener.java
+++ 
b/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/raft/CmgRaftGroupListener.java
@@ -32,6 +32,8 @@ import java.util.function.Consumer;
 import java.util.function.LongConsumer;
 import java.util.stream.Collectors;
 import org.apache.ignite.internal.cluster.management.ClusterState;
+import 
org.apache.ignite.internal.cluster.management.network.messages.CmgMessagesFactory;
+import 
org.apache.ignite.internal.cluster.management.raft.commands.ChangeMetastorageNodesCommand;
 import 
org.apache.ignite.internal.cluster.management.raft.commands.ClusterNodeMessage;
 import 
org.apache.ignite.internal.cluster.management.raft.commands.InitCmgStateCommand;
 import 
org.apache.ignite.internal.cluster.management.raft.commands.JoinReadyCommand;
@@ -70,6 +72,8 @@ public class CmgRaftGroupListener implements 
RaftGroupListener {
 
     private final LongConsumer onLogicalTopologyChanged;
 
+    private final CmgMessagesFactory cmgMessagesFactory = new 
CmgMessagesFactory();
+
     /**
      * Creates a new instance.
      *
@@ -152,6 +156,10 @@ public class CmgRaftGroupListener implements 
RaftGroupListener {
 
                 onLogicalTopologyChanged.accept(clo.term());
 
+                clo.result(null);
+            } else if (command instanceof ChangeMetastorageNodesCommand) {
+                changeMetastorageNodes((ChangeMetastorageNodesCommand) 
command);
+
                 clo.result(null);
             }
         }
@@ -244,6 +252,23 @@ public class CmgRaftGroupListener implements 
RaftGroupListener {
         }
     }
 
+    private void changeMetastorageNodes(ChangeMetastorageNodesCommand command) 
{
+        ClusterState existingState = storageManager.getClusterState();
+
+        assert existingState != null : "Cluster state is not initialized when 
got " + command;
+
+        ClusterState newState = cmgMessagesFactory.clusterState()
+                .cmgNodes(Set.copyOf(existingState.cmgNodes()))
+                .metaStorageNodes(Set.copyOf(command.metaStorageNodes()))
+                .version(existingState.version())
+                .clusterTag(existingState.clusterTag())
+                
.initialClusterConfiguration(existingState.initialClusterConfiguration())
+                .formerClusterIds(existingState.formerClusterIds())
+                .build();
+
+        storageManager.putClusterState(newState);
+    }
+
     @Override
     public void onSnapshotSave(Path path, Consumer<Throwable> doneClo) {
         storageManager.snapshot(path)
diff --git 
a/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/raft/commands/ChangeMetastorageNodesCommand.java
 
b/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/raft/commands/ChangeMetastorageNodesCommand.java
new file mode 100644
index 0000000000..98d27f1417
--- /dev/null
+++ 
b/modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/raft/commands/ChangeMetastorageNodesCommand.java
@@ -0,0 +1,34 @@
+/*
+ * 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.ignite.internal.cluster.management.raft.commands;
+
+import java.util.Set;
+import 
org.apache.ignite.internal.cluster.management.network.messages.CmgMessageGroup;
+import org.apache.ignite.internal.network.annotations.Transferable;
+import org.apache.ignite.internal.raft.WriteCommand;
+
+/**
+ * Command for changing metastorage nodes.
+ */
+@Transferable(CmgMessageGroup.Commands.CHANGE_METASTORAGE_NODES)
+public interface ChangeMetastorageNodesCommand extends WriteCommand {
+    /**
+     * Returns node names that host Meta storage.
+     */
+    Set<String> metaStorageNodes();
+}
diff --git 
a/modules/cluster-management/src/test/java/org/apache/ignite/internal/cluster/management/raft/CmgRaftGroupListenerTest.java
 
b/modules/cluster-management/src/test/java/org/apache/ignite/internal/cluster/management/raft/CmgRaftGroupListenerTest.java
index 5d6ca0a4b8..6e47162a9a 100644
--- 
a/modules/cluster-management/src/test/java/org/apache/ignite/internal/cluster/management/raft/CmgRaftGroupListenerTest.java
+++ 
b/modules/cluster-management/src/test/java/org/apache/ignite/internal/cluster/management/raft/CmgRaftGroupListenerTest.java
@@ -17,11 +17,14 @@
 
 package org.apache.ignite.internal.cluster.management.raft;
 
+import static java.util.UUID.randomUUID;
 import static 
org.apache.ignite.internal.testframework.matchers.CompletableFutureMatcher.willCompleteSuccessfully;
 import static org.hamcrest.MatcherAssert.assertThat;
 import static org.hamcrest.Matchers.contains;
+import static org.hamcrest.Matchers.containsInAnyOrder;
 import static org.hamcrest.Matchers.empty;
 import static org.hamcrest.Matchers.is;
+import static org.hamcrest.Matchers.notNullValue;
 import static org.junit.jupiter.api.Assertions.assertAll;
 import static org.junit.jupiter.api.Assertions.assertEquals;
 import static org.junit.jupiter.api.Assertions.assertNull;
@@ -29,7 +32,6 @@ import static org.junit.jupiter.api.Assertions.assertTrue;
 import static org.mockito.ArgumentMatchers.any;
 import static org.mockito.ArgumentMatchers.anyLong;
 import static org.mockito.Mockito.doNothing;
-import static org.mockito.Mockito.mock;
 import static org.mockito.Mockito.never;
 import static org.mockito.Mockito.spy;
 import static org.mockito.Mockito.verify;
@@ -40,7 +42,6 @@ import java.util.Collection;
 import java.util.Iterator;
 import java.util.List;
 import java.util.Set;
-import java.util.UUID;
 import java.util.function.LongConsumer;
 import org.apache.ignite.internal.cluster.management.ClusterState;
 import org.apache.ignite.internal.cluster.management.ClusterTag;
@@ -60,16 +61,23 @@ import org.jetbrains.annotations.Nullable;
 import org.junit.jupiter.api.AfterEach;
 import org.junit.jupiter.api.BeforeEach;
 import org.junit.jupiter.api.Test;
+import org.junit.jupiter.api.extension.ExtendWith;
+import org.mockito.Mock;
+import org.mockito.Spy;
+import org.mockito.junit.jupiter.MockitoExtension;
 
 /**
  * Tests for the {@link CmgRaftGroupListener}.
  */
+@ExtendWith(MockitoExtension.class)
 public class CmgRaftGroupListenerTest extends BaseIgniteAbstractTest {
     private final ClusterStateStorage storage = spy(new 
TestClusterStateStorage());
 
-    private final LongConsumer onLogicalTopologyChanged = 
mock(LongConsumer.class);
+    @Mock
+    private LongConsumer onLogicalTopologyChanged;
 
-    private final LogicalTopology logicalTopology = spy(new 
LogicalTopologyImpl(storage, new ConstantClusterIdSupplier(UUID.randomUUID())));
+    @Spy
+    private final LogicalTopology logicalTopology = new 
LogicalTopologyImpl(storage, new ConstantClusterIdSupplier(randomUUID()));
 
     private CmgRaftGroupListener listener;
 
@@ -78,10 +86,12 @@ public class CmgRaftGroupListenerTest extends 
BaseIgniteAbstractTest {
     private final ClusterTag clusterTag = 
ClusterTag.randomClusterTag(msgFactory, "cluster");
 
     private final ClusterState state = msgFactory.clusterState()
-            .cmgNodes(Set.copyOf(Set.of("foo")))
-            .metaStorageNodes(Set.copyOf(Set.of("bar")))
+            .cmgNodes(Set.of("foo"))
+            .metaStorageNodes(Set.of("bar"))
             .version(IgniteProductVersion.CURRENT_VERSION.toString())
             .clusterTag(clusterTag)
+            .initialClusterConfiguration("some-config")
+            .formerClusterIds(List.of(randomUUID(), randomUUID()))
             .build();
 
     private final ClusterNodeMessage node = 
msgFactory.clusterNodeMessage().id("foo").name("bar").host("localhost").port(666).build();
@@ -159,8 +169,8 @@ public class CmgRaftGroupListenerTest extends 
BaseIgniteAbstractTest {
     @Test
     void absentClusterConfigUpdateErasesClusterConfig() {
         ClusterState clusterState = msgFactory.clusterState()
-                .cmgNodes(Set.copyOf(Set.of("foo")))
-                .metaStorageNodes(Set.copyOf(Set.of("bar")))
+                .cmgNodes(Set.of("foo"))
+                .metaStorageNodes(Set.of("bar"))
                 .version(IgniteProductVersion.CURRENT_VERSION.toString())
                 .clusterTag(clusterTag)
                 .initialClusterConfiguration("config")
@@ -191,6 +201,36 @@ public class CmgRaftGroupListenerTest extends 
BaseIgniteAbstractTest {
         );
     }
 
+    @Test
+    void changeMetastorageNodesChangesMsNodes() {
+        initCmgAndChangeMgNodes();
+
+        ClusterState updatedState = 
listener.storageManager().getClusterState();
+        assertThat(updatedState, is(notNullValue()));
+
+        assertThat(updatedState.cmgNodes(), is(state.cmgNodes()));
+        assertThat(updatedState.metaStorageNodes(), 
containsInAnyOrder("new-ms-1", "new-ms-2"));
+        assertThat(updatedState.clusterTag(), is(state.clusterTag()));
+        assertThat(updatedState.version(), is(state.version()));
+        assertThat(updatedState.initialClusterConfiguration(), 
is(state.initialClusterConfiguration()));
+        assertThat(updatedState.formerClusterIds(), 
is(state.formerClusterIds()));
+    }
+
+    private void initCmgAndChangeMgNodes() {
+        listener.onWrite(iterator(
+                msgFactory.initCmgStateCommand()
+                        .clusterState(state)
+                        .node(node)
+                        .build()
+        ));
+
+        listener.onWrite(iterator(
+                msgFactory.changeMetastorageNodesCommand()
+                        .metaStorageNodes(Set.of("new-ms-1", "new-ms-2"))
+                        .build()
+        ));
+    }
+
     private static <T extends Command> Iterator<CommandClosure<T>> iterator(T 
obj) {
         CommandClosure<T> closure = new CommandClosure<>() {
             @Override
diff --git a/modules/system-disaster-recovery/README.md 
b/modules/system-disaster-recovery/README.md
index f62882b30e..74a0756f2a 100644
--- a/modules/system-disaster-recovery/README.md
+++ b/modules/system-disaster-recovery/README.md
@@ -1,3 +1,5 @@
 # System disaster recovery module
 
 This module provides capability to recover from disasters related to the 
system Raft groups (CMG and Metastorage group).
+
+See 
https://cwiki.apache.org/confluence/display/IGNITE/IEP-128%3A+CMG+and+Metastorage+Disaster+Recovery

Reply via email to