rpuch commented on code in PR #5255: URL: https://github.com/apache/ignite-3/pull/5255#discussion_r1977280745
########## modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/ClusterInitializer.java: ########## @@ -184,6 +188,20 @@ public CompletableFuture<Void> initCluster( } } + /** + * Validates physical topology before initialization for duplicate consistent ids. Throws {@link InternalInitException} if such + * duplicate is found. + */ + private void validateTopology() { + Set<String> consistentIds = new HashSet<>(); + clusterService.topologyService().allMembers().forEach(node -> { + if (!consistentIds.add(node.name())) { + LOG.error("Initialization failed, node \"{}\" has duplicate in the physical topology", node.name()); + throw new InternalInitException(format("Duplicate consistent id \"{}\"", node.name()), true); Review Comment: Does this error message propagate to the user? If yes, is 'consistent ID' the term we use in the documentation? Should it be just 'name'? ########## modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/ClusterInitializer.java: ########## @@ -124,6 +126,8 @@ public CompletableFuture<Void> initCluster( } try { + validateTopology(); Review Comment: It seems that a race is possible here. 1. Nodes A, B are in the cluster (A, B are their names) 2. User requests an init with cmgNodes=msNodes={A} 3. We validate the topology, it's ok, no duplicates are found 4. Then 3rd node A' having the same name A joins the physical topology 5. Then CMG/MS nodes are resolved, and so on To fix this, you could take the topology snapshot, then validate and then use the snapshot to resolve the CMG/MS nodes. ########## modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/ClusterManagementGroupManager.java: ########## @@ -871,6 +883,17 @@ private void sendClusterState(CmgRaftService raftService, Collection<ClusterNode return nullCompletedFuture(); } + Collection<ClusterNode> duplicates = findDuplicateConsistentIds(topology, nodes); + if (!duplicates.isEmpty()) { + CancelJoinMessage msg = msgFactory.cancelJoinMessage() + .reason("Duplicate consistent id detected") + .build(); + for (ClusterNode duplicate : duplicates) { + sendWithRetry(duplicate, msg); + } + return nullCompletedFuture(); Review Comment: Imagine that a node becomes a CMG leader. While there was not leader, 2 nodes are added to the physical topology: A' and B, where A' is a duplicate of some existing node (A is already in the logical topology), and B has no duplicates. In such case, I would expect A' to get an error, but B would still get an invitation. But current code just refuses to send invitations to anyone if there is a single duplicate detected. It would be nice to have a test for this case. ########## modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/ClusterManagementGroupManager.java: ########## @@ -871,6 +883,17 @@ private void sendClusterState(CmgRaftService raftService, Collection<ClusterNode return nullCompletedFuture(); } + Collection<ClusterNode> duplicates = findDuplicateConsistentIds(topology, nodes); + if (!duplicates.isEmpty()) { + CancelJoinMessage msg = msgFactory.cancelJoinMessage() + .reason("Duplicate consistent id detected") Review Comment: Same thing about 'consistent ID' term; also, the name should probably be included in the message (even though we can deduce it, it might give us some clarity), so a message per recipient should be created ########## modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/ClusterManagementGroupManager.java: ########## @@ -631,6 +638,11 @@ private void handleCancelInit(CancelInitMessage msg) { this.scheduledExecutor.execute(this::destroyCmgWithEvents); } + private void handleCancelJoin(CancelJoinMessage msg) { + LOG.info("CMG initialization cancelled [reason={}]", msg.reason()); Review Comment: ```suggestion LOG.info("Join cancelled [reason={}]", msg.reason()); ``` ########## modules/network/src/main/java/org/apache/ignite/internal/network/netty/ConnectionManager.java: ########## @@ -297,32 +302,32 @@ public InetSocketAddress localAddress() { /** * Gets a {@link NettySender}, that sends data from this node to another node with the specified address. * - * @param consistentId Another node's consistent id. - * @param address Another node's address. + * @param nodeId Another node's id. + * @param address Another node's address. * @return Sender. */ - public OrderingFuture<NettySender> channel(@Nullable String consistentId, ChannelType type, InetSocketAddress address) { - return getChannelWithRetry(consistentId, type, address, 0); + public OrderingFuture<NettySender> channel(UUID nodeId, ChannelType type, InetSocketAddress address) { + return getChannelWithRetry(nodeId, type, address, 0); } private OrderingFuture<NettySender> getChannelWithRetry( - @Nullable String consistentId, + UUID nodeId, ChannelType type, InetSocketAddress address, int attempt ) { if (attempt > MAX_RETRIES_TO_OPEN_CHANNEL) { - return OrderingFuture.failedFuture(new IllegalStateException("Too many attempts to open channel to " + consistentId)); + return OrderingFuture.failedFuture(new IllegalStateException("Too many attempts to open channel to " + nodeId)); Review Comment: Let's also add node address to the error message ########## modules/runner/src/integrationTest/java/org/apache/ignite/internal/cluster/management/ItDuplicateNodesTest.java: ########## @@ -0,0 +1,172 @@ +/* + * 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; + +import static java.util.stream.Collectors.joining; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_CLIENT_PORT; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_HTTP_PORT; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_PORT; +import static org.apache.ignite.internal.testframework.IgniteTestUtils.shortTestMethodName; +import static org.apache.ignite.internal.testframework.IgniteTestUtils.testNodeName; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureExceptionMatcher.willThrow; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureExceptionMatcher.willThrowWithCauseOrSuppressed; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureMatcher.willCompleteSuccessfully; +import static org.awaitility.Awaitility.await; +import static org.hamcrest.MatcherAssert.assertThat; +import static org.hamcrest.Matchers.equalTo; +import static org.hamcrest.Matchers.hasSize; +import static org.hamcrest.Matchers.is; + +import java.lang.reflect.Method; +import java.nio.file.Path; +import java.util.Collection; +import java.util.HashMap; +import java.util.Map; +import java.util.stream.IntStream; +import org.apache.ignite.IgniteServer; +import org.apache.ignite.InitParameters; +import org.apache.ignite.internal.app.IgniteServerImpl; +import org.apache.ignite.internal.lang.IgniteStringFormatter; +import org.apache.ignite.internal.testframework.BaseIgniteAbstractTest; +import org.apache.ignite.internal.testframework.TestIgnitionManager; +import org.apache.ignite.internal.testframework.WorkDirectory; +import org.apache.ignite.internal.testframework.WorkDirectoryExtension; +import org.apache.ignite.network.ClusterNode; +import org.junit.jupiter.api.AfterEach; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.TestInfo; +import org.junit.jupiter.api.extension.ExtendWith; + +@ExtendWith(WorkDirectoryExtension.class) +class ItDuplicateNodesTest extends BaseIgniteAbstractTest { Review Comment: This says about duplicates, but it's not clear what is duplicated. You could rename the class to mention 'duplicate names', or explain this in a javadoc ########## modules/network/src/main/java/org/apache/ignite/internal/network/scalecube/ScaleCubeTopologyService.java: ########## @@ -57,8 +58,8 @@ final class ScaleCubeTopologyService extends AbstractTopologyService { /** Topology members from the network address to the cluster node.. */ private final ConcurrentMap<NetworkAddress, ClusterNode> members = new ConcurrentHashMap<>(); - /** Topology members map from the consistent id to the cluster node. */ - private final ConcurrentMap<String, ClusterNode> consistentIdToMemberMap = new ConcurrentHashMap<>(); + /** Topology members map from the consistent id to the map from the id to the cluster node. */ + private final ConcurrentMap<String, Map<UUID, ClusterNode>> consistentIdToMemberMap = new ConcurrentHashMap<>(); Review Comment: ```suggestion private final ConcurrentMap<String, Map<UUID, ClusterNode>> membersByConsistentId = new ConcurrentHashMap<>(); ``` ########## modules/network/src/main/java/org/apache/ignite/internal/network/scalecube/ScaleCubeTopologyService.java: ########## @@ -187,8 +190,9 @@ public ClusterNode getByAddress(NetworkAddress addr) { /** {@inheritDoc} */ @Override - public ClusterNode getByConsistentId(String consistentId) { - return consistentIdToMemberMap.get(consistentId); + public @Nullable ClusterNode getByConsistentId(String consistentId) { + Map<UUID, ClusterNode> nodes = consistentIdToMemberMap.get(consistentId); + return nodes != null ? nodes.values().iterator().next() : null; Review Comment: It seems that, even if node A' (having same name as node A) is not in the logical topology, it can still be returned here instead of node A. It seems that we look up a node by name in the following cases: 1. In `ClusterInitializer`, but this should go away when fixing the race I commented about 2. In 'business code' which only makes sense for a node in the logical topology So my suggestion is to listen to 'node joined the logical topology' events and only add a node to this map when it gets added to the LT, not to the PT. ########## modules/runner/src/integrationTest/java/org/apache/ignite/internal/cluster/management/ItDuplicateNodesTest.java: ########## @@ -0,0 +1,172 @@ +/* + * 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; + +import static java.util.stream.Collectors.joining; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_CLIENT_PORT; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_HTTP_PORT; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_PORT; +import static org.apache.ignite.internal.testframework.IgniteTestUtils.shortTestMethodName; +import static org.apache.ignite.internal.testframework.IgniteTestUtils.testNodeName; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureExceptionMatcher.willThrow; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureExceptionMatcher.willThrowWithCauseOrSuppressed; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureMatcher.willCompleteSuccessfully; +import static org.awaitility.Awaitility.await; +import static org.hamcrest.MatcherAssert.assertThat; +import static org.hamcrest.Matchers.equalTo; +import static org.hamcrest.Matchers.hasSize; +import static org.hamcrest.Matchers.is; + +import java.lang.reflect.Method; +import java.nio.file.Path; +import java.util.Collection; +import java.util.HashMap; +import java.util.Map; +import java.util.stream.IntStream; +import org.apache.ignite.IgniteServer; +import org.apache.ignite.InitParameters; +import org.apache.ignite.internal.app.IgniteServerImpl; +import org.apache.ignite.internal.lang.IgniteStringFormatter; +import org.apache.ignite.internal.testframework.BaseIgniteAbstractTest; +import org.apache.ignite.internal.testframework.TestIgnitionManager; +import org.apache.ignite.internal.testframework.WorkDirectory; +import org.apache.ignite.internal.testframework.WorkDirectoryExtension; +import org.apache.ignite.network.ClusterNode; +import org.junit.jupiter.api.AfterEach; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.TestInfo; +import org.junit.jupiter.api.extension.ExtendWith; + +@ExtendWith(WorkDirectoryExtension.class) +class ItDuplicateNodesTest extends BaseIgniteAbstractTest { + private static final String NODE_BOOTSTRAP_CFG_TEMPLATE = "ignite {\n" + + " network: {\n" + + " port: {},\n" + + " nodeFinder.netClusterNodes: [ {} ]\n" + + " },\n" + + " clientConnector.port: {},\n" + + " rest.port: {},\n" + + "}"; + + @WorkDirectory + private static Path WORK_DIR; + + private final Map<Integer, IgniteServer> servers = new HashMap<>(); + + @AfterEach + void shutdownNodes() { + servers.values().forEach(IgniteServer::shutdown); + servers.clear(); + } + + @Test + void physicalTopology(TestInfo testInfo) { + int nodesCount = 2; + + IgniteServer node1 = startEmbeddedNode(testInfo, false, 0, nodesCount); + IgniteServer node2 = startEmbeddedNode(testInfo, false, 1, nodesCount); + + assertThat(node1.name(), is(equalTo(node2.name()))); + + await().untilAsserted(() -> { + assertThat(getPhysicalTopologyMembers(node1), hasSize(nodesCount)); + assertThat(getPhysicalTopologyMembers(node2), hasSize(nodesCount)); + }); + } + + @Test + void logicalTopology(TestInfo testInfo) { + int nodesCount = 3; + + IgniteServer metaStorageAndCmgNode = startEmbeddedNode(testInfo, true, 0, nodesCount); + startEmbeddedNode(testInfo, false, 1, nodesCount); + startEmbeddedNode(testInfo, false, 2, nodesCount); Review Comment: This is pretty cryptic, it's difficult to understand which node will get which name, so who is a duplicate of whom and who is not. Is it possible to pass node name explicitly? Also, let's not pass `TestInfo` via a parameter, even if it's needed. It can be captured by a `@BeforeEach` method and put to a field; this will reduce noise in the test methods. ########## modules/cluster-management/src/test/java/org/apache/ignite/internal/cluster/management/ClusterInitializerTest.java: ########## @@ -200,11 +196,10 @@ void testInitNoCancel() { "cluster" ); - InternalInitException e = assertFutureThrows(InternalInitException.class, initFuture); - - assertThat(e.getMessage(), containsString(String.format("Got error response from node \"%s\": foobar", cmgNode.name()))); + String errorMessageFragment = String.format("Got error response from node \"%s\": foobar", cmgNode.name()); + assertThat(initFuture, willThrow(InternalInitException.class, errorMessageFragment)); - verify(messagingService, never()).send(eq(metastorageNode), any(CancelInitMessage.class)); + verify(messagingService, never()).send(eq(cmgNode), any(CancelInitMessage.class)); Review Comment: Why is this change introduced? ########## modules/cluster-management/src/test/java/org/apache/ignite/internal/cluster/management/ClusterInitializerTest.java: ########## @@ -235,16 +230,21 @@ void testInitIllegalArguments() { void testUnresolvableNode() { CompletableFuture<Void> initFuture = clusterInitializer.initCluster(List.of("foo"), List.of("bar"), "cluster"); - IllegalArgumentException e = assertFutureThrows(IllegalArgumentException.class, initFuture); - - assertThat(e.getMessage(), containsString("Node \"foo\" is not present in the physical topology")); + assertThat(initFuture, willThrow(IllegalArgumentException.class, "Node \"foo\" is not present in the physical topology")); } - private static <T extends Throwable> T assertFutureThrows(Class<T> expected, CompletableFuture<?> future) { - ExecutionException e = assertThrows(ExecutionException.class, () -> future.get(1, TimeUnit.SECONDS)); + @Test + void testDuplicateConsistentId() { + // Different nodes with same consistent ids + ClusterNode node1 = new ClusterNodeImpl(randomUUID(), "node", new NetworkAddress("foo", 123)); + ClusterNode node2 = new ClusterNodeImpl(randomUUID(), "node", new NetworkAddress("bar", 456)); + + when(topologyService.allMembers()).thenReturn(List.of(node1, node2)); + + CompletableFuture<Void> initFuture = clusterInitializer.initCluster(List.of(node1.name()), List.of(node1.name()), "cluster"); - assertThat(e.getCause(), isA(expected)); + assertThat(initFuture, willThrow(InternalInitException.class, "Duplicate consistent id \"node\"")); - return expected.cast(e.getCause()); + verify(messagingService, never()).invoke(any(ClusterNode.class), any(NetworkMessage.class), anyLong()); Review Comment: Let's also add a verification that the overload that uses consistent ID as first argument is never called ########## modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/ClusterManagementGroupManager.java: ########## @@ -895,6 +918,11 @@ private void sendClusterState(CmgRaftService raftService, Collection<ClusterNode }); } + private static Collection<ClusterNode> findDuplicateConsistentIds(LogicalTopologySnapshot topology, Collection<ClusterNode> nodes) { Review Comment: ```suggestion private static Collection<ClusterNode> findDuplicateConsistentIdsOfExistingNodes(LogicalTopologySnapshot existingTopology, Collection<ClusterNode> candidatesForAddition) { ``` ########## modules/network/src/main/java/org/apache/ignite/internal/network/netty/ConnectionManager.java: ########## @@ -111,8 +111,8 @@ public class ConnectionManager implements ChannelCreationListener { /** Message listeners. */ private final List<Consumer<InNetworkObject>> listeners = new CopyOnWriteArrayList<>(); - /** Node consistent id. */ - private final String consistentId; + /** Node id. */ Review Comment: ```suggestion /** Node ephemeral ID. */ ``` ########## modules/cluster-management/src/main/java/org/apache/ignite/internal/cluster/management/network/messages/CancelJoinMessage.java: ########## @@ -0,0 +1,32 @@ +/* + * 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.network.messages; + +import org.apache.ignite.internal.network.NetworkMessage; +import org.apache.ignite.internal.network.annotations.Transferable; + +/** + * Message signaling that the join process has failed and needs to be aborted. Review Comment: 'Cancellation' is something that might be initiated by any party, but in our case it's rather that the cluster refuses to join a node. Should it be something 'RefuseJoinMessage' or something similar? ########## modules/runner/src/integrationTest/java/org/apache/ignite/internal/cluster/management/ItDuplicateNodesTest.java: ########## @@ -0,0 +1,172 @@ +/* + * 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; + +import static java.util.stream.Collectors.joining; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_CLIENT_PORT; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_HTTP_PORT; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_PORT; +import static org.apache.ignite.internal.testframework.IgniteTestUtils.shortTestMethodName; +import static org.apache.ignite.internal.testframework.IgniteTestUtils.testNodeName; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureExceptionMatcher.willThrow; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureExceptionMatcher.willThrowWithCauseOrSuppressed; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureMatcher.willCompleteSuccessfully; +import static org.awaitility.Awaitility.await; +import static org.hamcrest.MatcherAssert.assertThat; +import static org.hamcrest.Matchers.equalTo; +import static org.hamcrest.Matchers.hasSize; +import static org.hamcrest.Matchers.is; + +import java.lang.reflect.Method; +import java.nio.file.Path; +import java.util.Collection; +import java.util.HashMap; +import java.util.Map; +import java.util.stream.IntStream; +import org.apache.ignite.IgniteServer; +import org.apache.ignite.InitParameters; +import org.apache.ignite.internal.app.IgniteServerImpl; +import org.apache.ignite.internal.lang.IgniteStringFormatter; +import org.apache.ignite.internal.testframework.BaseIgniteAbstractTest; +import org.apache.ignite.internal.testframework.TestIgnitionManager; +import org.apache.ignite.internal.testframework.WorkDirectory; +import org.apache.ignite.internal.testframework.WorkDirectoryExtension; +import org.apache.ignite.network.ClusterNode; +import org.junit.jupiter.api.AfterEach; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.TestInfo; +import org.junit.jupiter.api.extension.ExtendWith; + +@ExtendWith(WorkDirectoryExtension.class) +class ItDuplicateNodesTest extends BaseIgniteAbstractTest { + private static final String NODE_BOOTSTRAP_CFG_TEMPLATE = "ignite {\n" + + " network: {\n" + + " port: {},\n" + + " nodeFinder.netClusterNodes: [ {} ]\n" + + " },\n" + + " clientConnector.port: {},\n" + + " rest.port: {},\n" + + "}"; + + @WorkDirectory + private static Path WORK_DIR; + + private final Map<Integer, IgniteServer> servers = new HashMap<>(); + + @AfterEach + void shutdownNodes() { + servers.values().forEach(IgniteServer::shutdown); + servers.clear(); + } + + @Test + void physicalTopology(TestInfo testInfo) { Review Comment: Please either rename the method to explain what it tests, or add a javadoc about this (or both) ########## modules/runner/src/integrationTest/java/org/apache/ignite/internal/cluster/management/ItDuplicateNodesTest.java: ########## @@ -0,0 +1,172 @@ +/* + * 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; + +import static java.util.stream.Collectors.joining; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_CLIENT_PORT; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_HTTP_PORT; +import static org.apache.ignite.internal.ClusterConfiguration.DEFAULT_BASE_PORT; +import static org.apache.ignite.internal.testframework.IgniteTestUtils.shortTestMethodName; +import static org.apache.ignite.internal.testframework.IgniteTestUtils.testNodeName; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureExceptionMatcher.willThrow; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureExceptionMatcher.willThrowWithCauseOrSuppressed; +import static org.apache.ignite.internal.testframework.matchers.CompletableFutureMatcher.willCompleteSuccessfully; +import static org.awaitility.Awaitility.await; +import static org.hamcrest.MatcherAssert.assertThat; +import static org.hamcrest.Matchers.equalTo; +import static org.hamcrest.Matchers.hasSize; +import static org.hamcrest.Matchers.is; + +import java.lang.reflect.Method; +import java.nio.file.Path; +import java.util.Collection; +import java.util.HashMap; +import java.util.Map; +import java.util.stream.IntStream; +import org.apache.ignite.IgniteServer; +import org.apache.ignite.InitParameters; +import org.apache.ignite.internal.app.IgniteServerImpl; +import org.apache.ignite.internal.lang.IgniteStringFormatter; +import org.apache.ignite.internal.testframework.BaseIgniteAbstractTest; +import org.apache.ignite.internal.testframework.TestIgnitionManager; +import org.apache.ignite.internal.testframework.WorkDirectory; +import org.apache.ignite.internal.testframework.WorkDirectoryExtension; +import org.apache.ignite.network.ClusterNode; +import org.junit.jupiter.api.AfterEach; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.TestInfo; +import org.junit.jupiter.api.extension.ExtendWith; + +@ExtendWith(WorkDirectoryExtension.class) +class ItDuplicateNodesTest extends BaseIgniteAbstractTest { + private static final String NODE_BOOTSTRAP_CFG_TEMPLATE = "ignite {\n" + + " network: {\n" + + " port: {},\n" + + " nodeFinder.netClusterNodes: [ {} ]\n" + + " },\n" + + " clientConnector.port: {},\n" + + " rest.port: {},\n" + + "}"; + + @WorkDirectory + private static Path WORK_DIR; + + private final Map<Integer, IgniteServer> servers = new HashMap<>(); + + @AfterEach + void shutdownNodes() { + servers.values().forEach(IgniteServer::shutdown); + servers.clear(); + } + + @Test + void physicalTopology(TestInfo testInfo) { + int nodesCount = 2; + + IgniteServer node1 = startEmbeddedNode(testInfo, false, 0, nodesCount); + IgniteServer node2 = startEmbeddedNode(testInfo, false, 1, nodesCount); + + assertThat(node1.name(), is(equalTo(node2.name()))); + + await().untilAsserted(() -> { + assertThat(getPhysicalTopologyMembers(node1), hasSize(nodesCount)); + assertThat(getPhysicalTopologyMembers(node2), hasSize(nodesCount)); + }); + } + + @Test + void logicalTopology(TestInfo testInfo) { + int nodesCount = 3; + + IgniteServer metaStorageAndCmgNode = startEmbeddedNode(testInfo, true, 0, nodesCount); + startEmbeddedNode(testInfo, false, 1, nodesCount); + startEmbeddedNode(testInfo, false, 2, nodesCount); + + InitParameters initParameters = InitParameters.builder() + .metaStorageNodes(metaStorageAndCmgNode) + .clusterName("cluster") + .build(); + + // Can't init cluster with duplicate node names + assertThat( + metaStorageAndCmgNode.initClusterAsync(initParameters), + willThrow(InitException.class, "Unable to initialize the cluster: Duplicate consistent id") + ); + + // When duplicate node is stopped + stopNode(2); + + await().until(() -> getPhysicalTopologyMembers(metaStorageAndCmgNode), hasSize(2)); + + // Then cluster is initialized successfully + assertThat(metaStorageAndCmgNode.initClusterAsync(initParameters), willCompleteSuccessfully()); + + // New node with duplicate name can't join the cluster. It's added to the list of duplicate ids on nodes 0 and 1 and node 1 is added Review Comment: Where is that list of duplicate ids located? -- 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