You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@iotdb.apache.org by GitBox <gi...@apache.org> on 2022/11/24 09:10:22 UTC

[GitHub] [iotdb] CRZbulabula opened a new pull request, #8133: [IOTDB-4334] Verify durable cases of RegionGroup extension

CRZbulabula opened a new pull request, #8133:
URL: https://github.com/apache/iotdb/pull/8133

   I'll add destription later~


-- 
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: reviews-unsubscribe@iotdb.apache.org

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


[GitHub] [iotdb] Beyyes commented on a diff in pull request #8133: [IOTDB-4334] Verify durable cases of RegionGroup extension

Posted by GitBox <gi...@apache.org>.
Beyyes commented on code in PR #8133:
URL: https://github.com/apache/iotdb/pull/8133#discussion_r1032430005


##########
integration-test/src/main/java/org/apache/iotdb/itbase/env/BaseEnv.java:
##########
@@ -189,12 +189,22 @@ default SessionPool getSessionPool(
   /** @return The index of ConfigNode-Leader in configNodeWrapperList */
   int getLeaderConfigNodeIndex() throws IOException, InterruptedException;
 
+  /** Start an existed ConfigNode */
   void startConfigNode(int index);
 
+  /** Shutdown an existed ConfigNode */
   void shutdownConfigNode(int index);
 
+  /** @return The TDataNodeLocation of the specified DataNode */
+  DataNodeWrapper getDataNodeWrapper(int index);
+
+  /** Register a new DataNode */
+  void registerNewDataNode();
+
+  /** Start an existed DataNode */
   void startDataNode(int index);
 
+  /** Start an existed DataNode */

Review Comment:
   ```suggestion
     /** Shutdown an existed DataNode */
   ```



-- 
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: reviews-unsubscribe@iotdb.apache.org

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


[GitHub] [iotdb] Caideyipi commented on a diff in pull request #8133: [IOTDB-4334] Verify durable cases of RegionGroup extension

Posted by GitBox <gi...@apache.org>.
Caideyipi commented on code in PR #8133:
URL: https://github.com/apache/iotdb/pull/8133#discussion_r1031981769


##########
integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionDurableIT.java:
##########
@@ -0,0 +1,626 @@
+/*
+ * 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.iotdb.confignode.it.partition;
+
+import org.apache.iotdb.common.rpc.thrift.TConsensusGroupId;
+import org.apache.iotdb.common.rpc.thrift.TDataNodeLocation;
+import org.apache.iotdb.common.rpc.thrift.TEndPoint;
+import org.apache.iotdb.common.rpc.thrift.TSStatus;
+import org.apache.iotdb.common.rpc.thrift.TSeriesPartitionSlot;
+import org.apache.iotdb.common.rpc.thrift.TTimePartitionSlot;
+import org.apache.iotdb.commons.client.sync.SyncConfigNodeIServiceClient;
+import org.apache.iotdb.commons.cluster.NodeStatus;
+import org.apache.iotdb.commons.cluster.RegionStatus;
+import org.apache.iotdb.commons.exception.IllegalPathException;
+import org.apache.iotdb.confignode.it.utils.ConfigNodeTestUtils;
+import org.apache.iotdb.confignode.rpc.thrift.TDataNodeInfo;
+import org.apache.iotdb.confignode.rpc.thrift.TDataPartitionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TDataPartitionTableResp;
+import org.apache.iotdb.confignode.rpc.thrift.TRegionInfo;
+import org.apache.iotdb.confignode.rpc.thrift.TSchemaPartitionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TSchemaPartitionTableResp;
+import org.apache.iotdb.confignode.rpc.thrift.TSetDataNodeStatusReq;
+import org.apache.iotdb.confignode.rpc.thrift.TSetStorageGroupReq;
+import org.apache.iotdb.confignode.rpc.thrift.TShowClusterResp;
+import org.apache.iotdb.confignode.rpc.thrift.TShowDataNodesResp;
+import org.apache.iotdb.confignode.rpc.thrift.TShowRegionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TShowRegionResp;
+import org.apache.iotdb.confignode.rpc.thrift.TStorageGroupSchema;
+import org.apache.iotdb.consensus.ConsensusFactory;
+import org.apache.iotdb.it.env.ConfigFactory;
+import org.apache.iotdb.it.env.DataNodeWrapper;
+import org.apache.iotdb.it.env.EnvFactory;
+import org.apache.iotdb.it.framework.IoTDBTestRunner;
+import org.apache.iotdb.itbase.category.ClusterIT;
+import org.apache.iotdb.rpc.TSStatusCode;
+
+import org.apache.thrift.TException;
+import org.junit.After;
+import org.junit.Assert;
+import org.junit.Before;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+import org.junit.runner.RunWith;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.IOException;
+import java.util.HashSet;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicBoolean;
+import java.util.concurrent.atomic.AtomicInteger;
+
+@RunWith(IoTDBTestRunner.class)
+@Category({ClusterIT.class})
+public class IoTDBPartitionDurableIT {
+
+  private static final Logger LOGGER = LoggerFactory.getLogger(IoTDBPartitionDurableIT.class);
+
+  private static String originalConfigNodeConsensusProtocolClass;
+  private static String originalSchemaRegionConsensusProtocolClass;
+  private static String originalDataRegionConsensusProtocolClass;
+  private static final String testConsensusProtocolClass = ConsensusFactory.RATIS_CONSENSUS;
+
+  private static int originalSchemaReplicationFactor;
+  private static int originalDataReplicationFactor;
+  private static final int testReplicationFactor = 3;
+
+  private static long originalTimePartitionInterval;
+  private static final long testTimePartitionInterval = 604800000;
+
+  private static final int testDataNodeId = 0;
+  private static final String sg = "root.sg";
+  final String d0 = sg + ".d0.s";
+  final String d1 = sg + ".d1.s";
+  private static final int testSeriesPartitionBatchSize = 1;
+  private static final int testTimePartitionBatchSize = 1;
+  private static final TEndPoint defaultEndPoint = new TEndPoint("-1", -1);
+  private static final TDataNodeLocation defaultDataNode =
+      new TDataNodeLocation(
+          -1,
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint));
+
+  @Before
+  public void setUp() throws Exception {
+    originalConfigNodeConsensusProtocolClass =
+        ConfigFactory.getConfig().getConfigNodeConsesusProtocolClass();
+    originalSchemaRegionConsensusProtocolClass =
+        ConfigFactory.getConfig().getSchemaRegionConsensusProtocolClass();
+    originalDataRegionConsensusProtocolClass =
+        ConfigFactory.getConfig().getDataRegionConsensusProtocolClass();
+    ConfigFactory.getConfig().setConfigNodeConsesusProtocolClass(testConsensusProtocolClass);
+    ConfigFactory.getConfig().setSchemaRegionConsensusProtocolClass(testConsensusProtocolClass);
+    ConfigFactory.getConfig().setDataRegionConsensusProtocolClass(testConsensusProtocolClass);
+
+    originalSchemaReplicationFactor = ConfigFactory.getConfig().getSchemaReplicationFactor();
+    originalDataReplicationFactor = ConfigFactory.getConfig().getDataReplicationFactor();
+    ConfigFactory.getConfig().setSchemaReplicationFactor(testReplicationFactor);
+    ConfigFactory.getConfig().setDataReplicationFactor(testReplicationFactor);
+
+    originalTimePartitionInterval = ConfigFactory.getConfig().getTimePartitionInterval();
+    ConfigFactory.getConfig().setTimePartitionInterval(testTimePartitionInterval);
+
+    // Init 1C3D environment
+    EnvFactory.getEnv().initClusterEnvironment(1, 3);
+
+    setStorageGroup();
+  }
+
+  private void setStorageGroup() throws IOException, InterruptedException, TException {
+    try (SyncConfigNodeIServiceClient client =
+        (SyncConfigNodeIServiceClient) EnvFactory.getEnv().getLeaderConfigNodeConnection()) {
+      TSetStorageGroupReq setStorageGroupReq = new TSetStorageGroupReq(new TStorageGroupSchema(sg));
+      TSStatus status = client.setStorageGroup(setStorageGroupReq);
+      Assert.assertEquals(TSStatusCode.SUCCESS_STATUS.getStatusCode(), status.getCode());
+    }
+  }
+
+  @After
+  public void tearDown() {
+    EnvFactory.getEnv().cleanAfterClass();
+
+    ConfigFactory.getConfig()
+        .setConfigNodeConsesusProtocolClass(originalConfigNodeConsensusProtocolClass);
+    ConfigFactory.getConfig()
+        .setSchemaRegionConsensusProtocolClass(originalSchemaRegionConsensusProtocolClass);
+    ConfigFactory.getConfig()
+        .setDataRegionConsensusProtocolClass(originalDataRegionConsensusProtocolClass);
+
+    ConfigFactory.getConfig().setSchemaReplicationFactor(originalSchemaReplicationFactor);
+    ConfigFactory.getConfig().setDataReplicationFactor(originalDataReplicationFactor);
+
+    ConfigFactory.getConfig().setTimePartitionInterval(originalTimePartitionInterval);
+  }
+
+  @Test
+  public void testRemovingDataNode()
+      throws IOException, InterruptedException, TException, IllegalPathException {
+    try (SyncConfigNodeIServiceClient client =
+        (SyncConfigNodeIServiceClient) EnvFactory.getEnv().getLeaderConfigNodeConnection()) {
+
+      /* Test getOrCreateSchemaPartition, ConfigNode should create SchemaPartition and return */
+      TSchemaPartitionReq schemaPartitionReq =
+          new TSchemaPartitionReq()
+              .setPathPatternTree(ConfigNodeTestUtils.generatePatternTreeBuffer(new String[] {d0}));
+      TSchemaPartitionTableResp schemaPartitionTableResp =
+          client.getOrCreateSchemaPartitionTable(schemaPartitionReq);
+      Assert.assertEquals(
+          TSStatusCode.SUCCESS_STATUS.getStatusCode(),
+          schemaPartitionTableResp.getStatus().getCode());
+      Map<String, Map<TSeriesPartitionSlot, TConsensusGroupId>> schemaPartitionTable =
+          schemaPartitionTableResp.getSchemaPartitionTable();
+      // Successfully create a SchemaPartition
+      Assert.assertTrue(schemaPartitionTable.containsKey(sg));
+      Assert.assertEquals(1, schemaPartitionTable.get(sg).size());
+
+      /* Check Region distribution */
+      TShowRegionResp showRegionResp = client.showRegion(new TShowRegionReq());
+      Assert.assertEquals(
+          TSStatusCode.SUCCESS_STATUS.getStatusCode(), showRegionResp.getStatus().getCode());
+      // Create exactly one RegionGroup
+      Assert.assertEquals(3, showRegionResp.getRegionInfoListSize());
+      // Each DataNode has exactly one Region
+      Set<Integer> dataNodeIdSet = new HashSet<>();
+      showRegionResp
+          .getRegionInfoList()
+          .forEach(regionInfo -> dataNodeIdSet.add(regionInfo.getDataNodeId()));
+      Assert.assertEquals(3, dataNodeIdSet.size());
+
+      /* Change the NodeStatus of the test DataNode to Removing */
+      TSetDataNodeStatusReq setDataNodeStatusReq = new TSetDataNodeStatusReq();
+      DataNodeWrapper dataNodeWrapper = EnvFactory.getEnv().getDataNodeWrapper(testDataNodeId);
+      setDataNodeStatusReq.setTargetDataNode(
+          new TDataNodeLocation(defaultDataNode)
+              .setInternalEndPoint(
+                  new TEndPoint()
+                      .setIp(dataNodeWrapper.getInternalAddress())
+                      .setPort(dataNodeWrapper.getInternalPort())));
+      setDataNodeStatusReq.setStatus(NodeStatus.Removing.getStatus());
+      client.setDataNodeStatus(setDataNodeStatusReq);
+      // Waiting for heartbeat update
+      while (true) {
+        AtomicBoolean containRemoving = new AtomicBoolean(false);
+        TShowDataNodesResp showDataNodesResp = client.showDataNodes();
+        showDataNodesResp
+            .getDataNodesInfoList()
+            .forEach(
+                dataNodeInfo -> {
+                  if (NodeStatus.Removing.getStatus().equals(dataNodeInfo.getStatus())) {
+                    containRemoving.set(true);
+                  }
+                });
+
+        if (containRemoving.get()) {
+          break;
+        }
+        TimeUnit.SECONDS.sleep(1);
+      }
+
+      /* Test getOrCreateSchemaPartition, the result should be NO_ENOUGH_DATANODE */

Review Comment:
   ```suggestion
         /* Test getOrCreateSchemaPartition, the result should be NO_AVAILABLE_REGION_GROUP */
   ```



-- 
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: reviews-unsubscribe@iotdb.apache.org

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


[GitHub] [iotdb] Caideyipi commented on a diff in pull request #8133: [IOTDB-4334] Verify durable cases of RegionGroup extension

Posted by GitBox <gi...@apache.org>.
Caideyipi commented on code in PR #8133:
URL: https://github.com/apache/iotdb/pull/8133#discussion_r1031974032


##########
integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionDurableIT.java:
##########
@@ -0,0 +1,626 @@
+/*
+ * 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.iotdb.confignode.it.partition;
+
+import org.apache.iotdb.common.rpc.thrift.TConsensusGroupId;
+import org.apache.iotdb.common.rpc.thrift.TDataNodeLocation;
+import org.apache.iotdb.common.rpc.thrift.TEndPoint;
+import org.apache.iotdb.common.rpc.thrift.TSStatus;
+import org.apache.iotdb.common.rpc.thrift.TSeriesPartitionSlot;
+import org.apache.iotdb.common.rpc.thrift.TTimePartitionSlot;
+import org.apache.iotdb.commons.client.sync.SyncConfigNodeIServiceClient;
+import org.apache.iotdb.commons.cluster.NodeStatus;
+import org.apache.iotdb.commons.cluster.RegionStatus;
+import org.apache.iotdb.commons.exception.IllegalPathException;
+import org.apache.iotdb.confignode.it.utils.ConfigNodeTestUtils;
+import org.apache.iotdb.confignode.rpc.thrift.TDataNodeInfo;
+import org.apache.iotdb.confignode.rpc.thrift.TDataPartitionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TDataPartitionTableResp;
+import org.apache.iotdb.confignode.rpc.thrift.TRegionInfo;
+import org.apache.iotdb.confignode.rpc.thrift.TSchemaPartitionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TSchemaPartitionTableResp;
+import org.apache.iotdb.confignode.rpc.thrift.TSetDataNodeStatusReq;
+import org.apache.iotdb.confignode.rpc.thrift.TSetStorageGroupReq;
+import org.apache.iotdb.confignode.rpc.thrift.TShowClusterResp;
+import org.apache.iotdb.confignode.rpc.thrift.TShowDataNodesResp;
+import org.apache.iotdb.confignode.rpc.thrift.TShowRegionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TShowRegionResp;
+import org.apache.iotdb.confignode.rpc.thrift.TStorageGroupSchema;
+import org.apache.iotdb.consensus.ConsensusFactory;
+import org.apache.iotdb.it.env.ConfigFactory;
+import org.apache.iotdb.it.env.DataNodeWrapper;
+import org.apache.iotdb.it.env.EnvFactory;
+import org.apache.iotdb.it.framework.IoTDBTestRunner;
+import org.apache.iotdb.itbase.category.ClusterIT;
+import org.apache.iotdb.rpc.TSStatusCode;
+
+import org.apache.thrift.TException;
+import org.junit.After;
+import org.junit.Assert;
+import org.junit.Before;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+import org.junit.runner.RunWith;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.IOException;
+import java.util.HashSet;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicBoolean;
+import java.util.concurrent.atomic.AtomicInteger;
+
+@RunWith(IoTDBTestRunner.class)
+@Category({ClusterIT.class})
+public class IoTDBPartitionDurableIT {
+
+  private static final Logger LOGGER = LoggerFactory.getLogger(IoTDBPartitionDurableIT.class);
+
+  private static String originalConfigNodeConsensusProtocolClass;
+  private static String originalSchemaRegionConsensusProtocolClass;
+  private static String originalDataRegionConsensusProtocolClass;
+  private static final String testConsensusProtocolClass = ConsensusFactory.RATIS_CONSENSUS;
+
+  private static int originalSchemaReplicationFactor;
+  private static int originalDataReplicationFactor;
+  private static final int testReplicationFactor = 3;
+
+  private static long originalTimePartitionInterval;
+  private static final long testTimePartitionInterval = 604800000;
+
+  private static final int testDataNodeId = 0;
+  private static final String sg = "root.sg";
+  final String d0 = sg + ".d0.s";
+  final String d1 = sg + ".d1.s";
+  private static final int testSeriesPartitionBatchSize = 1;
+  private static final int testTimePartitionBatchSize = 1;
+  private static final TEndPoint defaultEndPoint = new TEndPoint("-1", -1);
+  private static final TDataNodeLocation defaultDataNode =
+      new TDataNodeLocation(
+          -1,
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint));
+
+  @Before
+  public void setUp() throws Exception {
+    originalConfigNodeConsensusProtocolClass =
+        ConfigFactory.getConfig().getConfigNodeConsesusProtocolClass();
+    originalSchemaRegionConsensusProtocolClass =
+        ConfigFactory.getConfig().getSchemaRegionConsensusProtocolClass();
+    originalDataRegionConsensusProtocolClass =
+        ConfigFactory.getConfig().getDataRegionConsensusProtocolClass();
+    ConfigFactory.getConfig().setConfigNodeConsesusProtocolClass(testConsensusProtocolClass);
+    ConfigFactory.getConfig().setSchemaRegionConsensusProtocolClass(testConsensusProtocolClass);
+    ConfigFactory.getConfig().setDataRegionConsensusProtocolClass(testConsensusProtocolClass);
+
+    originalSchemaReplicationFactor = ConfigFactory.getConfig().getSchemaReplicationFactor();
+    originalDataReplicationFactor = ConfigFactory.getConfig().getDataReplicationFactor();
+    ConfigFactory.getConfig().setSchemaReplicationFactor(testReplicationFactor);
+    ConfigFactory.getConfig().setDataReplicationFactor(testReplicationFactor);
+
+    originalTimePartitionInterval = ConfigFactory.getConfig().getTimePartitionInterval();
+    ConfigFactory.getConfig().setTimePartitionInterval(testTimePartitionInterval);
+
+    // Init 1C3D environment
+    EnvFactory.getEnv().initClusterEnvironment(1, 3);
+
+    setStorageGroup();
+  }
+
+  private void setStorageGroup() throws IOException, InterruptedException, TException {
+    try (SyncConfigNodeIServiceClient client =
+        (SyncConfigNodeIServiceClient) EnvFactory.getEnv().getLeaderConfigNodeConnection()) {
+      TSetStorageGroupReq setStorageGroupReq = new TSetStorageGroupReq(new TStorageGroupSchema(sg));
+      TSStatus status = client.setStorageGroup(setStorageGroupReq);
+      Assert.assertEquals(TSStatusCode.SUCCESS_STATUS.getStatusCode(), status.getCode());
+    }
+  }
+
+  @After
+  public void tearDown() {
+    EnvFactory.getEnv().cleanAfterClass();
+
+    ConfigFactory.getConfig()
+        .setConfigNodeConsesusProtocolClass(originalConfigNodeConsensusProtocolClass);
+    ConfigFactory.getConfig()
+        .setSchemaRegionConsensusProtocolClass(originalSchemaRegionConsensusProtocolClass);
+    ConfigFactory.getConfig()
+        .setDataRegionConsensusProtocolClass(originalDataRegionConsensusProtocolClass);
+
+    ConfigFactory.getConfig().setSchemaReplicationFactor(originalSchemaReplicationFactor);
+    ConfigFactory.getConfig().setDataReplicationFactor(originalDataReplicationFactor);
+
+    ConfigFactory.getConfig().setTimePartitionInterval(originalTimePartitionInterval);
+  }
+
+  @Test
+  public void testRemovingDataNode()
+      throws IOException, InterruptedException, TException, IllegalPathException {
+    try (SyncConfigNodeIServiceClient client =
+        (SyncConfigNodeIServiceClient) EnvFactory.getEnv().getLeaderConfigNodeConnection()) {
+
+      /* Test getOrCreateSchemaPartition, ConfigNode should create SchemaPartition and return */
+      TSchemaPartitionReq schemaPartitionReq =
+          new TSchemaPartitionReq()
+              .setPathPatternTree(ConfigNodeTestUtils.generatePatternTreeBuffer(new String[] {d0}));
+      TSchemaPartitionTableResp schemaPartitionTableResp =
+          client.getOrCreateSchemaPartitionTable(schemaPartitionReq);
+      Assert.assertEquals(
+          TSStatusCode.SUCCESS_STATUS.getStatusCode(),
+          schemaPartitionTableResp.getStatus().getCode());
+      Map<String, Map<TSeriesPartitionSlot, TConsensusGroupId>> schemaPartitionTable =
+          schemaPartitionTableResp.getSchemaPartitionTable();
+      // Successfully create a SchemaPartition
+      Assert.assertTrue(schemaPartitionTable.containsKey(sg));
+      Assert.assertEquals(1, schemaPartitionTable.get(sg).size());
+
+      /* Check Region distribution */
+      TShowRegionResp showRegionResp = client.showRegion(new TShowRegionReq());
+      Assert.assertEquals(
+          TSStatusCode.SUCCESS_STATUS.getStatusCode(), showRegionResp.getStatus().getCode());
+      // Create exactly one RegionGroup
+      Assert.assertEquals(3, showRegionResp.getRegionInfoListSize());
+      // Each DataNode has exactly one Region
+      Set<Integer> dataNodeIdSet = new HashSet<>();
+      showRegionResp
+          .getRegionInfoList()
+          .forEach(regionInfo -> dataNodeIdSet.add(regionInfo.getDataNodeId()));
+      Assert.assertEquals(3, dataNodeIdSet.size());
+
+      /* Change the NodeStatus of the test DataNode to Removing */
+      TSetDataNodeStatusReq setDataNodeStatusReq = new TSetDataNodeStatusReq();
+      DataNodeWrapper dataNodeWrapper = EnvFactory.getEnv().getDataNodeWrapper(testDataNodeId);
+      setDataNodeStatusReq.setTargetDataNode(
+          new TDataNodeLocation(defaultDataNode)
+              .setInternalEndPoint(
+                  new TEndPoint()
+                      .setIp(dataNodeWrapper.getInternalAddress())
+                      .setPort(dataNodeWrapper.getInternalPort())));
+      setDataNodeStatusReq.setStatus(NodeStatus.Removing.getStatus());
+      client.setDataNodeStatus(setDataNodeStatusReq);
+      // Waiting for heartbeat update
+      while (true) {
+        AtomicBoolean containRemoving = new AtomicBoolean(false);
+        TShowDataNodesResp showDataNodesResp = client.showDataNodes();
+        showDataNodesResp
+            .getDataNodesInfoList()
+            .forEach(
+                dataNodeInfo -> {
+                  if (NodeStatus.Removing.getStatus().equals(dataNodeInfo.getStatus())) {
+                    containRemoving.set(true);
+                  }
+                });
+
+        if (containRemoving.get()) {
+          break;
+        }
+        TimeUnit.SECONDS.sleep(1);
+      }
+
+      /* Test getOrCreateSchemaPartition, the result should be NO_ENOUGH_DATANODE */
+      schemaPartitionReq =
+          new TSchemaPartitionReq()
+              .setPathPatternTree(ConfigNodeTestUtils.generatePatternTreeBuffer(new String[] {d1}));
+      schemaPartitionTableResp = client.getOrCreateSchemaPartitionTable(schemaPartitionReq);
+      Assert.assertEquals(
+          TSStatusCode.NO_AVAILABLE_REGION_GROUP.getStatusCode(),

Review Comment:
   ```suggestion
             TSStatusCode.NO_ENOUGH_DATANODE.getStatusCode(),
   ```



-- 
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: reviews-unsubscribe@iotdb.apache.org

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


[GitHub] [iotdb] Beyyes merged pull request #8133: [IOTDB-4334] Verify durable cases of RegionGroup extension

Posted by GitBox <gi...@apache.org>.
Beyyes merged PR #8133:
URL: https://github.com/apache/iotdb/pull/8133


-- 
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: reviews-unsubscribe@iotdb.apache.org

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


[GitHub] [iotdb] Beyyes commented on a diff in pull request #8133: [IOTDB-4334] Verify durable cases of RegionGroup extension

Posted by GitBox <gi...@apache.org>.
Beyyes commented on code in PR #8133:
URL: https://github.com/apache/iotdb/pull/8133#discussion_r1032443327


##########
integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionGetterIT.java:
##########
@@ -94,9 +96,9 @@ public class IoTDBPartitionGetterIT {
 
   private static final String sg = "root.sg";
   private static final int storageGroupNum = 5;
-  private static final int seriesPartitionSlotsNum = 10000;
-  private static final int seriesPartitionBatchSize = 100;
-  private static final int timePartitionSlotsNum = 10;
+  private static final int testSeriesPartitionSlotNum = 100;

Review Comment:
   This test fails twice



-- 
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: reviews-unsubscribe@iotdb.apache.org

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


[GitHub] [iotdb] CRZbulabula commented on a diff in pull request #8133: [IOTDB-4334] Verify durable cases of RegionGroup extension

Posted by GitBox <gi...@apache.org>.
CRZbulabula commented on code in PR #8133:
URL: https://github.com/apache/iotdb/pull/8133#discussion_r1032451043


##########
integration-test/src/main/java/org/apache/iotdb/itbase/env/BaseEnv.java:
##########
@@ -189,12 +189,22 @@ default SessionPool getSessionPool(
   /** @return The index of ConfigNode-Leader in configNodeWrapperList */
   int getLeaderConfigNodeIndex() throws IOException, InterruptedException;
 
+  /** Start an existed ConfigNode */
   void startConfigNode(int index);
 
+  /** Shutdown an existed ConfigNode */
   void shutdownConfigNode(int index);
 
+  /** @return The TDataNodeLocation of the specified DataNode */
+  DataNodeWrapper getDataNodeWrapper(int index);
+
+  /** Register a new DataNode */
+  void registerNewDataNode();
+
+  /** Start an existed DataNode */
   void startDataNode(int index);
 
+  /** Start an existed DataNode */

Review Comment:
   FIxed~



-- 
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: reviews-unsubscribe@iotdb.apache.org

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


[GitHub] [iotdb] CRZbulabula commented on a diff in pull request #8133: [IOTDB-4334] Verify durable cases of RegionGroup extension

Posted by GitBox <gi...@apache.org>.
CRZbulabula commented on code in PR #8133:
URL: https://github.com/apache/iotdb/pull/8133#discussion_r1032033975


##########
integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionDurableIT.java:
##########
@@ -0,0 +1,626 @@
+/*
+ * 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.iotdb.confignode.it.partition;
+
+import org.apache.iotdb.common.rpc.thrift.TConsensusGroupId;
+import org.apache.iotdb.common.rpc.thrift.TDataNodeLocation;
+import org.apache.iotdb.common.rpc.thrift.TEndPoint;
+import org.apache.iotdb.common.rpc.thrift.TSStatus;
+import org.apache.iotdb.common.rpc.thrift.TSeriesPartitionSlot;
+import org.apache.iotdb.common.rpc.thrift.TTimePartitionSlot;
+import org.apache.iotdb.commons.client.sync.SyncConfigNodeIServiceClient;
+import org.apache.iotdb.commons.cluster.NodeStatus;
+import org.apache.iotdb.commons.cluster.RegionStatus;
+import org.apache.iotdb.commons.exception.IllegalPathException;
+import org.apache.iotdb.confignode.it.utils.ConfigNodeTestUtils;
+import org.apache.iotdb.confignode.rpc.thrift.TDataNodeInfo;
+import org.apache.iotdb.confignode.rpc.thrift.TDataPartitionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TDataPartitionTableResp;
+import org.apache.iotdb.confignode.rpc.thrift.TRegionInfo;
+import org.apache.iotdb.confignode.rpc.thrift.TSchemaPartitionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TSchemaPartitionTableResp;
+import org.apache.iotdb.confignode.rpc.thrift.TSetDataNodeStatusReq;
+import org.apache.iotdb.confignode.rpc.thrift.TSetStorageGroupReq;
+import org.apache.iotdb.confignode.rpc.thrift.TShowClusterResp;
+import org.apache.iotdb.confignode.rpc.thrift.TShowDataNodesResp;
+import org.apache.iotdb.confignode.rpc.thrift.TShowRegionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TShowRegionResp;
+import org.apache.iotdb.confignode.rpc.thrift.TStorageGroupSchema;
+import org.apache.iotdb.consensus.ConsensusFactory;
+import org.apache.iotdb.it.env.ConfigFactory;
+import org.apache.iotdb.it.env.DataNodeWrapper;
+import org.apache.iotdb.it.env.EnvFactory;
+import org.apache.iotdb.it.framework.IoTDBTestRunner;
+import org.apache.iotdb.itbase.category.ClusterIT;
+import org.apache.iotdb.rpc.TSStatusCode;
+
+import org.apache.thrift.TException;
+import org.junit.After;
+import org.junit.Assert;
+import org.junit.Before;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+import org.junit.runner.RunWith;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.IOException;
+import java.util.HashSet;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicBoolean;
+import java.util.concurrent.atomic.AtomicInteger;
+
+@RunWith(IoTDBTestRunner.class)
+@Category({ClusterIT.class})
+public class IoTDBPartitionDurableIT {
+
+  private static final Logger LOGGER = LoggerFactory.getLogger(IoTDBPartitionDurableIT.class);
+
+  private static String originalConfigNodeConsensusProtocolClass;
+  private static String originalSchemaRegionConsensusProtocolClass;
+  private static String originalDataRegionConsensusProtocolClass;
+  private static final String testConsensusProtocolClass = ConsensusFactory.RATIS_CONSENSUS;
+
+  private static int originalSchemaReplicationFactor;
+  private static int originalDataReplicationFactor;
+  private static final int testReplicationFactor = 3;
+
+  private static long originalTimePartitionInterval;
+  private static final long testTimePartitionInterval = 604800000;
+
+  private static final int testDataNodeId = 0;
+  private static final String sg = "root.sg";
+  final String d0 = sg + ".d0.s";
+  final String d1 = sg + ".d1.s";
+  private static final int testSeriesPartitionBatchSize = 1;
+  private static final int testTimePartitionBatchSize = 1;
+  private static final TEndPoint defaultEndPoint = new TEndPoint("-1", -1);
+  private static final TDataNodeLocation defaultDataNode =
+      new TDataNodeLocation(
+          -1,
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint));
+
+  @Before
+  public void setUp() throws Exception {
+    originalConfigNodeConsensusProtocolClass =
+        ConfigFactory.getConfig().getConfigNodeConsesusProtocolClass();
+    originalSchemaRegionConsensusProtocolClass =
+        ConfigFactory.getConfig().getSchemaRegionConsensusProtocolClass();
+    originalDataRegionConsensusProtocolClass =
+        ConfigFactory.getConfig().getDataRegionConsensusProtocolClass();
+    ConfigFactory.getConfig().setConfigNodeConsesusProtocolClass(testConsensusProtocolClass);
+    ConfigFactory.getConfig().setSchemaRegionConsensusProtocolClass(testConsensusProtocolClass);
+    ConfigFactory.getConfig().setDataRegionConsensusProtocolClass(testConsensusProtocolClass);
+
+    originalSchemaReplicationFactor = ConfigFactory.getConfig().getSchemaReplicationFactor();
+    originalDataReplicationFactor = ConfigFactory.getConfig().getDataReplicationFactor();
+    ConfigFactory.getConfig().setSchemaReplicationFactor(testReplicationFactor);
+    ConfigFactory.getConfig().setDataReplicationFactor(testReplicationFactor);
+
+    originalTimePartitionInterval = ConfigFactory.getConfig().getTimePartitionInterval();
+    ConfigFactory.getConfig().setTimePartitionInterval(testTimePartitionInterval);
+
+    // Init 1C3D environment
+    EnvFactory.getEnv().initClusterEnvironment(1, 3);
+
+    setStorageGroup();
+  }
+
+  private void setStorageGroup() throws IOException, InterruptedException, TException {
+    try (SyncConfigNodeIServiceClient client =
+        (SyncConfigNodeIServiceClient) EnvFactory.getEnv().getLeaderConfigNodeConnection()) {
+      TSetStorageGroupReq setStorageGroupReq = new TSetStorageGroupReq(new TStorageGroupSchema(sg));
+      TSStatus status = client.setStorageGroup(setStorageGroupReq);
+      Assert.assertEquals(TSStatusCode.SUCCESS_STATUS.getStatusCode(), status.getCode());
+    }
+  }
+
+  @After
+  public void tearDown() {
+    EnvFactory.getEnv().cleanAfterClass();
+
+    ConfigFactory.getConfig()
+        .setConfigNodeConsesusProtocolClass(originalConfigNodeConsensusProtocolClass);
+    ConfigFactory.getConfig()
+        .setSchemaRegionConsensusProtocolClass(originalSchemaRegionConsensusProtocolClass);
+    ConfigFactory.getConfig()
+        .setDataRegionConsensusProtocolClass(originalDataRegionConsensusProtocolClass);
+
+    ConfigFactory.getConfig().setSchemaReplicationFactor(originalSchemaReplicationFactor);
+    ConfigFactory.getConfig().setDataReplicationFactor(originalDataReplicationFactor);
+
+    ConfigFactory.getConfig().setTimePartitionInterval(originalTimePartitionInterval);
+  }
+
+  @Test
+  public void testRemovingDataNode()
+      throws IOException, InterruptedException, TException, IllegalPathException {
+    try (SyncConfigNodeIServiceClient client =
+        (SyncConfigNodeIServiceClient) EnvFactory.getEnv().getLeaderConfigNodeConnection()) {
+
+      /* Test getOrCreateSchemaPartition, ConfigNode should create SchemaPartition and return */
+      TSchemaPartitionReq schemaPartitionReq =
+          new TSchemaPartitionReq()
+              .setPathPatternTree(ConfigNodeTestUtils.generatePatternTreeBuffer(new String[] {d0}));
+      TSchemaPartitionTableResp schemaPartitionTableResp =
+          client.getOrCreateSchemaPartitionTable(schemaPartitionReq);
+      Assert.assertEquals(
+          TSStatusCode.SUCCESS_STATUS.getStatusCode(),
+          schemaPartitionTableResp.getStatus().getCode());
+      Map<String, Map<TSeriesPartitionSlot, TConsensusGroupId>> schemaPartitionTable =
+          schemaPartitionTableResp.getSchemaPartitionTable();
+      // Successfully create a SchemaPartition
+      Assert.assertTrue(schemaPartitionTable.containsKey(sg));
+      Assert.assertEquals(1, schemaPartitionTable.get(sg).size());
+
+      /* Check Region distribution */
+      TShowRegionResp showRegionResp = client.showRegion(new TShowRegionReq());
+      Assert.assertEquals(
+          TSStatusCode.SUCCESS_STATUS.getStatusCode(), showRegionResp.getStatus().getCode());
+      // Create exactly one RegionGroup
+      Assert.assertEquals(3, showRegionResp.getRegionInfoListSize());
+      // Each DataNode has exactly one Region
+      Set<Integer> dataNodeIdSet = new HashSet<>();
+      showRegionResp
+          .getRegionInfoList()
+          .forEach(regionInfo -> dataNodeIdSet.add(regionInfo.getDataNodeId()));
+      Assert.assertEquals(3, dataNodeIdSet.size());
+
+      /* Change the NodeStatus of the test DataNode to Removing */
+      TSetDataNodeStatusReq setDataNodeStatusReq = new TSetDataNodeStatusReq();
+      DataNodeWrapper dataNodeWrapper = EnvFactory.getEnv().getDataNodeWrapper(testDataNodeId);
+      setDataNodeStatusReq.setTargetDataNode(
+          new TDataNodeLocation(defaultDataNode)
+              .setInternalEndPoint(
+                  new TEndPoint()
+                      .setIp(dataNodeWrapper.getInternalAddress())
+                      .setPort(dataNodeWrapper.getInternalPort())));
+      setDataNodeStatusReq.setStatus(NodeStatus.Removing.getStatus());
+      client.setDataNodeStatus(setDataNodeStatusReq);
+      // Waiting for heartbeat update
+      while (true) {
+        AtomicBoolean containRemoving = new AtomicBoolean(false);
+        TShowDataNodesResp showDataNodesResp = client.showDataNodes();
+        showDataNodesResp
+            .getDataNodesInfoList()
+            .forEach(
+                dataNodeInfo -> {
+                  if (NodeStatus.Removing.getStatus().equals(dataNodeInfo.getStatus())) {
+                    containRemoving.set(true);
+                  }
+                });
+
+        if (containRemoving.get()) {
+          break;
+        }
+        TimeUnit.SECONDS.sleep(1);
+      }
+
+      /* Test getOrCreateSchemaPartition, the result should be NO_ENOUGH_DATANODE */

Review Comment:
   Fixed



-- 
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: reviews-unsubscribe@iotdb.apache.org

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


[GitHub] [iotdb] Caideyipi commented on a diff in pull request #8133: [IOTDB-4334] Verify durable cases of RegionGroup extension

Posted by GitBox <gi...@apache.org>.
Caideyipi commented on code in PR #8133:
URL: https://github.com/apache/iotdb/pull/8133#discussion_r1031951272


##########
integration-test/src/main/java/org/apache/iotdb/itbase/env/BaseEnv.java:
##########
@@ -189,12 +189,22 @@ default SessionPool getSessionPool(
   /** @return The index of ConfigNode-Leader in configNodeWrapperList */
   int getLeaderConfigNodeIndex() throws IOException, InterruptedException;
 
+  /** Start an existed ConfigNode */
   void startConfigNode(int index);
 
+  /** Shutdown an existed ConfigNode */
   void shutdownConfigNode(int index);
 
+  /** @return The TDataNodeLocation of the specified DataNode */
+  DataNodeWrapper getDataNodeWrapper(int index);
+
+  /** Register a new DataNode */
+  void registerNewDataNode();
+
+  /** Start an existed DataNode */
   void startDataNode(int index);
 
+  /** Start an existed DataNode */

Review Comment:
   ```suggestion
     /** Shutdown an existed DataNode */
   ```



-- 
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: reviews-unsubscribe@iotdb.apache.org

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


[GitHub] [iotdb] Caideyipi commented on a diff in pull request #8133: [IOTDB-4334] Verify durable cases of RegionGroup extension

Posted by GitBox <gi...@apache.org>.
Caideyipi commented on code in PR #8133:
URL: https://github.com/apache/iotdb/pull/8133#discussion_r1031974032


##########
integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionDurableIT.java:
##########
@@ -0,0 +1,626 @@
+/*
+ * 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.iotdb.confignode.it.partition;
+
+import org.apache.iotdb.common.rpc.thrift.TConsensusGroupId;
+import org.apache.iotdb.common.rpc.thrift.TDataNodeLocation;
+import org.apache.iotdb.common.rpc.thrift.TEndPoint;
+import org.apache.iotdb.common.rpc.thrift.TSStatus;
+import org.apache.iotdb.common.rpc.thrift.TSeriesPartitionSlot;
+import org.apache.iotdb.common.rpc.thrift.TTimePartitionSlot;
+import org.apache.iotdb.commons.client.sync.SyncConfigNodeIServiceClient;
+import org.apache.iotdb.commons.cluster.NodeStatus;
+import org.apache.iotdb.commons.cluster.RegionStatus;
+import org.apache.iotdb.commons.exception.IllegalPathException;
+import org.apache.iotdb.confignode.it.utils.ConfigNodeTestUtils;
+import org.apache.iotdb.confignode.rpc.thrift.TDataNodeInfo;
+import org.apache.iotdb.confignode.rpc.thrift.TDataPartitionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TDataPartitionTableResp;
+import org.apache.iotdb.confignode.rpc.thrift.TRegionInfo;
+import org.apache.iotdb.confignode.rpc.thrift.TSchemaPartitionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TSchemaPartitionTableResp;
+import org.apache.iotdb.confignode.rpc.thrift.TSetDataNodeStatusReq;
+import org.apache.iotdb.confignode.rpc.thrift.TSetStorageGroupReq;
+import org.apache.iotdb.confignode.rpc.thrift.TShowClusterResp;
+import org.apache.iotdb.confignode.rpc.thrift.TShowDataNodesResp;
+import org.apache.iotdb.confignode.rpc.thrift.TShowRegionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TShowRegionResp;
+import org.apache.iotdb.confignode.rpc.thrift.TStorageGroupSchema;
+import org.apache.iotdb.consensus.ConsensusFactory;
+import org.apache.iotdb.it.env.ConfigFactory;
+import org.apache.iotdb.it.env.DataNodeWrapper;
+import org.apache.iotdb.it.env.EnvFactory;
+import org.apache.iotdb.it.framework.IoTDBTestRunner;
+import org.apache.iotdb.itbase.category.ClusterIT;
+import org.apache.iotdb.rpc.TSStatusCode;
+
+import org.apache.thrift.TException;
+import org.junit.After;
+import org.junit.Assert;
+import org.junit.Before;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+import org.junit.runner.RunWith;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.IOException;
+import java.util.HashSet;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicBoolean;
+import java.util.concurrent.atomic.AtomicInteger;
+
+@RunWith(IoTDBTestRunner.class)
+@Category({ClusterIT.class})
+public class IoTDBPartitionDurableIT {
+
+  private static final Logger LOGGER = LoggerFactory.getLogger(IoTDBPartitionDurableIT.class);
+
+  private static String originalConfigNodeConsensusProtocolClass;
+  private static String originalSchemaRegionConsensusProtocolClass;
+  private static String originalDataRegionConsensusProtocolClass;
+  private static final String testConsensusProtocolClass = ConsensusFactory.RATIS_CONSENSUS;
+
+  private static int originalSchemaReplicationFactor;
+  private static int originalDataReplicationFactor;
+  private static final int testReplicationFactor = 3;
+
+  private static long originalTimePartitionInterval;
+  private static final long testTimePartitionInterval = 604800000;
+
+  private static final int testDataNodeId = 0;
+  private static final String sg = "root.sg";
+  final String d0 = sg + ".d0.s";
+  final String d1 = sg + ".d1.s";
+  private static final int testSeriesPartitionBatchSize = 1;
+  private static final int testTimePartitionBatchSize = 1;
+  private static final TEndPoint defaultEndPoint = new TEndPoint("-1", -1);
+  private static final TDataNodeLocation defaultDataNode =
+      new TDataNodeLocation(
+          -1,
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint),
+          new TEndPoint(defaultEndPoint));
+
+  @Before
+  public void setUp() throws Exception {
+    originalConfigNodeConsensusProtocolClass =
+        ConfigFactory.getConfig().getConfigNodeConsesusProtocolClass();
+    originalSchemaRegionConsensusProtocolClass =
+        ConfigFactory.getConfig().getSchemaRegionConsensusProtocolClass();
+    originalDataRegionConsensusProtocolClass =
+        ConfigFactory.getConfig().getDataRegionConsensusProtocolClass();
+    ConfigFactory.getConfig().setConfigNodeConsesusProtocolClass(testConsensusProtocolClass);
+    ConfigFactory.getConfig().setSchemaRegionConsensusProtocolClass(testConsensusProtocolClass);
+    ConfigFactory.getConfig().setDataRegionConsensusProtocolClass(testConsensusProtocolClass);
+
+    originalSchemaReplicationFactor = ConfigFactory.getConfig().getSchemaReplicationFactor();
+    originalDataReplicationFactor = ConfigFactory.getConfig().getDataReplicationFactor();
+    ConfigFactory.getConfig().setSchemaReplicationFactor(testReplicationFactor);
+    ConfigFactory.getConfig().setDataReplicationFactor(testReplicationFactor);
+
+    originalTimePartitionInterval = ConfigFactory.getConfig().getTimePartitionInterval();
+    ConfigFactory.getConfig().setTimePartitionInterval(testTimePartitionInterval);
+
+    // Init 1C3D environment
+    EnvFactory.getEnv().initClusterEnvironment(1, 3);
+
+    setStorageGroup();
+  }
+
+  private void setStorageGroup() throws IOException, InterruptedException, TException {
+    try (SyncConfigNodeIServiceClient client =
+        (SyncConfigNodeIServiceClient) EnvFactory.getEnv().getLeaderConfigNodeConnection()) {
+      TSetStorageGroupReq setStorageGroupReq = new TSetStorageGroupReq(new TStorageGroupSchema(sg));
+      TSStatus status = client.setStorageGroup(setStorageGroupReq);
+      Assert.assertEquals(TSStatusCode.SUCCESS_STATUS.getStatusCode(), status.getCode());
+    }
+  }
+
+  @After
+  public void tearDown() {
+    EnvFactory.getEnv().cleanAfterClass();
+
+    ConfigFactory.getConfig()
+        .setConfigNodeConsesusProtocolClass(originalConfigNodeConsensusProtocolClass);
+    ConfigFactory.getConfig()
+        .setSchemaRegionConsensusProtocolClass(originalSchemaRegionConsensusProtocolClass);
+    ConfigFactory.getConfig()
+        .setDataRegionConsensusProtocolClass(originalDataRegionConsensusProtocolClass);
+
+    ConfigFactory.getConfig().setSchemaReplicationFactor(originalSchemaReplicationFactor);
+    ConfigFactory.getConfig().setDataReplicationFactor(originalDataReplicationFactor);
+
+    ConfigFactory.getConfig().setTimePartitionInterval(originalTimePartitionInterval);
+  }
+
+  @Test
+  public void testRemovingDataNode()
+      throws IOException, InterruptedException, TException, IllegalPathException {
+    try (SyncConfigNodeIServiceClient client =
+        (SyncConfigNodeIServiceClient) EnvFactory.getEnv().getLeaderConfigNodeConnection()) {
+
+      /* Test getOrCreateSchemaPartition, ConfigNode should create SchemaPartition and return */
+      TSchemaPartitionReq schemaPartitionReq =
+          new TSchemaPartitionReq()
+              .setPathPatternTree(ConfigNodeTestUtils.generatePatternTreeBuffer(new String[] {d0}));
+      TSchemaPartitionTableResp schemaPartitionTableResp =
+          client.getOrCreateSchemaPartitionTable(schemaPartitionReq);
+      Assert.assertEquals(
+          TSStatusCode.SUCCESS_STATUS.getStatusCode(),
+          schemaPartitionTableResp.getStatus().getCode());
+      Map<String, Map<TSeriesPartitionSlot, TConsensusGroupId>> schemaPartitionTable =
+          schemaPartitionTableResp.getSchemaPartitionTable();
+      // Successfully create a SchemaPartition
+      Assert.assertTrue(schemaPartitionTable.containsKey(sg));
+      Assert.assertEquals(1, schemaPartitionTable.get(sg).size());
+
+      /* Check Region distribution */
+      TShowRegionResp showRegionResp = client.showRegion(new TShowRegionReq());
+      Assert.assertEquals(
+          TSStatusCode.SUCCESS_STATUS.getStatusCode(), showRegionResp.getStatus().getCode());
+      // Create exactly one RegionGroup
+      Assert.assertEquals(3, showRegionResp.getRegionInfoListSize());
+      // Each DataNode has exactly one Region
+      Set<Integer> dataNodeIdSet = new HashSet<>();
+      showRegionResp
+          .getRegionInfoList()
+          .forEach(regionInfo -> dataNodeIdSet.add(regionInfo.getDataNodeId()));
+      Assert.assertEquals(3, dataNodeIdSet.size());
+
+      /* Change the NodeStatus of the test DataNode to Removing */
+      TSetDataNodeStatusReq setDataNodeStatusReq = new TSetDataNodeStatusReq();
+      DataNodeWrapper dataNodeWrapper = EnvFactory.getEnv().getDataNodeWrapper(testDataNodeId);
+      setDataNodeStatusReq.setTargetDataNode(
+          new TDataNodeLocation(defaultDataNode)
+              .setInternalEndPoint(
+                  new TEndPoint()
+                      .setIp(dataNodeWrapper.getInternalAddress())
+                      .setPort(dataNodeWrapper.getInternalPort())));
+      setDataNodeStatusReq.setStatus(NodeStatus.Removing.getStatus());
+      client.setDataNodeStatus(setDataNodeStatusReq);
+      // Waiting for heartbeat update
+      while (true) {
+        AtomicBoolean containRemoving = new AtomicBoolean(false);
+        TShowDataNodesResp showDataNodesResp = client.showDataNodes();
+        showDataNodesResp
+            .getDataNodesInfoList()
+            .forEach(
+                dataNodeInfo -> {
+                  if (NodeStatus.Removing.getStatus().equals(dataNodeInfo.getStatus())) {
+                    containRemoving.set(true);
+                  }
+                });
+
+        if (containRemoving.get()) {
+          break;
+        }
+        TimeUnit.SECONDS.sleep(1);
+      }
+
+      /* Test getOrCreateSchemaPartition, the result should be NO_ENOUGH_DATANODE */
+      schemaPartitionReq =
+          new TSchemaPartitionReq()
+              .setPathPatternTree(ConfigNodeTestUtils.generatePatternTreeBuffer(new String[] {d1}));
+      schemaPartitionTableResp = client.getOrCreateSchemaPartitionTable(schemaPartitionReq);
+      Assert.assertEquals(
+          TSStatusCode.NO_AVAILABLE_REGION_GROUP.getStatusCode(),

Review Comment:
   ```suggestion
             TSStatusCode.NO_ENOUGH_DATANODE.getStatusCode(),
   ```



##########
integration-test/src/main/java/org/apache/iotdb/itbase/env/BaseEnv.java:
##########
@@ -189,12 +189,22 @@ default SessionPool getSessionPool(
   /** @return The index of ConfigNode-Leader in configNodeWrapperList */
   int getLeaderConfigNodeIndex() throws IOException, InterruptedException;
 
+  /** Start an existed ConfigNode */
   void startConfigNode(int index);
 
+  /** Shutdown an existed ConfigNode */
   void shutdownConfigNode(int index);
 
+  /** @return The TDataNodeLocation of the specified DataNode */
+  DataNodeWrapper getDataNodeWrapper(int index);
+
+  /** Register a new DataNode */
+  void registerNewDataNode();
+
+  /** Start an existed DataNode */
   void startDataNode(int index);
 
+  /** Start an existed DataNode */

Review Comment:
   ```suggestion
     /** Shutdown an existed DataNode */
   ```



-- 
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: reviews-unsubscribe@iotdb.apache.org

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