You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@iotdb.apache.org by yo...@apache.org on 2023/02/02 06:56:23 UTC
[iotdb] branch master updated: [IOTDB-5445] Support cluster Database heterogeneous through database definition (#8942)
This is an automated email from the ASF dual-hosted git repository.
yongzao pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/iotdb.git
The following commit(s) were added to refs/heads/master by this push:
new a0c7c0bbe7 [IOTDB-5445] Support cluster Database heterogeneous through database definition (#8942)
a0c7c0bbe7 is described below
commit a0c7c0bbe74594678be0b81f4b9a4ce69048d855
Author: YongzaoDan <33...@users.noreply.github.com>
AuthorDate: Thu Feb 2 14:56:16 2023 +0800
[IOTDB-5445] Support cluster Database heterogeneous through database definition (#8942)
---
.../org/apache/iotdb/db/qp/sql/IoTDBSqlParser.g4 | 4 +-
.../antlr4/org/apache/iotdb/db/qp/sql/SqlLexer.g4 | 8 ++
.../iotdb/confignode/conf/ConfigNodeConfig.java | 72 ++++--------
.../confignode/conf/ConfigNodeDescriptor.java | 38 +++---
.../confignode/conf/ConfigNodeStartupCheck.java | 9 +-
...eption.java => DatabaseNotExistsException.java} | 6 +-
.../confignode/manager/ClusterSchemaManager.java | 129 ++++++++++-----------
.../iotdb/confignode/manager/ConfigManager.java | 13 +--
.../iotdb/confignode/manager/load/LoadManager.java | 6 +-
.../manager/load/balancer/RegionBalancer.java | 6 +-
.../manager/partition/PartitionManager.java | 62 +++++-----
.../persistence/executor/ConfigPlanExecutor.java | 2 +-
.../persistence/partition/PartitionInfo.java | 8 +-
.../persistence/schema/ClusterSchemaInfo.java | 50 ++++++--
.../procedure/env/ConfigNodeProcedureEnv.java | 11 +-
.../statemachine/CreateRegionGroupsProcedure.java | 4 +-
.../thrift/ConfigNodeRPCServiceProcessor.java | 67 +++++++++--
.../persistence/ClusterSchemaInfoTest.java | 3 +-
docs/UserGuide/Reference/Common-Config-Manual.md | 47 +++++---
.../zh/UserGuide/Reference/Common-Config-Manual.md | 81 +++++++------
.../iotdb/it/env/cluster/MppCommonConfig.java | 17 ++-
.../it/env/cluster/MppSharedCommonConfig.java | 19 +--
.../iotdb/it/env/remote/RemoteCommonConfig.java | 9 +-
.../org/apache/iotdb/itbase/env/CommonConfig.java | 6 +-
.../it/cluster/IoTDBClusterNodeGetterIT.java | 3 -
.../partition/IoTDBAutoRegionGroupExtensionIT.java | 78 +++++++++----
.../IoTDBCustomRegionGroupExtensionIT.java | 4 +-
.../it/partition/IoTDBPartitionCreationIT.java | 4 +-
.../it/partition/IoTDBPartitionGetterIT.java | 6 +-
.../confignode/it/utils/ConfigNodeTestUtils.java | 1 -
.../scalar/IoTDBDiffFunction2IT.java | 2 +-
.../scalar/IoTDBDiffFunction3IT.java | 2 +-
.../iotdb/zeppelin/it/IoTDBInterpreterIT.java | 12 +-
.../resources/conf/iotdb-common.properties | 44 ++++---
.../db/mpp/common/header/ColumnHeaderConstant.java | 33 ++++--
.../db/mpp/common/header/DatasetHeaderFactory.java | 6 +-
.../iotdb/db/mpp/plan/analyze/AnalyzeVisitor.java | 3 +-
.../config/executor/ClusterConfigTaskExecutor.java | 3 +-
.../config/metadata/SetStorageGroupTask.java | 7 ++
.../config/metadata/ShowStorageGroupTask.java | 42 -------
.../config/metadata/ShowVariablesTask.java | 4 -
.../iotdb/db/mpp/plan/parser/ASTVisitor.java | 25 ++--
.../metadata/SetStorageGroupStatement.java | 72 ++++++++----
.../metadata/ShowStorageGroupStatement.java | 66 +++++++++++
.../java/org/apache/iotdb/rpc/TSStatusCode.java | 1 +
.../src/main/thrift/confignode.thrift | 19 +--
46 files changed, 633 insertions(+), 481 deletions(-)
diff --git a/antlr/src/main/antlr4/org/apache/iotdb/db/qp/sql/IoTDBSqlParser.g4 b/antlr/src/main/antlr4/org/apache/iotdb/db/qp/sql/IoTDBSqlParser.g4
index 0d6a2cc46f..cc8807580f 100644
--- a/antlr/src/main/antlr4/org/apache/iotdb/db/qp/sql/IoTDBSqlParser.g4
+++ b/antlr/src/main/antlr4/org/apache/iotdb/db/qp/sql/IoTDBSqlParser.g4
@@ -83,7 +83,7 @@ storageGroupAttributesClause
;
storageGroupAttributeClause
- : (TTL | SCHEMA_REPLICATION_FACTOR | DATA_REPLICATION_FACTOR | TIME_PARTITION_INTERVAL) '=' INTEGER_LITERAL
+ : (TTL | SCHEMA_REPLICATION_FACTOR | DATA_REPLICATION_FACTOR | TIME_PARTITION_INTERVAL | SCHEMA_REGION_GROUP_NUM | DATA_REGION_GROUP_NUM) '=' INTEGER_LITERAL
;
// Create Timeseries
@@ -291,7 +291,7 @@ stopTrigger
// Show Storage Group
showStorageGroup
- : SHOW (STORAGE GROUP | DATABASES) prefixPath?
+ : SHOW (STORAGE GROUP | DATABASES) DETAILS? prefixPath?
;
// Show Devices
diff --git a/antlr/src/main/antlr4/org/apache/iotdb/db/qp/sql/SqlLexer.g4 b/antlr/src/main/antlr4/org/apache/iotdb/db/qp/sql/SqlLexer.g4
index a407eaf5f4..e30801000d 100644
--- a/antlr/src/main/antlr4/org/apache/iotdb/db/qp/sql/SqlLexer.g4
+++ b/antlr/src/main/antlr4/org/apache/iotdb/db/qp/sql/SqlLexer.g4
@@ -892,6 +892,14 @@ TIME_PARTITION_INTERVAL
: T I M E '_' P A R T I T I O N '_' I N T E R V A L
;
+SCHEMA_REGION_GROUP_NUM
+ : S C H E M A '_' R E G I O N '_' G R O U P '_' N U M
+ ;
+
+DATA_REGION_GROUP_NUM
+ : D A T A '_' R E G I O N '_' G R O U P '_' N U M
+ ;
+
APPLY_TEMPLATE
: A P P L Y '_' T E M P L A T E
;
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeConfig.java b/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeConfig.java
index a46e4ba648..9ce488a649 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeConfig.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeConfig.java
@@ -75,35 +75,34 @@ public class ConfigNodeConfig {
private String seriesPartitionExecutorClass =
"org.apache.iotdb.commons.partition.executor.hash.BKDRHashExecutor";
- /** The maximum number of SchemaRegions expected to be managed by each DataNode. */
- private double schemaRegionPerDataNode = schemaReplicationFactor;
-
/** The policy of extension SchemaRegionGroup for each Database. */
private RegionGroupExtensionPolicy schemaRegionGroupExtensionPolicy =
RegionGroupExtensionPolicy.AUTO;
- /** The number of SchemaRegionGroups for each Database when using CUSTOM extension policy */
- private int schemaRegionGroupPerDatabase = 1;
+ /**
+ * When set schema_region_group_extension_policy=CUSTOM, this parameter is the default number of
+ * SchemaRegionGroups for each Database. When set schema_region_group_extension_policy=AUTO, this
+ * parameter is the default minimal number of SchemaRegionGroups for each Database.
+ */
+ private int defaultSchemaRegionGroupNumPerDatabase = 1;
+
+ /** The maximum number of SchemaRegions expected to be managed by each DataNode. */
+ private double schemaRegionPerDataNode = schemaReplicationFactor;
/** The policy of extension DataRegionGroup for each Database. */
private RegionGroupExtensionPolicy dataRegionGroupExtensionPolicy =
RegionGroupExtensionPolicy.AUTO;
- /** The number of DataRegionGroups for each Database when using CUSTOM extension policy */
- private int dataRegionGroupPerDatabase = 1;
+ /**
+ * When set data_region_group_extension_policy=CUSTOM, this parameter is the default number of
+ * DataRegionGroups for each Database. When set data_region_group_extension_policy=AUTO, this
+ * parameter is the default minimal number of DataRegionGroups for each Database.
+ */
+ private int defaultDataRegionGroupNumPerDatabase = 2;
/** The maximum number of DataRegions expected to be managed by each DataNode. */
private double dataRegionPerProcessor = 1.0;
- /** The least number of SchemaRegionGroup for each Database. */
- private volatile int leastSchemaRegionGroupNum = 1;
-
- /** The least number of DataRegionGroup for each Database. */
- private volatile int leastDataRegionGroupNum = 5;
-
- /** Indicate whether the leastDataRegionGroupNum is set by user, if true, lock it. */
- private volatile boolean leastDataRegionGroupNumSetByUser = false;
-
/** RegionGroup allocate policy. */
private RegionBalancer.RegionGroupAllocatePolicy regionGroupAllocatePolicy =
RegionBalancer.RegionGroupAllocatePolicy.GREEDY;
@@ -491,12 +490,13 @@ public class ConfigNodeConfig {
this.schemaRegionGroupExtensionPolicy = schemaRegionGroupExtensionPolicy;
}
- public int getSchemaRegionGroupPerDatabase() {
- return schemaRegionGroupPerDatabase;
+ public int getDefaultSchemaRegionGroupNumPerDatabase() {
+ return defaultSchemaRegionGroupNumPerDatabase;
}
- public void setSchemaRegionGroupPerDatabase(int schemaRegionGroupPerDatabase) {
- this.schemaRegionGroupPerDatabase = schemaRegionGroupPerDatabase;
+ public void setDefaultSchemaRegionGroupNumPerDatabase(
+ int defaultSchemaRegionGroupNumPerDatabase) {
+ this.defaultSchemaRegionGroupNumPerDatabase = defaultSchemaRegionGroupNumPerDatabase;
}
public RegionGroupExtensionPolicy getDataRegionGroupExtensionPolicy() {
@@ -508,12 +508,12 @@ public class ConfigNodeConfig {
this.dataRegionGroupExtensionPolicy = dataRegionGroupExtensionPolicy;
}
- public int getDataRegionGroupPerDatabase() {
- return dataRegionGroupPerDatabase;
+ public int getDefaultDataRegionGroupNumPerDatabase() {
+ return defaultDataRegionGroupNumPerDatabase;
}
- public void setDataRegionGroupPerDatabase(int dataRegionGroupPerDatabase) {
- this.dataRegionGroupPerDatabase = dataRegionGroupPerDatabase;
+ public void setDefaultDataRegionGroupNumPerDatabase(int defaultDataRegionGroupNumPerDatabase) {
+ this.defaultDataRegionGroupNumPerDatabase = defaultDataRegionGroupNumPerDatabase;
}
public double getSchemaRegionPerDataNode() {
@@ -540,30 +540,6 @@ public class ConfigNodeConfig {
this.dataRegionPerProcessor = dataRegionPerProcessor;
}
- public int getLeastSchemaRegionGroupNum() {
- return leastSchemaRegionGroupNum;
- }
-
- public void setLeastSchemaRegionGroupNum(int leastSchemaRegionGroupNum) {
- this.leastSchemaRegionGroupNum = leastSchemaRegionGroupNum;
- }
-
- public int getLeastDataRegionGroupNum() {
- return leastDataRegionGroupNum;
- }
-
- public void setLeastDataRegionGroupNum(int leastDataRegionGroupNum) {
- this.leastDataRegionGroupNum = leastDataRegionGroupNum;
- }
-
- public boolean isLeastDataRegionGroupNumSetByUser() {
- return leastDataRegionGroupNumSetByUser;
- }
-
- public void setLeastDataRegionGroupNumSetByUser(boolean leastDataRegionGroupNumSetByUser) {
- this.leastDataRegionGroupNumSetByUser = leastDataRegionGroupNumSetByUser;
- }
-
public RegionBalancer.RegionGroupAllocatePolicy getRegionGroupAllocatePolicy() {
return regionGroupAllocatePolicy;
}
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeDescriptor.java b/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeDescriptor.java
index 787bc84cbc..857b5a96c5 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeDescriptor.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeDescriptor.java
@@ -215,25 +215,27 @@ public class ConfigNodeDescriptor {
"data_replication_factor", String.valueOf(conf.getDataReplicationFactor()))
.trim()));
- conf.setSchemaRegionPerDataNode(
- Double.parseDouble(
- properties
- .getProperty(
- "schema_region_per_data_node",
- String.valueOf(conf.getSchemaReplicationFactor()))
- .trim()));
-
conf.setSchemaRegionGroupExtensionPolicy(
RegionGroupExtensionPolicy.parse(
properties.getProperty(
"schema_region_group_extension_policy",
conf.getSchemaRegionGroupExtensionPolicy().getPolicy().trim())));
- conf.setSchemaRegionGroupPerDatabase(
+ conf.setDefaultSchemaRegionGroupNumPerDatabase(
Integer.parseInt(
- properties.getProperty(
- "schema_region_group_per_database",
- String.valueOf(conf.getSchemaRegionGroupPerDatabase()).trim())));
+ properties
+ .getProperty(
+ "default_schema_region_group_num_per_database",
+ String.valueOf(conf.getDefaultSchemaRegionGroupNumPerDatabase()))
+ .trim()));
+
+ conf.setSchemaRegionPerDataNode(
+ Double.parseDouble(
+ properties
+ .getProperty(
+ "schema_region_per_data_node",
+ String.valueOf(conf.getSchemaReplicationFactor()))
+ .trim()));
conf.setDataRegionGroupExtensionPolicy(
RegionGroupExtensionPolicy.parse(
@@ -241,11 +243,11 @@ public class ConfigNodeDescriptor {
"data_region_group_extension_policy",
conf.getDataRegionGroupExtensionPolicy().getPolicy().trim())));
- conf.setDataRegionGroupPerDatabase(
+ conf.setDefaultDataRegionGroupNumPerDatabase(
Integer.parseInt(
properties.getProperty(
- "data_region_group_per_database",
- String.valueOf(conf.getDataRegionGroupPerDatabase()).trim())));
+ "default_data_region_group_num_per_database",
+ String.valueOf(conf.getDefaultDataRegionGroupNumPerDatabase()).trim())));
conf.setDataRegionPerProcessor(
Double.parseDouble(
@@ -254,12 +256,6 @@ public class ConfigNodeDescriptor {
"data_region_per_processor", String.valueOf(conf.getDataRegionPerProcessor()))
.trim()));
- if (properties.getProperty("least_data_region_group_num") != null) {
- conf.setLeastDataRegionGroupNum(
- Integer.parseInt(properties.getProperty("least_data_region_group_num")));
- conf.setLeastDataRegionGroupNumSetByUser(true);
- }
-
try {
conf.setRegionAllocateStrategy(
RegionBalancer.RegionGroupAllocatePolicy.valueOf(
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeStartupCheck.java b/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeStartupCheck.java
index b70e20bb9c..b444928dcf 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeStartupCheck.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/conf/ConfigNodeStartupCheck.java
@@ -112,9 +112,12 @@ public class ConfigNodeStartupCheck {
"The ip address of any target_config_node_list couldn't be 0.0.0.0");
}
- // The least DataRegionGroup number should be positive
- if (CONF.getLeastDataRegionGroupNum() <= 0) {
- throw new ConfigurationException("The least_data_region_group_num should be positive");
+ // The least RegionGroupNum should be positive
+ if (CONF.getDefaultSchemaRegionGroupNumPerDatabase() <= 0) {
+ throw new ConfigurationException("The default_schema_region_group_num should be positive");
+ }
+ if (CONF.getDefaultDataRegionGroupNumPerDatabase() <= 0) {
+ throw new ConfigurationException("The default_data_region_group_num should be positive");
}
}
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/exception/StorageGroupNotExistsException.java b/confignode/src/main/java/org/apache/iotdb/confignode/exception/DatabaseNotExistsException.java
similarity index 80%
rename from confignode/src/main/java/org/apache/iotdb/confignode/exception/StorageGroupNotExistsException.java
rename to confignode/src/main/java/org/apache/iotdb/confignode/exception/DatabaseNotExistsException.java
index aa1a198496..38ac9a8ee0 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/exception/StorageGroupNotExistsException.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/exception/DatabaseNotExistsException.java
@@ -18,9 +18,9 @@
*/
package org.apache.iotdb.confignode.exception;
-public class StorageGroupNotExistsException extends ConfigNodeException {
+public class DatabaseNotExistsException extends ConfigNodeException {
- public StorageGroupNotExistsException(String storageGroup) {
- super(String.format("StorageGroup: %s doesn't exist.", storageGroup));
+ public DatabaseNotExistsException(String database) {
+ super(String.format("Database: %s doesn't exist.", database));
}
}
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/manager/ClusterSchemaManager.java b/confignode/src/main/java/org/apache/iotdb/confignode/manager/ClusterSchemaManager.java
index 7e6a7e57e0..bd9217acb4 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/manager/ClusterSchemaManager.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/manager/ClusterSchemaManager.java
@@ -59,7 +59,7 @@ import org.apache.iotdb.confignode.consensus.response.PathInfoResp;
import org.apache.iotdb.confignode.consensus.response.StorageGroupSchemaResp;
import org.apache.iotdb.confignode.consensus.response.TemplateInfoResp;
import org.apache.iotdb.confignode.consensus.response.TemplateSetInfoResp;
-import org.apache.iotdb.confignode.exception.StorageGroupNotExistsException;
+import org.apache.iotdb.confignode.exception.DatabaseNotExistsException;
import org.apache.iotdb.confignode.manager.node.NodeManager;
import org.apache.iotdb.confignode.manager.observer.NodeStatisticsEvent;
import org.apache.iotdb.confignode.manager.partition.PartitionManager;
@@ -194,9 +194,9 @@ public class ClusterSchemaManager {
Map<String, TStorageGroupInfo> infoMap = new ConcurrentHashMap<>();
for (TStorageGroupSchema storageGroupSchema : storageGroupSchemaResp.getSchemaMap().values()) {
- String name = storageGroupSchema.getName();
+ String database = storageGroupSchema.getName();
TStorageGroupInfo storageGroupInfo = new TStorageGroupInfo();
- storageGroupInfo.setName(name);
+ storageGroupInfo.setName(database);
storageGroupInfo.setTTL(storageGroupSchema.getTTL());
storageGroupInfo.setSchemaReplicationFactor(storageGroupSchema.getSchemaReplicationFactor());
storageGroupInfo.setDataReplicationFactor(storageGroupSchema.getDataReplicationFactor());
@@ -204,10 +204,18 @@ public class ClusterSchemaManager {
try {
storageGroupInfo.setSchemaRegionNum(
- getPartitionManager().getRegionGroupCount(name, TConsensusGroupType.SchemaRegion));
+ getPartitionManager().getRegionGroupCount(database, TConsensusGroupType.SchemaRegion));
storageGroupInfo.setDataRegionNum(
- getPartitionManager().getRegionGroupCount(name, TConsensusGroupType.DataRegion));
- } catch (StorageGroupNotExistsException e) {
+ getPartitionManager().getRegionGroupCount(database, TConsensusGroupType.DataRegion));
+ storageGroupInfo.setMinSchemaRegionNum(
+ getMinRegionGroupNum(database, TConsensusGroupType.SchemaRegion));
+ storageGroupInfo.setMaxSchemaRegionNum(
+ getMaxRegionGroupNum(database, TConsensusGroupType.SchemaRegion));
+ storageGroupInfo.setMinDataRegionNum(
+ getMinRegionGroupNum(database, TConsensusGroupType.DataRegion));
+ storageGroupInfo.setMaxDataRegionNum(
+ getMaxRegionGroupNum(database, TConsensusGroupType.DataRegion));
+ } catch (DatabaseNotExistsException e) {
// Return immediately if some StorageGroups doesn't exist
return new TShowStorageGroupResp()
.setStatus(
@@ -215,7 +223,7 @@ public class ClusterSchemaManager {
.setMessage(e.getMessage()));
}
- infoMap.put(name, storageGroupInfo);
+ infoMap.put(database, storageGroupInfo);
}
return new TShowStorageGroupResp().setStorageGroupInfoMap(infoMap).setStatus(StatusUtils.OK);
@@ -316,7 +324,7 @@ public class ClusterSchemaManager {
public synchronized void adjustMaxRegionGroupNum() {
// Get all StorageGroupSchemas
Map<String, TStorageGroupSchema> storageGroupSchemaMap =
- getMatchedStorageGroupSchemasByName(getStorageGroupNames());
+ getMatchedDatabaseSchemasByName(getDatabaseNames());
if (storageGroupSchemaMap.size() == 0) {
// Skip when there are no StorageGroups
return;
@@ -326,25 +334,6 @@ public class ClusterSchemaManager {
int totalCpuCoreNum = getNodeManager().getTotalCpuCoreCount();
int storageGroupNum = storageGroupSchemaMap.size();
- // Adjust least_data_region_group_num
- // TODO: The least_data_region_group_num should be maintained separately by different
- // TODO: StorageGroup
- if (!CONF.isLeastDataRegionGroupNumSetByUser()) {
- int leastDataRegionGroupNum =
- (int)
- Math.ceil(
- (double) totalCpuCoreNum
- / (double) (storageGroupNum * CONF.getDataReplicationFactor()));
- if (leastDataRegionGroupNum < CONF.getLeastDataRegionGroupNum()) {
- // The leastDataRegionGroupNum should be the maximum integer that satisfy:
- // 1 <= leastDataRegionGroupNum <= 5(default)
- CONF.setLeastDataRegionGroupNum(leastDataRegionGroupNum);
- LOGGER.info(
- "[AdjustRegionGroupNum] The least number of DataRegionGroups per Database is adjusted to: {}",
- leastDataRegionGroupNum);
- }
- }
-
AdjustMaxRegionGroupNumPlan adjustMaxRegionGroupNumPlan = new AdjustMaxRegionGroupNumPlan();
for (TStorageGroupSchema storageGroupSchema : storageGroupSchemaMap.values()) {
try {
@@ -357,7 +346,7 @@ public class ClusterSchemaManager {
storageGroupSchema.getName(), TConsensusGroupType.SchemaRegion);
int maxSchemaRegionGroupNum =
calcMaxRegionGroupNum(
- CONF.getLeastSchemaRegionGroupNum(),
+ storageGroupSchema.getMinSchemaRegionGroupNum(),
SCHEMA_REGION_PER_DATA_NODE,
dataNodeNum,
storageGroupNum,
@@ -376,7 +365,7 @@ public class ClusterSchemaManager {
.getRegionGroupCount(storageGroupSchema.getName(), TConsensusGroupType.DataRegion);
int maxDataRegionGroupNum =
calcMaxRegionGroupNum(
- CONF.getLeastDataRegionGroupNum(),
+ storageGroupSchema.getMinDataRegionGroupNum(),
DATA_REGION_PER_PROCESSOR,
totalCpuCoreNum,
storageGroupNum,
@@ -390,7 +379,7 @@ public class ClusterSchemaManager {
adjustMaxRegionGroupNumPlan.putEntry(
storageGroupSchema.getName(),
new Pair<>(maxSchemaRegionGroupNum, maxDataRegionGroupNum));
- } catch (StorageGroupNotExistsException e) {
+ } catch (DatabaseNotExistsException e) {
LOGGER.warn("Adjust maxRegionGroupNum failed because StorageGroup doesn't exist", e);
}
}
@@ -398,15 +387,15 @@ public class ClusterSchemaManager {
}
public static int calcMaxRegionGroupNum(
- int leastRegionGroupNum,
+ int minRegionGroupNum,
double resourceWeight,
int resource,
int storageGroupNum,
int replicationFactor,
int allocatedRegionGroupCount) {
return Math.max(
- // The maxRegionGroupNum should be great or equal to the leastRegionGroupNum
- leastRegionGroupNum,
+ // The maxRegionGroupNum should be great or equal to the minRegionGroupNum
+ minRegionGroupNum,
Math.max(
(int)
// Use Math.ceil here to ensure that the maxRegionGroupNum
@@ -424,63 +413,73 @@ public class ClusterSchemaManager {
// ======================================================
/**
- * Only leader use this interface.
+ * Only leader use this interface. Get all Databases name
*
- * @param storageGroup StorageGroupName
- * @return The specific StorageGroupSchema
- * @throws StorageGroupNotExistsException When the specific StorageGroup doesn't exist
+ * @return List<DatabaseName>, all Databases' name
*/
- public TStorageGroupSchema getStorageGroupSchemaByName(String storageGroup)
- throws StorageGroupNotExistsException {
- return clusterSchemaInfo.getMatchedStorageGroupSchemaByName(storageGroup);
+ public List<String> getDatabaseNames() {
+ return clusterSchemaInfo.getDatabaseNames();
}
/**
- * Only leader use this interface
+ * Only leader use this interface. Get the specified Database's schema
*
- * @param storageGroup StorageGroupName
- * @param consensusGroupType SchemaRegion for SchemaReplicationFactor and DataRegion for
- * DataReplicationFactor
- * @return SchemaReplicationFactor or DataReplicationFactor
- * @throws StorageGroupNotExistsException When the specific StorageGroup doesn't exist
+ * @param database DatabaseName
+ * @return The specific DatabaseSchema
+ * @throws DatabaseNotExistsException When the specific Database doesn't exist
*/
- public int getReplicationFactor(String storageGroup, TConsensusGroupType consensusGroupType)
- throws StorageGroupNotExistsException {
- TStorageGroupSchema storageGroupSchema = getStorageGroupSchemaByName(storageGroup);
- return consensusGroupType == TConsensusGroupType.SchemaRegion
- ? storageGroupSchema.getSchemaReplicationFactor()
- : storageGroupSchema.getDataReplicationFactor();
+ public TStorageGroupSchema getDatabaseSchemaByName(String database)
+ throws DatabaseNotExistsException {
+ return clusterSchemaInfo.getMatchedDatabaseSchemaByName(database);
}
/**
- * Only leader use this interface.
+ * Only leader use this interface. Get the specified Databases' schema
*
- * @param rawPathList List<StorageGroupName>
- * @return the matched StorageGroupSchemas
+ * @param rawPathList List<DatabaseName>
+ * @return the matched DatabaseSchemas
*/
- public Map<String, TStorageGroupSchema> getMatchedStorageGroupSchemasByName(
+ public Map<String, TStorageGroupSchema> getMatchedDatabaseSchemasByName(
List<String> rawPathList) {
- return clusterSchemaInfo.getMatchedStorageGroupSchemasByName(rawPathList);
+ return clusterSchemaInfo.getMatchedDatabaseSchemasByName(rawPathList);
}
/**
- * Only leader use this interface.
+ * Only leader use this interface. Get the replication factor of specified Database
*
- * @return List<StorageGroupName>, all StorageGroups' name
+ * @param database DatabaseName
+ * @param consensusGroupType SchemaRegion or DataRegion
+ * @return SchemaReplicationFactor or DataReplicationFactor
+ * @throws DatabaseNotExistsException When the specific StorageGroup doesn't exist
+ */
+ public int getReplicationFactor(String database, TConsensusGroupType consensusGroupType)
+ throws DatabaseNotExistsException {
+ TStorageGroupSchema storageGroupSchema = getDatabaseSchemaByName(database);
+ return TConsensusGroupType.SchemaRegion.equals(consensusGroupType)
+ ? storageGroupSchema.getSchemaReplicationFactor()
+ : storageGroupSchema.getDataReplicationFactor();
+ }
+
+ /**
+ * Only leader use this interface. Get the maxRegionGroupNum of specified Database.
+ *
+ * @param database DatabaseName
+ * @param consensusGroupType SchemaRegion or DataRegion
+ * @return minSchemaRegionGroupNum or minDataRegionGroupNum
*/
- public List<String> getStorageGroupNames() {
- return clusterSchemaInfo.getStorageGroupNames();
+ public int getMinRegionGroupNum(String database, TConsensusGroupType consensusGroupType) {
+ return clusterSchemaInfo.getMinRegionGroupNum(database, consensusGroupType);
}
/**
- * Only leader use this interface. Get the maxRegionGroupNum of specific StorageGroup.
+ * Only leader use this interface. Get the maxRegionGroupNum of specified Database.
*
- * @param storageGroup StorageGroupName
+ * @param database DatabaseName
* @param consensusGroupType SchemaRegion or DataRegion
* @return maxSchemaRegionGroupNum or maxDataRegionGroupNum
*/
- public int getMaxRegionGroupNum(String storageGroup, TConsensusGroupType consensusGroupType) {
- return clusterSchemaInfo.getMaxRegionGroupNum(storageGroup, consensusGroupType);
+ public int getMaxRegionGroupNum(String database, TConsensusGroupType consensusGroupType) {
+ return clusterSchemaInfo.getMaxRegionGroupNum(database, consensusGroupType);
}
/**
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/manager/ConfigManager.java b/confignode/src/main/java/org/apache/iotdb/confignode/manager/ConfigManager.java
index 47ece8115c..36371078e8 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/manager/ConfigManager.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/manager/ConfigManager.java
@@ -439,7 +439,6 @@ public class ConfigManager implements IManager {
clusterParameters.setSchemaRegionPerDataNode(CONF.getSchemaRegionPerDataNode());
clusterParameters.setDiskSpaceWarningThreshold(COMMON_CONF.getDiskSpaceWarningThreshold());
clusterParameters.setReadConsistencyLevel(CONF.getReadConsistencyLevel());
- clusterParameters.setLeastDataRegionGroupNum(CONF.getLeastDataRegionGroupNum());
return clusterParameters;
}
@@ -526,7 +525,7 @@ public class ConfigManager implements IManager {
if (status.getCode() == TSStatusCode.SUCCESS_STATUS.getStatusCode()) {
// remove wild
Map<String, TStorageGroupSchema> deleteStorageSchemaMap =
- getClusterSchemaManager().getMatchedStorageGroupSchemasByName(deletedPaths);
+ getClusterSchemaManager().getMatchedDatabaseSchemasByName(deletedPaths);
if (deleteStorageSchemaMap.isEmpty()) {
return RpcUtils.getStatus(
TSStatusCode.PATH_NOT_EXIST.getStatusCode(),
@@ -569,7 +568,7 @@ public class ConfigManager implements IManager {
// Build GetSchemaPartitionPlan
Map<String, Set<TSeriesPartitionSlot>> partitionSlotsMap = new HashMap<>();
List<PartialPath> relatedPaths = patternTree.getAllPathPatterns();
- List<String> allStorageGroups = getClusterSchemaManager().getStorageGroupNames();
+ List<String> allStorageGroups = getClusterSchemaManager().getDatabaseNames();
List<PartialPath> allStorageGroupPaths = new ArrayList<>();
for (String storageGroup : allStorageGroups) {
try {
@@ -627,7 +626,7 @@ public class ConfigManager implements IManager {
}
List<String> devicePaths = patternTree.getAllDevicePatterns();
- List<String> storageGroups = getClusterSchemaManager().getStorageGroupNames();
+ List<String> storageGroups = getClusterSchemaManager().getDatabaseNames();
// Build GetOrCreateSchemaPartitionPlan
Map<String, List<TSeriesPartitionSlot>> partitionSlotsMap = new HashMap<>();
@@ -936,10 +935,6 @@ public class ConfigManager implements IManager {
return errorStatus.setMessage(errorPrefix + "disk_space_warning_threshold" + errorSuffix);
}
- if (clusterParameters.getLeastDataRegionGroupNum() != CONF.getLeastDataRegionGroupNum()) {
- return errorStatus.setMessage(errorPrefix + "least_data_region_group_num" + errorSuffix);
- }
-
return null;
}
@@ -1216,7 +1211,7 @@ public class ConfigManager implements IManager {
return noExistSg;
}
for (PartialPath storageGroup : storageGroups) {
- if (!clusterSchemaManager.getStorageGroupNames().contains(storageGroup.toString())) {
+ if (!clusterSchemaManager.getDatabaseNames().contains(storageGroup.toString())) {
noExistSg.add(storageGroup);
}
}
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/manager/load/LoadManager.java b/confignode/src/main/java/org/apache/iotdb/confignode/manager/load/LoadManager.java
index b23d2fa11f..33d9ed4398 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/manager/load/LoadManager.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/manager/load/LoadManager.java
@@ -36,9 +36,9 @@ import org.apache.iotdb.confignode.client.async.handlers.AsyncClientHandler;
import org.apache.iotdb.confignode.conf.ConfigNodeConfig;
import org.apache.iotdb.confignode.conf.ConfigNodeDescriptor;
import org.apache.iotdb.confignode.consensus.request.write.region.CreateRegionGroupsPlan;
+import org.apache.iotdb.confignode.exception.DatabaseNotExistsException;
import org.apache.iotdb.confignode.exception.NoAvailableRegionGroupException;
import org.apache.iotdb.confignode.exception.NotEnoughDataNodeException;
-import org.apache.iotdb.confignode.exception.StorageGroupNotExistsException;
import org.apache.iotdb.confignode.manager.ClusterSchemaManager;
import org.apache.iotdb.confignode.manager.ConsensusManager;
import org.apache.iotdb.confignode.manager.IManager;
@@ -119,11 +119,11 @@ public class LoadManager {
* @param consensusGroupType TConsensusGroupType of RegionGroup to be allocated
* @return CreateRegionGroupsPlan
* @throws NotEnoughDataNodeException If there are not enough DataNodes
- * @throws StorageGroupNotExistsException If some specific StorageGroups don't exist
+ * @throws DatabaseNotExistsException If some specific StorageGroups don't exist
*/
public CreateRegionGroupsPlan allocateRegionGroups(
Map<String, Integer> allotmentMap, TConsensusGroupType consensusGroupType)
- throws NotEnoughDataNodeException, StorageGroupNotExistsException {
+ throws NotEnoughDataNodeException, DatabaseNotExistsException {
return regionBalancer.genRegionGroupsAllocationPlan(allotmentMap, consensusGroupType);
}
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/manager/load/balancer/RegionBalancer.java b/confignode/src/main/java/org/apache/iotdb/confignode/manager/load/balancer/RegionBalancer.java
index 7b95777536..2a33f6bcd7 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/manager/load/balancer/RegionBalancer.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/manager/load/balancer/RegionBalancer.java
@@ -25,8 +25,8 @@ import org.apache.iotdb.common.rpc.thrift.TRegionReplicaSet;
import org.apache.iotdb.commons.cluster.NodeStatus;
import org.apache.iotdb.confignode.conf.ConfigNodeDescriptor;
import org.apache.iotdb.confignode.consensus.request.write.region.CreateRegionGroupsPlan;
+import org.apache.iotdb.confignode.exception.DatabaseNotExistsException;
import org.apache.iotdb.confignode.exception.NotEnoughDataNodeException;
-import org.apache.iotdb.confignode.exception.StorageGroupNotExistsException;
import org.apache.iotdb.confignode.manager.ClusterSchemaManager;
import org.apache.iotdb.confignode.manager.IManager;
import org.apache.iotdb.confignode.manager.load.balancer.region.CopySetRegionGroupAllocator;
@@ -67,11 +67,11 @@ public class RegionBalancer {
* @param consensusGroupType TConsensusGroupType of the new RegionGroups
* @return CreateRegionGroupsPlan
* @throws NotEnoughDataNodeException When the number of DataNodes is not enough for allocation
- * @throws StorageGroupNotExistsException When some StorageGroups don't exist
+ * @throws DatabaseNotExistsException When some StorageGroups don't exist
*/
public CreateRegionGroupsPlan genRegionGroupsAllocationPlan(
Map<String, Integer> allotmentMap, TConsensusGroupType consensusGroupType)
- throws NotEnoughDataNodeException, StorageGroupNotExistsException {
+ throws NotEnoughDataNodeException, DatabaseNotExistsException {
// The new RegionGroups will occupy online DataNodes firstly
List<TDataNodeConfiguration> onlineDataNodes =
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/manager/partition/PartitionManager.java b/confignode/src/main/java/org/apache/iotdb/confignode/manager/partition/PartitionManager.java
index fafb478be7..588b9c99f2 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/manager/partition/PartitionManager.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/manager/partition/PartitionManager.java
@@ -59,9 +59,9 @@ import org.apache.iotdb.confignode.consensus.response.GetTimeSlotListResp;
import org.apache.iotdb.confignode.consensus.response.RegionInfoListResp;
import org.apache.iotdb.confignode.consensus.response.SchemaNodeManagementResp;
import org.apache.iotdb.confignode.consensus.response.SchemaPartitionResp;
+import org.apache.iotdb.confignode.exception.DatabaseNotExistsException;
import org.apache.iotdb.confignode.exception.NoAvailableRegionGroupException;
import org.apache.iotdb.confignode.exception.NotEnoughDataNodeException;
-import org.apache.iotdb.confignode.exception.StorageGroupNotExistsException;
import org.apache.iotdb.confignode.manager.ClusterSchemaManager;
import org.apache.iotdb.confignode.manager.ConsensusManager;
import org.apache.iotdb.confignode.manager.IManager;
@@ -107,11 +107,8 @@ public class PartitionManager {
private static final ConfigNodeConfig CONF = ConfigNodeDescriptor.getInstance().getConf();
private static final RegionGroupExtensionPolicy SCHEMA_REGION_GROUP_EXTENSION_POLICY =
CONF.getSchemaRegionGroupExtensionPolicy();
- private static final int SCHEMA_REGION_GROUP_PER_DATABASE =
- CONF.getSchemaRegionGroupPerDatabase();
private static final RegionGroupExtensionPolicy DATA_REGION_GROUP_EXTENSION_POLICY =
CONF.getDataRegionGroupExtensionPolicy();
- private static final int DATA_REGION_GROUP_PER_DATABASE = CONF.getDataRegionGroupPerDatabase();
private final IManager configManager;
private final PartitionInfo partitionInfo;
@@ -366,7 +363,7 @@ public class PartitionManager {
LOGGER.error(prompt);
result.setCode(TSStatusCode.NO_ENOUGH_DATANODE.getStatusCode());
result.setMessage(prompt);
- } catch (StorageGroupNotExistsException e) {
+ } catch (DatabaseNotExistsException e) {
String prompt = "ConfigNode failed to extend Region because some StorageGroup doesn't exist.";
LOGGER.error(prompt);
result.setCode(TSStatusCode.DATABASE_NOT_EXIST.getStatusCode());
@@ -378,22 +375,21 @@ public class PartitionManager {
private TSStatus customExtendRegionGroupIfNecessary(
Map<String, Integer> unassignedPartitionSlotsCountMap, TConsensusGroupType consensusGroupType)
- throws StorageGroupNotExistsException, NotEnoughDataNodeException {
+ throws DatabaseNotExistsException, NotEnoughDataNodeException {
// Map<StorageGroup, Region allotment>
Map<String, Integer> allotmentMap = new ConcurrentHashMap<>();
for (Map.Entry<String, Integer> entry : unassignedPartitionSlotsCountMap.entrySet()) {
- final String storageGroup = entry.getKey();
- float allocatedRegionGroupCount =
- partitionInfo.getRegionGroupCount(storageGroup, consensusGroupType);
-
- if (allocatedRegionGroupCount == 0) {
- allotmentMap.put(
- storageGroup,
- TConsensusGroupType.SchemaRegion.equals(consensusGroupType)
- ? SCHEMA_REGION_GROUP_PER_DATABASE
- : DATA_REGION_GROUP_PER_DATABASE);
+ final String database = entry.getKey();
+ int minRegionGroupNum =
+ getClusterSchemaManager().getMinRegionGroupNum(database, consensusGroupType);
+ int allocatedRegionGroupCount =
+ partitionInfo.getRegionGroupCount(database, consensusGroupType);
+
+ // Extend RegionGroups until allocatedRegionGroupCount == minRegionGroupNum
+ if (allocatedRegionGroupCount < minRegionGroupNum) {
+ allotmentMap.put(database, minRegionGroupNum - allocatedRegionGroupCount);
}
}
@@ -402,39 +398,37 @@ public class PartitionManager {
private TSStatus autoExtendRegionGroupIfNecessary(
Map<String, Integer> unassignedPartitionSlotsCountMap, TConsensusGroupType consensusGroupType)
- throws NotEnoughDataNodeException, StorageGroupNotExistsException {
+ throws NotEnoughDataNodeException, DatabaseNotExistsException {
// Map<StorageGroup, Region allotment>
Map<String, Integer> allotmentMap = new ConcurrentHashMap<>();
for (Map.Entry<String, Integer> entry : unassignedPartitionSlotsCountMap.entrySet()) {
- final String storageGroup = entry.getKey();
+ final String database = entry.getKey();
final int unassignedPartitionSlotsCount = entry.getValue();
float allocatedRegionGroupCount =
- partitionInfo.getRegionGroupCount(storageGroup, consensusGroupType);
+ partitionInfo.getRegionGroupCount(database, consensusGroupType);
// The slotCount equals to the sum of assigned slot count and unassigned slot count
float slotCount =
- (float) partitionInfo.getAssignedSeriesPartitionSlotsCount(storageGroup)
+ (float) partitionInfo.getAssignedSeriesPartitionSlotsCount(database)
+ unassignedPartitionSlotsCount;
float maxRegionGroupCount =
- getClusterSchemaManager().getMaxRegionGroupNum(storageGroup, consensusGroupType);
+ getClusterSchemaManager().getMaxRegionGroupNum(database, consensusGroupType);
float maxSlotCount = CONF.getSeriesSlotNum();
/* RegionGroup extension is required in the following cases */
// 1. The number of current RegionGroup of the StorageGroup is less than the least number
- int leastRegionGroupNum =
- TConsensusGroupType.SchemaRegion.equals(consensusGroupType)
- ? CONF.getLeastSchemaRegionGroupNum()
- : CONF.getLeastDataRegionGroupNum();
- if (allocatedRegionGroupCount < leastRegionGroupNum) {
+ int minRegionGroupNum =
+ getClusterSchemaManager().getMinRegionGroupNum(database, consensusGroupType);
+ if (allocatedRegionGroupCount < minRegionGroupNum) {
// Let the sum of unassignedPartitionSlotsCount and allocatedRegionGroupCount
// no less than the leastRegionGroupNum
int delta =
(int)
Math.min(
- unassignedPartitionSlotsCount, leastRegionGroupNum - allocatedRegionGroupCount);
- allotmentMap.put(storageGroup, delta);
+ unassignedPartitionSlotsCount, minRegionGroupNum - allocatedRegionGroupCount);
+ allotmentMap.put(database, delta);
continue;
}
@@ -453,15 +447,15 @@ public class PartitionManager {
Math.ceil(
slotCount * maxRegionGroupCount / maxSlotCount
- allocatedRegionGroupCount)));
- allotmentMap.put(storageGroup, delta);
+ allotmentMap.put(database, delta);
continue;
}
// 3. All RegionGroups in the specified StorageGroup are disabled currently
if (allocatedRegionGroupCount
- == filterRegionGroupThroughStatus(storageGroup, RegionGroupStatus.Disabled).size()
+ == filterRegionGroupThroughStatus(database, RegionGroupStatus.Disabled).size()
&& allocatedRegionGroupCount < maxRegionGroupCount) {
- allotmentMap.put(storageGroup, 1);
+ allotmentMap.put(database, 1);
}
}
@@ -470,7 +464,7 @@ public class PartitionManager {
private TSStatus generateAndAllocateRegionGroups(
Map<String, Integer> allotmentMap, TConsensusGroupType consensusGroupType)
- throws NotEnoughDataNodeException, StorageGroupNotExistsException {
+ throws NotEnoughDataNodeException, DatabaseNotExistsException {
if (!allotmentMap.isEmpty()) {
CreateRegionGroupsPlan createRegionGroupsPlan =
getLoadManager().allocateRegionGroups(allotmentMap, consensusGroupType);
@@ -562,10 +556,10 @@ public class PartitionManager {
* @param storageGroup StorageGroupName
* @param type SchemaRegion or DataRegion
* @return Number of Regions currently owned by the specific StorageGroup
- * @throws StorageGroupNotExistsException When the specific StorageGroup doesn't exist
+ * @throws DatabaseNotExistsException When the specific StorageGroup doesn't exist
*/
public int getRegionGroupCount(String storageGroup, TConsensusGroupType type)
- throws StorageGroupNotExistsException {
+ throws DatabaseNotExistsException {
return partitionInfo.getRegionGroupCount(storageGroup, type);
}
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/persistence/executor/ConfigPlanExecutor.java b/confignode/src/main/java/org/apache/iotdb/confignode/persistence/executor/ConfigPlanExecutor.java
index 47a44c754d..2919db18c7 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/persistence/executor/ConfigPlanExecutor.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/persistence/executor/ConfigPlanExecutor.java
@@ -489,7 +489,7 @@ public class ConfigPlanExecutor {
if (showRegionReq != null && showRegionReq.isSetStorageGroups()) {
final List<String> storageGroups = showRegionReq.getStorageGroups();
final List<String> matchedStorageGroups =
- clusterSchemaInfo.getMatchedStorageGroupSchemasByName(storageGroups).values().stream()
+ clusterSchemaInfo.getMatchedDatabaseSchemasByName(storageGroups).values().stream()
.map(TStorageGroupSchema::getName)
.collect(Collectors.toList());
if (!matchedStorageGroups.isEmpty()) {
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/persistence/partition/PartitionInfo.java b/confignode/src/main/java/org/apache/iotdb/confignode/persistence/partition/PartitionInfo.java
index b8015707cb..b8f5a51099 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/persistence/partition/PartitionInfo.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/persistence/partition/PartitionInfo.java
@@ -53,7 +53,7 @@ import org.apache.iotdb.confignode.consensus.response.GetTimeSlotListResp;
import org.apache.iotdb.confignode.consensus.response.RegionInfoListResp;
import org.apache.iotdb.confignode.consensus.response.SchemaNodeManagementResp;
import org.apache.iotdb.confignode.consensus.response.SchemaPartitionResp;
-import org.apache.iotdb.confignode.exception.StorageGroupNotExistsException;
+import org.apache.iotdb.confignode.exception.DatabaseNotExistsException;
import org.apache.iotdb.confignode.persistence.metric.PartitionInfoMetrics;
import org.apache.iotdb.confignode.persistence.partition.maintainer.RegionMaintainTask;
import org.apache.iotdb.confignode.rpc.thrift.TRegionInfo;
@@ -614,12 +614,12 @@ public class PartitionInfo implements SnapshotProcessor {
* @param storageGroup StorageGroupName
* @param type SchemaRegion or DataRegion
* @return Number of Regions currently owned by the specific StorageGroup
- * @throws StorageGroupNotExistsException When the specific StorageGroup doesn't exist
+ * @throws DatabaseNotExistsException When the specific StorageGroup doesn't exist
*/
public int getRegionGroupCount(String storageGroup, TConsensusGroupType type)
- throws StorageGroupNotExistsException {
+ throws DatabaseNotExistsException {
if (!isStorageGroupExisted(storageGroup)) {
- throw new StorageGroupNotExistsException(storageGroup);
+ throw new DatabaseNotExistsException(storageGroup);
}
return storageGroupPartitionTables.get(storageGroup).getRegionGroupCount(type);
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/persistence/schema/ClusterSchemaInfo.java b/confignode/src/main/java/org/apache/iotdb/confignode/persistence/schema/ClusterSchemaInfo.java
index 29c1ed7bb8..e697d9a15c 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/persistence/schema/ClusterSchemaInfo.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/persistence/schema/ClusterSchemaInfo.java
@@ -52,7 +52,7 @@ import org.apache.iotdb.confignode.consensus.response.PathInfoResp;
import org.apache.iotdb.confignode.consensus.response.StorageGroupSchemaResp;
import org.apache.iotdb.confignode.consensus.response.TemplateInfoResp;
import org.apache.iotdb.confignode.consensus.response.TemplateSetInfoResp;
-import org.apache.iotdb.confignode.exception.StorageGroupNotExistsException;
+import org.apache.iotdb.confignode.exception.DatabaseNotExistsException;
import org.apache.iotdb.confignode.rpc.thrift.TStorageGroupSchema;
import org.apache.iotdb.db.metadata.mtree.ConfigMTree;
import org.apache.iotdb.db.metadata.template.Template;
@@ -353,7 +353,7 @@ public class ClusterSchemaInfo implements SnapshotProcessor {
*
* @return List<StorageGroupName>, all storageGroups' name
*/
- public List<String> getStorageGroupNames() {
+ public List<String> getDatabaseNames() {
List<String> storageGroups = new ArrayList<>();
storageGroupReadWriteLock.readLock().lock();
try {
@@ -387,17 +387,17 @@ public class ClusterSchemaInfo implements SnapshotProcessor {
*
* @param storageGroup StorageGroupName
* @return The specific StorageGroupSchema
- * @throws StorageGroupNotExistsException When the specific StorageGroup doesn't exist
+ * @throws DatabaseNotExistsException When the specific StorageGroup doesn't exist
*/
- public TStorageGroupSchema getMatchedStorageGroupSchemaByName(String storageGroup)
- throws StorageGroupNotExistsException {
+ public TStorageGroupSchema getMatchedDatabaseSchemaByName(String storageGroup)
+ throws DatabaseNotExistsException {
storageGroupReadWriteLock.readLock().lock();
try {
return mTree
.getStorageGroupNodeByStorageGroupPath(new PartialPath(storageGroup))
.getStorageGroupSchema();
} catch (MetadataException e) {
- throw new StorageGroupNotExistsException(storageGroup);
+ throw new DatabaseNotExistsException(storageGroup);
} finally {
storageGroupReadWriteLock.readLock().unlock();
}
@@ -409,7 +409,7 @@ public class ClusterSchemaInfo implements SnapshotProcessor {
* @param rawPathList StorageGroups' path patterns or full paths
* @return All StorageGroupSchemas that matches to the specific StorageGroup patterns
*/
- public Map<String, TStorageGroupSchema> getMatchedStorageGroupSchemasByName(
+ public Map<String, TStorageGroupSchema> getMatchedDatabaseSchemasByName(
List<String> rawPathList) {
Map<String, TStorageGroupSchema> schemaMap = new HashMap<>();
storageGroupReadWriteLock.readLock().lock();
@@ -431,16 +431,44 @@ public class ClusterSchemaInfo implements SnapshotProcessor {
}
/**
- * Only leader use this interface. Get the maxRegionGroupNum of specific StorageGroup.
+ * Only leader use this interface. Get the maxRegionGroupNum of specified Database.
*
- * @param storageGroup StorageGroupName
+ * @param database DatabaseName
+ * @param consensusGroupType SchemaRegion or DataRegion
+ * @return maxSchemaRegionGroupNum or maxDataRegionGroupNum
+ */
+ public int getMinRegionGroupNum(String database, TConsensusGroupType consensusGroupType) {
+ storageGroupReadWriteLock.readLock().lock();
+ try {
+ PartialPath path = new PartialPath(database);
+ TStorageGroupSchema storageGroupSchema =
+ mTree.getStorageGroupNodeByStorageGroupPath(path).getStorageGroupSchema();
+ switch (consensusGroupType) {
+ case SchemaRegion:
+ return storageGroupSchema.getMinSchemaRegionGroupNum();
+ case DataRegion:
+ default:
+ return storageGroupSchema.getMinDataRegionGroupNum();
+ }
+ } catch (MetadataException e) {
+ LOGGER.warn(ERROR_NAME, e);
+ return -1;
+ } finally {
+ storageGroupReadWriteLock.readLock().unlock();
+ }
+ }
+
+ /**
+ * Only leader use this interface. Get the maxRegionGroupNum of specified Database.
+ *
+ * @param database DatabaseName
* @param consensusGroupType SchemaRegion or DataRegion
* @return maxSchemaRegionGroupNum or maxDataRegionGroupNum
*/
- public int getMaxRegionGroupNum(String storageGroup, TConsensusGroupType consensusGroupType) {
+ public int getMaxRegionGroupNum(String database, TConsensusGroupType consensusGroupType) {
storageGroupReadWriteLock.readLock().lock();
try {
- PartialPath path = new PartialPath(storageGroup);
+ PartialPath path = new PartialPath(database);
TStorageGroupSchema storageGroupSchema =
mTree.getStorageGroupNodeByStorageGroupPath(path).getStorageGroupSchema();
switch (consensusGroupType) {
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/procedure/env/ConfigNodeProcedureEnv.java b/confignode/src/main/java/org/apache/iotdb/confignode/procedure/env/ConfigNodeProcedureEnv.java
index 4ef8c1d401..642c4ac669 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/procedure/env/ConfigNodeProcedureEnv.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/procedure/env/ConfigNodeProcedureEnv.java
@@ -42,7 +42,7 @@ import org.apache.iotdb.confignode.consensus.request.write.storagegroup.DeleteSt
import org.apache.iotdb.confignode.consensus.request.write.storagegroup.PreDeleteStorageGroupPlan;
import org.apache.iotdb.confignode.exception.AddConsensusGroupException;
import org.apache.iotdb.confignode.exception.AddPeerException;
-import org.apache.iotdb.confignode.exception.StorageGroupNotExistsException;
+import org.apache.iotdb.confignode.exception.DatabaseNotExistsException;
import org.apache.iotdb.confignode.manager.ClusterSchemaManager;
import org.apache.iotdb.confignode.manager.ConfigManager;
import org.apache.iotdb.confignode.manager.ConsensusManager;
@@ -496,9 +496,8 @@ public class ConfigNodeProcedureEnv {
for (String storageGroup : createRegionGroupsPlan.getRegionGroupMap().keySet()) {
try {
ttlMap.put(
- storageGroup,
- getClusterSchemaManager().getStorageGroupSchemaByName(storageGroup).getTTL());
- } catch (StorageGroupNotExistsException e) {
+ storageGroup, getClusterSchemaManager().getDatabaseSchemaByName(storageGroup).getTTL());
+ } catch (DatabaseNotExistsException e) {
// Notice: This line will never reach since we've checked before
LOG.error("StorageGroup doesn't exist", e);
}
@@ -523,8 +522,8 @@ public class ConfigNodeProcedureEnv {
return req;
}
- public long getTTL(String storageGroup) throws StorageGroupNotExistsException {
- return getClusterSchemaManager().getStorageGroupSchemaByName(storageGroup).getTTL();
+ public long getTTL(String storageGroup) throws DatabaseNotExistsException {
+ return getClusterSchemaManager().getDatabaseSchemaByName(storageGroup).getTTL();
}
public void persistRegionGroup(CreateRegionGroupsPlan createRegionGroupsPlan) {
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/procedure/impl/statemachine/CreateRegionGroupsProcedure.java b/confignode/src/main/java/org/apache/iotdb/confignode/procedure/impl/statemachine/CreateRegionGroupsProcedure.java
index 357ac703ed..80f0ab1e8b 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/procedure/impl/statemachine/CreateRegionGroupsProcedure.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/procedure/impl/statemachine/CreateRegionGroupsProcedure.java
@@ -26,7 +26,7 @@ import org.apache.iotdb.commons.utils.TestOnly;
import org.apache.iotdb.commons.utils.ThriftCommonsSerDeUtils;
import org.apache.iotdb.confignode.consensus.request.write.region.CreateRegionGroupsPlan;
import org.apache.iotdb.confignode.consensus.request.write.region.OfferRegionMaintainTasksPlan;
-import org.apache.iotdb.confignode.exception.StorageGroupNotExistsException;
+import org.apache.iotdb.confignode.exception.DatabaseNotExistsException;
import org.apache.iotdb.confignode.persistence.partition.maintainer.RegionCreateTask;
import org.apache.iotdb.confignode.persistence.partition.maintainer.RegionDeleteTask;
import org.apache.iotdb.confignode.procedure.env.ConfigNodeProcedureEnv;
@@ -123,7 +123,7 @@ public class CreateRegionGroupsProcedure
regionReplicaSet.getRegionId().getType())) {
try {
createTask.setTTL(env.getTTL(storageGroup));
- } catch (StorageGroupNotExistsException e) {
+ } catch (DatabaseNotExistsException e) {
LOGGER.error("Can't get TTL", e);
}
}
diff --git a/confignode/src/main/java/org/apache/iotdb/confignode/service/thrift/ConfigNodeRPCServiceProcessor.java b/confignode/src/main/java/org/apache/iotdb/confignode/service/thrift/ConfigNodeRPCServiceProcessor.java
index f3091c0066..08bbfcecca 100644
--- a/confignode/src/main/java/org/apache/iotdb/confignode/service/thrift/ConfigNodeRPCServiceProcessor.java
+++ b/confignode/src/main/java/org/apache/iotdb/confignode/service/thrift/ConfigNodeRPCServiceProcessor.java
@@ -30,6 +30,7 @@ import org.apache.iotdb.commons.path.PartialPath;
import org.apache.iotdb.commons.path.PathPatternTree;
import org.apache.iotdb.commons.utils.StatusUtils;
import org.apache.iotdb.commons.utils.TestOnly;
+import org.apache.iotdb.confignode.conf.ConfigNodeConfig;
import org.apache.iotdb.confignode.conf.ConfigNodeConstant;
import org.apache.iotdb.confignode.conf.ConfigNodeDescriptor;
import org.apache.iotdb.confignode.conf.SystemPropertiesUtils;
@@ -171,6 +172,9 @@ public class ConfigNodeRPCServiceProcessor implements IConfigNodeRPCService.Ifac
private static final Logger LOGGER = LoggerFactory.getLogger(ConfigNodeRPCServiceProcessor.class);
+ private static final ConfigNodeConfig CONFIG_NODE_CONFIG =
+ ConfigNodeDescriptor.getInstance().getConf();
+
private final ConfigManager configManager;
public ConfigNodeRPCServiceProcessor(ConfigManager configManager) {
@@ -266,34 +270,79 @@ public class ConfigNodeRPCServiceProcessor implements IConfigNodeRPCService.Ifac
}
@Override
- public TShowVariablesResp showVariables() throws TException {
+ public TShowVariablesResp showVariables() {
return configManager.showVariables();
}
@Override
public TSStatus setStorageGroup(TSetStorageGroupReq req) throws TException {
TStorageGroupSchema storageGroupSchema = req.getStorageGroup();
+ TSStatus errorResp = null;
// Set default configurations if necessary
if (!storageGroupSchema.isSetTTL()) {
storageGroupSchema.setTTL(CommonDescriptor.getInstance().getConfig().getDefaultTTLInMs());
+ } else if (storageGroupSchema.getTTL() <= 0) {
+ errorResp =
+ new TSStatus(TSStatusCode.DATABASE_CONFIG_ERROR.getStatusCode())
+ .setMessage("Failed to create database. The TTL should be positive.");
}
+
if (!storageGroupSchema.isSetSchemaReplicationFactor()) {
storageGroupSchema.setSchemaReplicationFactor(
- ConfigNodeDescriptor.getInstance().getConf().getSchemaReplicationFactor());
+ CONFIG_NODE_CONFIG.getSchemaReplicationFactor());
+ } else if (storageGroupSchema.getSchemaReplicationFactor() <= 0) {
+ errorResp =
+ new TSStatus(TSStatusCode.DATABASE_CONFIG_ERROR.getStatusCode())
+ .setMessage(
+ "Failed to create database. The schemaReplicationFactor should be positive.");
}
+
if (!storageGroupSchema.isSetDataReplicationFactor()) {
- storageGroupSchema.setDataReplicationFactor(
- ConfigNodeDescriptor.getInstance().getConf().getDataReplicationFactor());
+ storageGroupSchema.setDataReplicationFactor(CONFIG_NODE_CONFIG.getDataReplicationFactor());
+ } else if (storageGroupSchema.getDataReplicationFactor() <= 0) {
+ errorResp =
+ new TSStatus(TSStatusCode.DATABASE_CONFIG_ERROR.getStatusCode())
+ .setMessage(
+ "Failed to create database. The dataReplicationFactor should be positive.");
}
+
if (!storageGroupSchema.isSetTimePartitionInterval()) {
- storageGroupSchema.setTimePartitionInterval(
- ConfigNodeDescriptor.getInstance().getConf().getTimePartitionInterval());
+ storageGroupSchema.setTimePartitionInterval(CONFIG_NODE_CONFIG.getTimePartitionInterval());
+ } else if (storageGroupSchema.getTimePartitionInterval() <= 0) {
+ errorResp =
+ new TSStatus(TSStatusCode.DATABASE_CONFIG_ERROR.getStatusCode())
+ .setMessage(
+ "Failed to create database. The timePartitionInterval should be positive.");
+ }
+
+ if (!storageGroupSchema.isSetMinSchemaRegionGroupNum()) {
+ storageGroupSchema.setMinSchemaRegionGroupNum(
+ CONFIG_NODE_CONFIG.getDefaultSchemaRegionGroupNumPerDatabase());
+ } else if (storageGroupSchema.getMinSchemaRegionGroupNum() <= 0) {
+ errorResp =
+ new TSStatus(TSStatusCode.DATABASE_CONFIG_ERROR.getStatusCode())
+ .setMessage(
+ "Failed to create database. The schemaRegionGroupNum should be positive.");
+ }
+
+ if (!storageGroupSchema.isSetMinDataRegionGroupNum()) {
+ storageGroupSchema.setMinDataRegionGroupNum(
+ CONFIG_NODE_CONFIG.getDefaultDataRegionGroupNumPerDatabase());
+ } else if (storageGroupSchema.getMinDataRegionGroupNum() <= 0) {
+ errorResp =
+ new TSStatus(TSStatusCode.DATABASE_CONFIG_ERROR.getStatusCode())
+ .setMessage("Failed to create database. The dataRegionGroupNum should be positive.");
+ }
+
+ if (errorResp != null) {
+ LOGGER.warn("Execute SetStorageGroupRequest {} with result {}", req, errorResp);
+ return errorResp;
}
- // Initialize the maxSchemaRegionGroupCount and maxDataRegionGroupCount as 0
- storageGroupSchema.setMaxSchemaRegionGroupNum(0);
- storageGroupSchema.setMaxDataRegionGroupNum(0);
+ // The maxRegionGroupNum is equal to the minRegionGroupNum when initialize
+ storageGroupSchema.setMaxSchemaRegionGroupNum(storageGroupSchema.getMinSchemaRegionGroupNum());
+ storageGroupSchema.setMaxDataRegionGroupNum(storageGroupSchema.getMinDataRegionGroupNum());
SetStorageGroupPlan setReq = new SetStorageGroupPlan(storageGroupSchema);
TSStatus resp = configManager.setStorageGroup(setReq);
diff --git a/confignode/src/test/java/org/apache/iotdb/confignode/persistence/ClusterSchemaInfoTest.java b/confignode/src/test/java/org/apache/iotdb/confignode/persistence/ClusterSchemaInfoTest.java
index 03417a5a23..5d5b4923a3 100644
--- a/confignode/src/test/java/org/apache/iotdb/confignode/persistence/ClusterSchemaInfoTest.java
+++ b/confignode/src/test/java/org/apache/iotdb/confignode/persistence/ClusterSchemaInfoTest.java
@@ -97,8 +97,7 @@ public class ClusterSchemaInfoTest {
clusterSchemaInfo.clear();
clusterSchemaInfo.processLoadSnapshot(snapshotDir);
- Assert.assertEquals(
- storageGroupPathList.size(), clusterSchemaInfo.getStorageGroupNames().size());
+ Assert.assertEquals(storageGroupPathList.size(), clusterSchemaInfo.getDatabaseNames().size());
GetStorageGroupPlan getStorageGroupReq =
new GetStorageGroupPlan(Arrays.asList(PathUtils.splitPathToDetachedNodes("root.**")));
diff --git a/docs/UserGuide/Reference/Common-Config-Manual.md b/docs/UserGuide/Reference/Common-Config-Manual.md
index e2c79e093e..b67cd1e92e 100644
--- a/docs/UserGuide/Reference/Common-Config-Manual.md
+++ b/docs/UserGuide/Reference/Common-Config-Manual.md
@@ -102,6 +102,24 @@ Different configuration parameters take effect in the following three ways:
| Default | org.apache.iotdb.commons.partition.executor.hash.BKDRHashExecutor |
| Effective | Only allowed to be modified in first start up |
+* schema\_region\_group\_extension\_policy
+
+| Name | schema\_region\_group\_extension\_policy |
+|:-----------:|:------------------------------------------|
+| Description | The extension policy of SchemaRegionGroup |
+| Type | string |
+| Default | AUTO |
+| Effective | After restarting system |
+
+* default\_schema\_region\_group\_num\_per\_database
+
+| Name | default\_schema\_region\_group\_num\_per\_database |
+|:-----------:|:----------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------|
+| Description | The number of SchemaRegionGroups that each Database has when using the CUSTOM-SchemaRegionGroup extension policy. The least number of SchemaRegionGroups that each Database has when using the AUTO-SchemaRegionGroup extension policy. |
+| Type | int |
+| Default | 1 |
+| Effective | After restarting system |
+
* schema\_region\_per\_data\_node
| Name | schema\_region\_per\_data\_node |
@@ -120,14 +138,14 @@ Different configuration parameters take effect in the following three ways:
| Default | AUTO |
| Effective | After restarting system |
-* data\_region\_group\_per\_database
+* default\_data\_region\_group\_num\_per\_database
-| Name | data\_region\_group\_per\_database |
-|:-----------:|:-------------------------------------------------------------------------------------------------------------|
-| Description | The number of DataRegionGroups that each Database has when using the CUSTOM-DataRegionGroup extension policy |
-| Type | int |
-| Default | 1 |
-| Effective | After restarting system |
+| Name | default\_data\_region\_group\_num\_per\_database |
+|:-----------:|:--------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------|
+| Description | The number of DataRegionGroups that each Database has when using the CUSTOM-DataRegionGroup extension policy. The least number of DataRegionGroups that each Database has when using the AUTO-DataRegionGroup extension policy. |
+| Type | int |
+| Default | 1 |
+| Effective | After restarting system |
* data\_region\_per\_processor
@@ -138,15 +156,6 @@ Different configuration parameters take effect in the following three ways:
| Default | 1.0 |
| Effective | After restarting system |
-* least\_data\_region\_group\_num
-
-| Name | least\_data\_region\_group\_num |
-|:-----------:|:------------------------------------------------------|
-| Description | The least number of DataRegionGroup for each Database |
-| Type | int |
-| Default | 5 |
-| Effective | After restarting system |
-
* enable\_data\_partition\_inherit\_policy
| Name | enable\_data\_partition\_inherit\_policy |
@@ -188,7 +197,7 @@ Different configuration parameters take effect in the following three ways:
* time\_partition\_interval
| Name | time\_partition\_interval |
-| :---------: | :------------------------------------------------------------ |
+|:-----------:|:--------------------------------------------------------------|
| Description | Time partition interval of data when ConfigNode allocate data |
| Type | Long |
| Unit | ms |
@@ -198,7 +207,7 @@ Different configuration parameters take effect in the following three ways:
* heartbeat\_interval\_in\_ms
| Name | heartbeat\_interval\_in\_ms |
-| :---------: | :-------------------------------------- |
+|:-----------:|:----------------------------------------|
| Description | Heartbeat interval in the cluster nodes |
| Type | Long |
| Unit | ms |
@@ -208,7 +217,7 @@ Different configuration parameters take effect in the following three ways:
* disk\_space\_warning\_threshold
| Name | disk\_space\_warning\_threshold |
-| :---------: | :------------------------------ |
+|:-----------:|:--------------------------------|
| Description | Disk remaining threshold |
| Type | double(percentage) |
| Default | 0.05 |
diff --git a/docs/zh/UserGuide/Reference/Common-Config-Manual.md b/docs/zh/UserGuide/Reference/Common-Config-Manual.md
index 0c6c28fcfb..9ce344097d 100644
--- a/docs/zh/UserGuide/Reference/Common-Config-Manual.md
+++ b/docs/zh/UserGuide/Reference/Common-Config-Manual.md
@@ -102,6 +102,24 @@ IoTDB ConfigNode 和 DataNode 的公共配置参数位于 `conf` 目录下。
| 默认值 | org.apache.iotdb.commons.partition.executor.hash.BKDRHashExecutor |
| 改后生效方式 | 仅允许在第一次启动服务前修改 |
+* schema\_region\_group\_extension\_policy
+
+| 名字 | schema\_region\_group\_extension\_policy |
+|:------:|:-----------------------------------------|
+| 描述 | SchemaRegionGroup 的扩容策略 |
+| 类型 | string |
+| 默认值 | AUTO |
+| 改后生效方式 | 重启服务生效 |
+
+* default\_schema\_region\_group\_num\_per\_database
+
+| 名字 | default\_schema\_region\_group\_num\_per\_database |
+|:------:|:--------------------------------------------------------------------------------------------------------------------------------------------------------|
+| 描述 | 当选用 CUSTOM-SchemaRegionGroup 扩容策略时,此参数为每个 Database 拥有的 SchemaRegionGroup 数量;当选用 AUTO-SchemaRegionGroup 扩容策略时,此参数为每个 Database 最少拥有的 SchemaRegionGroup 数量 |
+| 类型 | int |
+| 默认值 | 1 |
+| 改后生效方式 | 重启服务生效 |
+
* schema\_region\_per\_data\_node
| 名字 | schema\_region\_per\_data\_node |
@@ -120,14 +138,14 @@ IoTDB ConfigNode 和 DataNode 的公共配置参数位于 `conf` 目录下。
| 默认值 | AUTO |
| 改后生效方式 | 重启服务生效 |
-* data\_region\_group\_per\_database
+* default\_data\_region\_group\_num\_per\_database
-| 名字 | data\_region\_group\_per\_database |
-|:------:|:--------------------------------------------------------------------|
-| 描述 | 当选用 CUSTOM-DataRegionGroup 扩容策略时,每个 Database 拥有的 DataRegionGroup 数量 |
-| 类型 | int |
-| 默认值 | 1 |
-| 改后生效方式 | 重启服务生效 |
+| 名字 | data\_region\_group\_per\_database |
+|:------:|:------------------------------------------------------------------------------------------------------------------------------------------------|
+| 描述 | 当选用 CUSTOM-DataRegionGroup 扩容策略时,此参数为每个 Database 拥有的 DataRegionGroup 数量;当选用 AUTO-DataRegionGroup 扩容策略时,此参数为每个 Database 最少拥有的 DataRegionGroup 数量 |
+| 类型 | int |
+| 默认值 | 2 |
+| 改后生效方式 | 重启服务生效 |
* data\_region\_per\_processor
@@ -138,15 +156,6 @@ IoTDB ConfigNode 和 DataNode 的公共配置参数位于 `conf` 目录下。
| 默认值 | 1.0 |
| 改后生效方式 | 重启服务生效 |
-* least\_data\_region\_group\_num
-
-| 名字 | least\_data\_region\_group\_num |
-|:------:|:------------------------------------|
-| 描述 | 每个 Database 的 DataRegionGroup 的最少数量 |
-| 类型 | int |
-| 默认值 | 5 |
-| 改后生效方式 | 重启服务生效 |
-
* enable\_data\_partition\_inherit\_policy
| 名字 | enable\_data\_partition\_inherit\_policy |
@@ -187,32 +196,32 @@ IoTDB ConfigNode 和 DataNode 的公共配置参数位于 `conf` 目录下。
* time\_partition\_interval
-| 名字 | time\_partition\_interval |
-| :----------: | :------------------------------ |
-| 描述 | Database 默认的数据时间分区间隔 |
-| 类型 | Long |
-| 单位 | 毫秒 |
-| 默认值 | 604800000 |
-| 改后生效方式 | 仅允许在第一次启动服务前修改 |
+| 名字 | time\_partition\_interval |
+|:------:|:--------------------------|
+| 描述 | Database 默认的数据时间分区间隔 |
+| 类型 | Long |
+| 单位 | 毫秒 |
+| 默认值 | 604800000 |
+| 改后生效方式 | 仅允许在第一次启动服务前修改 |
* heartbeat\_interval\_in\_ms
-| 名字 | heartbeat\_interval\_in\_ms |
-| :----------: | :-------------------------- |
-| 描述 | 集群节点间的心跳间隔 |
-| 类型 | Long |
-| 单位 | ms |
-| 默认值 | 1000 |
-| 改后生效方式 | 重启服务生效 |
+| 名字 | heartbeat\_interval\_in\_ms |
+|:------:|:----------------------------|
+| 描述 | 集群节点间的心跳间隔 |
+| 类型 | Long |
+| 单位 | ms |
+| 默认值 | 1000 |
+| 改后生效方式 | 重启服务生效 |
* disk\_space\_warning\_threshold
-| 名字 | disk\_space\_warning\_threshold |
-| :----------: | :------------------------------ |
-| 描述 | DataNode 磁盘剩余阈值 |
-| 类型 | double(percentage) |
-| 默认值 | 0.05 |
-| 改后生效方式 | 重启服务生效 |
+| 名字 | disk\_space\_warning\_threshold |
+|:------:|:--------------------------------|
+| 描述 | DataNode 磁盘剩余阈值 |
+| 类型 | double(percentage) |
+| 默认值 | 0.05 |
+| 改后生效方式 | 重启服务生效 |
### 内存控制配置
diff --git a/integration-test/src/main/java/org/apache/iotdb/it/env/cluster/MppCommonConfig.java b/integration-test/src/main/java/org/apache/iotdb/it/env/cluster/MppCommonConfig.java
index 092637998f..1457eb72c3 100644
--- a/integration-test/src/main/java/org/apache/iotdb/it/env/cluster/MppCommonConfig.java
+++ b/integration-test/src/main/java/org/apache/iotdb/it/env/cluster/MppCommonConfig.java
@@ -178,8 +178,10 @@ public class MppCommonConfig extends MppBaseConfig implements CommonConfig {
}
@Override
- public CommonConfig setSchemaRegionGroupPerDatabase(int schemaRegionGroupPerDatabase) {
- setProperty("schema_region_group_per_database", String.valueOf(schemaRegionGroupPerDatabase));
+ public CommonConfig setDefaultSchemaRegionGroupNumPerDatabase(int schemaRegionGroupPerDatabase) {
+ setProperty(
+ "default_schema_region_group_num_per_database",
+ String.valueOf(schemaRegionGroupPerDatabase));
return this;
}
@@ -190,8 +192,9 @@ public class MppCommonConfig extends MppBaseConfig implements CommonConfig {
}
@Override
- public CommonConfig setDataRegionGroupPerDatabase(int dataRegionGroupPerDatabase) {
- setProperty("data_region_group_per_database", String.valueOf(dataRegionGroupPerDatabase));
+ public CommonConfig setDefaultDataRegionGroupNumPerDatabase(int dataRegionGroupPerDatabase) {
+ setProperty(
+ "default_data_region_group_num_per_database", String.valueOf(dataRegionGroupPerDatabase));
return this;
}
@@ -305,12 +308,6 @@ public class MppCommonConfig extends MppBaseConfig implements CommonConfig {
return this;
}
- @Override
- public CommonConfig setLeastDataRegionGroupNum(int leastDataRegionGroupNum) {
- setProperty("least_data_region_group_num", String.valueOf(leastDataRegionGroupNum));
- return this;
- }
-
@Override
public CommonConfig setQueryThreadCount(int queryThreadCount) {
if (queryThreadCount <= 0) {
diff --git a/integration-test/src/main/java/org/apache/iotdb/it/env/cluster/MppSharedCommonConfig.java b/integration-test/src/main/java/org/apache/iotdb/it/env/cluster/MppSharedCommonConfig.java
index a176d4ac0f..896441f9b0 100644
--- a/integration-test/src/main/java/org/apache/iotdb/it/env/cluster/MppSharedCommonConfig.java
+++ b/integration-test/src/main/java/org/apache/iotdb/it/env/cluster/MppSharedCommonConfig.java
@@ -194,9 +194,9 @@ public class MppSharedCommonConfig implements CommonConfig {
}
@Override
- public CommonConfig setSchemaRegionGroupPerDatabase(int schemaRegionGroupPerDatabase) {
- cnConfig.setSchemaRegionGroupPerDatabase(schemaRegionGroupPerDatabase);
- dnConfig.setSchemaRegionGroupPerDatabase(schemaRegionGroupPerDatabase);
+ public CommonConfig setDefaultSchemaRegionGroupNumPerDatabase(int schemaRegionGroupPerDatabase) {
+ cnConfig.setDefaultSchemaRegionGroupNumPerDatabase(schemaRegionGroupPerDatabase);
+ dnConfig.setDefaultSchemaRegionGroupNumPerDatabase(schemaRegionGroupPerDatabase);
return this;
}
@@ -208,9 +208,9 @@ public class MppSharedCommonConfig implements CommonConfig {
}
@Override
- public CommonConfig setDataRegionGroupPerDatabase(int dataRegionGroupPerDatabase) {
- cnConfig.setDataRegionGroupPerDatabase(dataRegionGroupPerDatabase);
- dnConfig.setDataRegionGroupPerDatabase(dataRegionGroupPerDatabase);
+ public CommonConfig setDefaultDataRegionGroupNumPerDatabase(int dataRegionGroupPerDatabase) {
+ cnConfig.setDefaultDataRegionGroupNumPerDatabase(dataRegionGroupPerDatabase);
+ dnConfig.setDefaultDataRegionGroupNumPerDatabase(dataRegionGroupPerDatabase);
return this;
}
@@ -316,13 +316,6 @@ public class MppSharedCommonConfig implements CommonConfig {
return this;
}
- @Override
- public CommonConfig setLeastDataRegionGroupNum(int leastDataRegionGroupNum) {
- cnConfig.setLeastDataRegionGroupNum(leastDataRegionGroupNum);
- dnConfig.setLeastDataRegionGroupNum(leastDataRegionGroupNum);
- return this;
- }
-
@Override
public CommonConfig setQueryThreadCount(int queryThreadCount) {
cnConfig.setQueryThreadCount(queryThreadCount);
diff --git a/integration-test/src/main/java/org/apache/iotdb/it/env/remote/RemoteCommonConfig.java b/integration-test/src/main/java/org/apache/iotdb/it/env/remote/RemoteCommonConfig.java
index 29a60f4061..78b973db1b 100644
--- a/integration-test/src/main/java/org/apache/iotdb/it/env/remote/RemoteCommonConfig.java
+++ b/integration-test/src/main/java/org/apache/iotdb/it/env/remote/RemoteCommonConfig.java
@@ -139,7 +139,7 @@ public class RemoteCommonConfig implements CommonConfig {
}
@Override
- public CommonConfig setSchemaRegionGroupPerDatabase(int schemaRegionGroupPerDatabase) {
+ public CommonConfig setDefaultSchemaRegionGroupNumPerDatabase(int schemaRegionGroupPerDatabase) {
return this;
}
@@ -149,7 +149,7 @@ public class RemoteCommonConfig implements CommonConfig {
}
@Override
- public CommonConfig setDataRegionGroupPerDatabase(int dataRegionGroupPerDatabase) {
+ public CommonConfig setDefaultDataRegionGroupNumPerDatabase(int dataRegionGroupPerDatabase) {
return this;
}
@@ -227,11 +227,6 @@ public class RemoteCommonConfig implements CommonConfig {
return this;
}
- @Override
- public CommonConfig setLeastDataRegionGroupNum(int leastDataRegionGroupNum) {
- return this;
- }
-
@Override
public CommonConfig setQueryThreadCount(int queryThreadCount) {
return this;
diff --git a/integration-test/src/main/java/org/apache/iotdb/itbase/env/CommonConfig.java b/integration-test/src/main/java/org/apache/iotdb/itbase/env/CommonConfig.java
index 5ab718e06d..12f322bd66 100644
--- a/integration-test/src/main/java/org/apache/iotdb/itbase/env/CommonConfig.java
+++ b/integration-test/src/main/java/org/apache/iotdb/itbase/env/CommonConfig.java
@@ -67,11 +67,11 @@ public interface CommonConfig {
CommonConfig setSchemaRegionGroupExtensionPolicy(String schemaRegionGroupExtensionPolicy);
- CommonConfig setSchemaRegionGroupPerDatabase(int schemaRegionGroupPerDatabase);
+ CommonConfig setDefaultSchemaRegionGroupNumPerDatabase(int schemaRegionGroupPerDatabase);
CommonConfig setDataRegionGroupExtensionPolicy(String dataRegionGroupExtensionPolicy);
- CommonConfig setDataRegionGroupPerDatabase(int dataRegionGroupPerDatabase);
+ CommonConfig setDefaultDataRegionGroupNumPerDatabase(int dataRegionGroupPerDatabase);
CommonConfig setSchemaReplicationFactor(int schemaReplicationFactor);
@@ -103,8 +103,6 @@ public interface CommonConfig {
CommonConfig setEnableAutoLeaderBalanceForIoTConsensus(
boolean enableAutoLeaderBalanceForIoTConsensus);
- CommonConfig setLeastDataRegionGroupNum(int leastDataRegionGroupNum);
-
CommonConfig setQueryThreadCount(int queryThreadCount);
CommonConfig setDataRatisTriggerSnapshotThreshold(long threshold);
diff --git a/integration-test/src/test/java/org/apache/iotdb/confignode/it/cluster/IoTDBClusterNodeGetterIT.java b/integration-test/src/test/java/org/apache/iotdb/confignode/it/cluster/IoTDBClusterNodeGetterIT.java
index 5568536a5a..1c9b570859 100644
--- a/integration-test/src/test/java/org/apache/iotdb/confignode/it/cluster/IoTDBClusterNodeGetterIT.java
+++ b/integration-test/src/test/java/org/apache/iotdb/confignode/it/cluster/IoTDBClusterNodeGetterIT.java
@@ -175,9 +175,6 @@ public class IoTDBClusterNodeGetterIT {
Assert.assertEquals(
expectedParameters.getReadConsistencyLevel(),
clusterParameters.getReadConsistencyLevel());
- Assert.assertEquals(
- expectedParameters.getLeastDataRegionGroupNum(),
- clusterParameters.getLeastDataRegionGroupNum());
/* Test showConfigNodes */
TShowConfigNodesResp showConfigNodesResp = client.showConfigNodes();
diff --git a/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBAutoRegionGroupExtensionIT.java b/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBAutoRegionGroupExtensionIT.java
index b7e3c88613..1c887ef244 100644
--- a/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBAutoRegionGroupExtensionIT.java
+++ b/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBAutoRegionGroupExtensionIT.java
@@ -18,16 +18,18 @@
*/
package org.apache.iotdb.confignode.it.partition;
+import org.apache.iotdb.common.rpc.thrift.TConsensusGroupType;
import org.apache.iotdb.common.rpc.thrift.TSStatus;
import org.apache.iotdb.common.rpc.thrift.TSeriesPartitionSlot;
import org.apache.iotdb.commons.client.sync.SyncConfigNodeIServiceClient;
+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.TDeleteStorageGroupReq;
+import org.apache.iotdb.confignode.rpc.thrift.TSchemaPartitionReq;
+import org.apache.iotdb.confignode.rpc.thrift.TSchemaPartitionTableResp;
import org.apache.iotdb.confignode.rpc.thrift.TSetStorageGroupReq;
-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;
@@ -46,11 +48,15 @@ import org.junit.Test;
import org.junit.experimental.categories.Category;
import org.junit.runner.RunWith;
+import java.io.IOException;
+import java.nio.ByteBuffer;
import java.util.Map;
import java.util.concurrent.ConcurrentHashMap;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicInteger;
+import static org.apache.iotdb.confignode.it.utils.ConfigNodeTestUtils.generatePatternTreeBuffer;
+
@RunWith(IoTDBTestRunner.class)
@Category({ClusterIT.class})
public class IoTDBAutoRegionGroupExtensionIT {
@@ -61,7 +67,8 @@ public class IoTDBAutoRegionGroupExtensionIT {
private static final String sg = "root.sg";
private static final int testSgNum = 2;
private static final long testTimePartitionInterval = 604800000;
- private static final int testLeastDataRegionGroupNum = 5;
+ private static final int testMinSchemaRegionGroupNum = 2;
+ private static final int testMinDataRegionGroupNum = 2;
@Before
public void setUp() throws Exception {
@@ -116,18 +123,33 @@ public class IoTDBAutoRegionGroupExtensionIT {
}
private void setStorageGroupAndCheckRegionGroupDistribution(SyncConfigNodeIServiceClient client)
- throws TException {
+ throws TException, IllegalPathException, IOException {
+
for (int i = 0; i < testSgNum; i++) {
String curSg = sg + i;
TSStatus status =
- client.setStorageGroup(new TSetStorageGroupReq(new TStorageGroupSchema(curSg)));
+ client.setStorageGroup(
+ new TSetStorageGroupReq(
+ new TStorageGroupSchema(curSg)
+ .setMinSchemaRegionGroupNum(testMinSchemaRegionGroupNum)
+ .setMinDataRegionGroupNum(testMinDataRegionGroupNum)));
Assert.assertEquals(TSStatusCode.SUCCESS_STATUS.getStatusCode(), status.getCode());
- }
- for (int i = 0; i < testSgNum; i++) {
- String curSg = sg + i;
+ // Insert SchemaPartitions to create SchemaRegionGroups
+ String d0 = curSg + ".d0.s";
+ String d1 = curSg + ".d1.s";
+ String d2 = curSg + ".d2.s";
+ String d3 = curSg + ".d3.s";
+ TSchemaPartitionReq schemaPartitionReq = new TSchemaPartitionReq();
+ TSchemaPartitionTableResp schemaPartitionTableResp;
+ ByteBuffer buffer = generatePatternTreeBuffer(new String[] {d0, d1, d2, d3});
+ schemaPartitionReq.setPathPatternTree(buffer);
+ schemaPartitionTableResp = client.getOrCreateSchemaPartitionTable(schemaPartitionReq);
+ Assert.assertEquals(
+ TSStatusCode.SUCCESS_STATUS.getStatusCode(),
+ schemaPartitionTableResp.getStatus().getCode());
- /* Insert a DataPartition to create DataRegionGroups */
+ // Insert DataPartitions to create DataRegionGroups
Map<String, Map<TSeriesPartitionSlot, TTimeSlotList>> partitionSlotsMap =
ConfigNodeTestUtils.constructPartitionSlotsMap(
curSg, 0, 10, 0, 10, testTimePartitionInterval);
@@ -138,20 +160,10 @@ public class IoTDBAutoRegionGroupExtensionIT {
dataPartitionTableResp.getStatus().getCode());
}
- // Re-calculate the least DataRegionGroup num based on the test resource
- int totalCpuCoreNum = 0;
- TShowDataNodesResp showDataNodesResp = client.showDataNodes();
- for (TDataNodeInfo dataNodeInfo : showDataNodesResp.getDataNodesInfoList()) {
- totalCpuCoreNum += dataNodeInfo.getCpuCoreNum();
- }
- final int leastDataRegionGroupNum =
- Math.min(
- testLeastDataRegionGroupNum,
- (int)
- Math.ceil((double) totalCpuCoreNum / (double) (testSgNum * testReplicationFactor)));
-
- /* Check the number of DataRegionGroups */
- TShowRegionResp showRegionReq = client.showRegion(new TShowRegionReq());
+ // The number of SchemaRegionGroups should not less than the testMinSchemaRegionGroupNum
+ TShowRegionResp showRegionReq =
+ client.showRegion(
+ new TShowRegionReq().setConsensusGroupType(TConsensusGroupType.SchemaRegion));
Assert.assertEquals(
TSStatusCode.SUCCESS_STATUS.getStatusCode(), showRegionReq.getStatus().getCode());
Map<String, AtomicInteger> regionCounter = new ConcurrentHashMap<>();
@@ -164,6 +176,24 @@ public class IoTDBAutoRegionGroupExtensionIT {
.getAndIncrement());
Assert.assertEquals(testSgNum, regionCounter.size());
regionCounter.forEach(
- (sg, regionCount) -> Assert.assertEquals(leastDataRegionGroupNum, regionCount.get()));
+ (sg, regionCount) -> Assert.assertTrue(regionCount.get() >= testMinSchemaRegionGroupNum));
+
+ // The number of DataRegionGroups should not less than the testMinDataRegionGroupNum
+ showRegionReq =
+ client.showRegion(
+ new TShowRegionReq().setConsensusGroupType(TConsensusGroupType.DataRegion));
+ Assert.assertEquals(
+ TSStatusCode.SUCCESS_STATUS.getStatusCode(), showRegionReq.getStatus().getCode());
+ regionCounter.clear();
+ showRegionReq
+ .getRegionInfoList()
+ .forEach(
+ regionInfo ->
+ regionCounter
+ .computeIfAbsent(regionInfo.getStorageGroup(), empty -> new AtomicInteger(0))
+ .getAndIncrement());
+ Assert.assertEquals(testSgNum, regionCounter.size());
+ regionCounter.forEach(
+ (sg, regionCount) -> Assert.assertTrue(regionCount.get() >= testMinDataRegionGroupNum));
}
}
diff --git a/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBCustomRegionGroupExtensionIT.java b/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBCustomRegionGroupExtensionIT.java
index b787df976c..2342804fa2 100644
--- a/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBCustomRegionGroupExtensionIT.java
+++ b/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBCustomRegionGroupExtensionIT.java
@@ -77,9 +77,9 @@ public class IoTDBCustomRegionGroupExtensionIT {
.setSchemaReplicationFactor(testReplicationFactor)
.setDataReplicationFactor(testReplicationFactor)
.setSchemaRegionGroupExtensionPolicy(testSchemaRegionGroupExtensionPolicy)
- .setSchemaRegionGroupPerDatabase(testSchemaRegionGroupPerDatabase)
+ .setDefaultSchemaRegionGroupNumPerDatabase(testSchemaRegionGroupPerDatabase)
.setDataRegionGroupExtensionPolicy(testDataRegionGroupExtensionPolicy)
- .setDataRegionGroupPerDatabase(testDataRegionGroupPerDatabase)
+ .setDefaultDataRegionGroupNumPerDatabase(testDataRegionGroupPerDatabase)
.setTimePartitionInterval(testTimePartitionInterval);
// Init 1C3D environment
diff --git a/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionCreationIT.java b/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionCreationIT.java
index 193191d657..2dcccfa981 100644
--- a/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionCreationIT.java
+++ b/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionCreationIT.java
@@ -68,7 +68,7 @@ public class IoTDBPartitionCreationIT {
private static final String sg = "root.sg";
private static final int testSeriesPartitionBatchSize = 1;
private static final int testTimePartitionBatchSize = 1;
- private static final int testLeastDataRegionGroupNum = 4;
+ private static final int testDataRegionGroupPerDatabase = 4;
private static final TEndPoint defaultEndPoint = new TEndPoint("-1", -1);
private static final TDataNodeLocation defaultDataNode =
new TDataNodeLocation(
@@ -90,7 +90,7 @@ public class IoTDBPartitionCreationIT {
.setSchemaReplicationFactor(testReplicationFactor)
.setDataReplicationFactor(testReplicationFactor)
.setTimePartitionInterval(testTimePartitionInterval)
- .setLeastDataRegionGroupNum(testLeastDataRegionGroupNum);
+ .setDefaultDataRegionGroupNumPerDatabase(testDataRegionGroupPerDatabase);
// Init 1C3D environment
EnvFactory.getEnv().initClusterEnvironment(1, 3);
diff --git a/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionGetterIT.java b/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionGetterIT.java
index c03dd3b301..e66f36f135 100644
--- a/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionGetterIT.java
+++ b/integration-test/src/test/java/org/apache/iotdb/confignode/it/partition/IoTDBPartitionGetterIT.java
@@ -76,7 +76,7 @@ public class IoTDBPartitionGetterIT {
private static final String testConsensusProtocolClass = ConsensusFactory.RATIS_CONSENSUS;
private static final int testReplicationFactor = 3;
private static final long testTimePartitionInterval = 604800000;
- private static final int testLeastDataRegionGroupNum = 5;
+ private static final int testDataRegionGroupPerDatabase = 5;
private static final String sg = "root.sg";
private static final int storageGroupNum = 2;
@@ -96,7 +96,7 @@ public class IoTDBPartitionGetterIT {
.setSchemaReplicationFactor(testReplicationFactor)
.setDataReplicationFactor(testReplicationFactor)
.setTimePartitionInterval(testTimePartitionInterval)
- .setLeastDataRegionGroupNum(testLeastDataRegionGroupNum);
+ .setDefaultDataRegionGroupNumPerDatabase(testDataRegionGroupPerDatabase);
// .setSeriesSlotNum(testSeriesPartitionSlotNum);
// Init 1C3D environment
EnvFactory.getEnv().initClusterEnvironment(1, 3);
@@ -293,7 +293,7 @@ public class IoTDBPartitionGetterIT {
(int)
Math.ceil(
(double) totalCpuCoreNum / (double) (storageGroupNum * testReplicationFactor));
- leastDataRegionGroupNum = Math.min(leastDataRegionGroupNum, testLeastDataRegionGroupNum);
+ leastDataRegionGroupNum = Math.min(leastDataRegionGroupNum, testDataRegionGroupPerDatabase);
for (int i = 0; i < storageGroupNum; i++) {
String storageGroup = sg + i;
diff --git a/integration-test/src/test/java/org/apache/iotdb/confignode/it/utils/ConfigNodeTestUtils.java b/integration-test/src/test/java/org/apache/iotdb/confignode/it/utils/ConfigNodeTestUtils.java
index 915580add3..64046dd76d 100644
--- a/integration-test/src/test/java/org/apache/iotdb/confignode/it/utils/ConfigNodeTestUtils.java
+++ b/integration-test/src/test/java/org/apache/iotdb/confignode/it/utils/ConfigNodeTestUtils.java
@@ -237,7 +237,6 @@ public class ConfigNodeTestUtils {
clusterParameters.setSchemaRegionPerDataNode(1.0);
clusterParameters.setDiskSpaceWarningThreshold(0.05);
clusterParameters.setReadConsistencyLevel("strong");
- clusterParameters.setLeastDataRegionGroupNum(5);
return clusterParameters;
}
diff --git a/integration-test/src/test/java/org/apache/iotdb/db/it/builtinfunction/scalar/IoTDBDiffFunction2IT.java b/integration-test/src/test/java/org/apache/iotdb/db/it/builtinfunction/scalar/IoTDBDiffFunction2IT.java
index d67c1f32ee..7015e15caa 100644
--- a/integration-test/src/test/java/org/apache/iotdb/db/it/builtinfunction/scalar/IoTDBDiffFunction2IT.java
+++ b/integration-test/src/test/java/org/apache/iotdb/db/it/builtinfunction/scalar/IoTDBDiffFunction2IT.java
@@ -37,7 +37,7 @@ public class IoTDBDiffFunction2IT extends IoTDBDiffFunctionIT {
@BeforeClass
public static void setUp() throws Exception {
EnvFactory.getEnv().getConfig().getCommonConfig().setDataRegionGroupExtensionPolicy("CUSTOM");
- EnvFactory.getEnv().getConfig().getCommonConfig().setDataRegionGroupPerDatabase(2);
+ EnvFactory.getEnv().getConfig().getCommonConfig().setDefaultDataRegionGroupNumPerDatabase(2);
EnvFactory.getEnv().getConfig().getCommonConfig().setPartitionInterval(1000);
EnvFactory.getEnv().initClusterEnvironment();
prepareData(SQLs);
diff --git a/integration-test/src/test/java/org/apache/iotdb/db/it/builtinfunction/scalar/IoTDBDiffFunction3IT.java b/integration-test/src/test/java/org/apache/iotdb/db/it/builtinfunction/scalar/IoTDBDiffFunction3IT.java
index b684791bad..3b73c3efcf 100644
--- a/integration-test/src/test/java/org/apache/iotdb/db/it/builtinfunction/scalar/IoTDBDiffFunction3IT.java
+++ b/integration-test/src/test/java/org/apache/iotdb/db/it/builtinfunction/scalar/IoTDBDiffFunction3IT.java
@@ -37,7 +37,7 @@ public class IoTDBDiffFunction3IT extends IoTDBDiffFunctionIT {
@BeforeClass
public static void setUp() throws Exception {
EnvFactory.getEnv().getConfig().getCommonConfig().setDataRegionGroupExtensionPolicy("CUSTOM");
- EnvFactory.getEnv().getConfig().getCommonConfig().setDataRegionGroupPerDatabase(3);
+ EnvFactory.getEnv().getConfig().getCommonConfig().setDefaultDataRegionGroupNumPerDatabase(3);
EnvFactory.getEnv().getConfig().getCommonConfig().setPartitionInterval(1000);
EnvFactory.getEnv().initClusterEnvironment();
prepareData(SQLs);
diff --git a/integration-test/src/test/java/org/apache/iotdb/zeppelin/it/IoTDBInterpreterIT.java b/integration-test/src/test/java/org/apache/iotdb/zeppelin/it/IoTDBInterpreterIT.java
index 5a56c6d1a9..ba6170c0cb 100644
--- a/integration-test/src/test/java/org/apache/iotdb/zeppelin/it/IoTDBInterpreterIT.java
+++ b/integration-test/src/test/java/org/apache/iotdb/zeppelin/it/IoTDBInterpreterIT.java
@@ -54,7 +54,7 @@ public class IoTDBInterpreterIT {
static final String SET_TIMESTAMP_DISPLAY = "set time_display_type";
@BeforeClass
- public static void open() throws InterruptedException {
+ public static void open() {
EnvFactory.getEnv().initClusterEnvironment();
Properties properties = new Properties();
properties.put(IOTDB_HOST, EnvFactory.getEnv().getIP());
@@ -345,7 +345,7 @@ public class IoTDBInterpreterIT {
public void testShowAllTTL() {
interpreter.internalInterpret("SET TTL TO root.test.wf01 12345", null);
InterpreterResult actual = interpreter.internalInterpret("SHOW ALL TTL", null);
- String gt = "Database\tTTL(ms)\n" + "root.test.wf02\tnull\n" + "root.test.wf01\t12345";
+ String gt = "Database\tTTL\n" + "root.test.wf02\tnull\n" + "root.test.wf01\t12345";
Assert.assertNotNull(actual);
Assert.assertEquals(Code.SUCCESS, actual.code());
Assert.assertEquals(gt, actual.message().get(0).getData());
@@ -355,7 +355,7 @@ public class IoTDBInterpreterIT {
public void testShowTTL() {
interpreter.internalInterpret("SET TTL TO root.test.wf01 12345", null);
InterpreterResult actual = interpreter.internalInterpret("SHOW TTL ON root.test.wf01", null);
- String gt = "Database\tTTL(ms)\n" + "root.test.wf01\t12345";
+ String gt = "Database\tTTL\n" + "root.test.wf01\t12345";
Assert.assertNotNull(actual);
Assert.assertEquals(Code.SUCCESS, actual.code());
Assert.assertEquals(gt, actual.message().get(0).getData());
@@ -365,9 +365,9 @@ public class IoTDBInterpreterIT {
public void testShowStorageGroup() {
InterpreterResult actual = interpreter.internalInterpret("SHOW DATABASES", null);
String gt =
- "Database\tTTL(ms)\tSchemaReplicationFactor\tDataReplicationFactor\tTimePartitionInterval\tSchemaRegionNum\tDataRegionNum\n"
- + "root.test.wf02\tnull\t1\t1\t604800000\t1\t1\n"
- + "root.test.wf01\tnull\t1\t1\t604800000\t1\t1";
+ "Database\tTTL\tSchemaReplicationFactor\tDataReplicationFactor\tTimePartitionInterval\n"
+ + "root.test.wf02\tnull\t1\t1\t604800000\n"
+ + "root.test.wf01\tnull\t1\t1\t604800000";
Assert.assertNotNull(actual);
Assert.assertEquals(Code.SUCCESS, actual.code());
Assert.assertEquals(gt, actual.message().get(0).getData());
diff --git a/node-commons/src/assembly/resources/conf/iotdb-common.properties b/node-commons/src/assembly/resources/conf/iotdb-common.properties
index 2bfc8386cd..6d0ab7964b 100644
--- a/node-commons/src/assembly/resources/conf/iotdb-common.properties
+++ b/node-commons/src/assembly/resources/conf/iotdb-common.properties
@@ -85,13 +85,6 @@ cluster_name=defaultCluster
# series_partition_executor_class=org.apache.iotdb.commons.partition.executor.hash.BKDRHashExecutor
-# The maximum number of SchemaRegions expected to be managed by each DataNode.
-# Notice: Since each Database requires at least one SchemaRegionGroup to manage its schema,
-# this parameter doesn't limit the number of SchemaRegions when there are too many Databases.
-# Default is equal to the schema_replication_factor.
-# Datatype: Double
-# schema_region_per_data_node=1.0
-
# The policy of extension SchemaRegionGroup for each Database.
# These policies are currently supported:
# 1. CUSTOM(Each Database will allocate schema_region_group_per_database RegionGroups as soon as created)
@@ -99,10 +92,20 @@ cluster_name=defaultCluster
# Datatype: String
# schema_region_group_extension_policy=AUTO
-# The number of SchemaRegionGroups for each Database when using CUSTOM schema_region_group_extension_policy.
-# Notice: Each Database will allocate schema_region_group_per_database SchemaRegionGroups as soon as created.
+# When set schema_region_group_extension_policy=CUSTOM,
+# this parameter is the default number of SchemaRegionGroups for each Database.
+# When set schema_region_group_extension_policy=AUTO,
+# this parameter is the default minimal number of SchemaRegionGroups for each Database.
# Datatype: Integer
-# schema_region_group_per_database=1
+# default_schema_region_group_num_per_database=1
+
+# Only take effect when set schema_region_group_extension_policy=AUTO.
+# This parameter is the maximum number of SchemaRegions expected to be managed by each DataNode.
+# Notice: Since each Database requires at least one SchemaRegionGroup to manage its schema,
+# this parameter doesn't limit the upper bound of cluster SchemaRegions when there are too many Databases.
+# Default is equal to the schema_replication_factor.
+# Datatype: Double
+# schema_region_per_data_node=1.0
# The policy of extension DataRegionGroup for each Database.
# These policies are currently supported:
@@ -111,25 +114,20 @@ cluster_name=defaultCluster
# Datatype: String
# data_region_group_extension_policy=AUTO
-# The number of DataRegionGroups for each Database when using CUSTOM data_region_group_extension_policy.
-# Notice: Each Database will allocate data_region_group_per_database DataRegionGroups as soon as created.
+# When set data_region_group_extension_policy=CUSTOM,
+# this parameter is the default number of DataRegionGroups for each Database.
+# When set data_region_group_extension_policy=AUTO,
+# this parameter is the default minimal number of DataRegionGroups for each Database.
# Datatype: Integer
-# data_region_group_per_database=1
+# default_data_region_group_num_per_database=2
-# The maximum number of DataRegions expected to be managed by each processor
-# when using AUTO data_region_group_extension_policy.
+# Only take effect when set data_region_group_extension_policy=AUTO.
+# This parameter is the maximum number of DataRegions expected to be managed by each processor.
# Notice: Since each Database requires at least two DataRegionGroups to manage its data,
-# this parameter doesn't limit the number of DataRegions when there are too many Databases.
+# this parameter doesn't limit the upper bound of cluster DataRegions when there are too many Databases.
# Datatype: Double
# data_region_per_processor=1.0
-# The least number of DataRegionGroup for each StorageGroup.
-# The ConfigNode-leader will create a DataRegionGroup for each newborn SeriesPartitionSlot
-# for the newly created StorageGroup until the number of DataRegionGroup is equal to this parameter.
-# Notice: In order to ensure the efficiency of concurrent write, this parameter should greater than 1.
-# Datatype: int
-# least_data_region_group_num=5
-
# Whether to enable the DataPartition inherit policy.
# DataPartition within the same SeriesPartitionSlot will inherit
diff --git a/server/src/main/java/org/apache/iotdb/db/mpp/common/header/ColumnHeaderConstant.java b/server/src/main/java/org/apache/iotdb/db/mpp/common/header/ColumnHeaderConstant.java
index ebddd19f06..bc6df1cbf2 100644
--- a/server/src/main/java/org/apache/iotdb/db/mpp/common/header/ColumnHeaderConstant.java
+++ b/server/src/main/java/org/apache/iotdb/db/mpp/common/header/ColumnHeaderConstant.java
@@ -19,8 +19,6 @@
package org.apache.iotdb.db.mpp.common.header;
-import org.apache.iotdb.commons.conf.CommonConfig;
-import org.apache.iotdb.commons.conf.CommonDescriptor;
import org.apache.iotdb.tsfile.file.metadata.enums.TSDataType;
import com.google.common.collect.ImmutableList;
@@ -28,7 +26,6 @@ import com.google.common.collect.ImmutableList;
import java.util.List;
public class ColumnHeaderConstant {
- private static final CommonConfig commonConfig = CommonDescriptor.getInstance().getConfig();
// column names for query statement
public static final String TIME = "Time";
@@ -48,10 +45,16 @@ public class ColumnHeaderConstant {
public static final String DEADBAND_PARAMETERS = "DeadbandParameters";
public static final String IS_ALIGNED = "IsAligned";
public static final String COUNT = "Count";
- public static final String COLUMN_TTL = "TTL(ms)";
+ public static final String COLUMN_TTL = "TTL";
public static final String SCHEMA_REPLICATION_FACTOR = "SchemaReplicationFactor";
public static final String DATA_REPLICATION_FACTOR = "DataReplicationFactor";
public static final String TIME_PARTITION_INTERVAL = "TimePartitionInterval";
+ public static final String SCHEMA_REGION_GROUP_NUM = "SchemaRegionGroupNum";
+ public static final String MIN_SCHEMA_REGION_GROUP_NUM = "MinSchemaRegionGroupNum";
+ public static final String MAX_SCHEMA_REGION_GROUP_NUM = "MaxSchemaRegionGroupNum";
+ public static final String DATA_REGION_GROUP_NUM = "DataRegionGroupNum";
+ public static final String MIN_DATA_REGION_GROUP_NUM = "MinDataRegionGroupNum";
+ public static final String MAX_DATA_REGION_GROUP_NUM = "MaxDataRegionGroupNum";
public static final String CHILD_PATHS = "ChildPaths";
public static final String NODE_TYPES = "NodeTypes";
public static final String CHILD_NODES = "ChildNodes";
@@ -71,7 +74,6 @@ public class ColumnHeaderConstant {
public static final String NODE_ID = "NodeID";
public static final String NODE_TYPE = "NodeType";
public static final String STATUS = "Status";
- public static final String HOST = "Host";
public static final String INTERNAL_ADDRESS = "InternalAddress";
public static final String INTERNAL_PORT = "InternalPort";
public static final String CONFIG_CONSENSUS_PORT = "ConfigConsensusPort";
@@ -110,7 +112,6 @@ public class ColumnHeaderConstant {
public static final String DATA_REGION_PER_PROCESSOR = "DataRegionPerProcessor";
public static final String READ_CONSISTENCY_LEVEL = "ReadConsistencyLevel";
public static final String DISK_SPACE_WARNING_THRESHOLD = "DiskSpaceWarningThreshold";
- public static final String LEAST_DATA_REGION_GROUP_NUM = "LeastDataRegionGroupNum";
// column names for show region statement
public static final String REGION_ID = "RegionId";
@@ -122,8 +123,8 @@ public class ColumnHeaderConstant {
public static final String ROLE = "Role";
// column names for show datanodes
- public static final String DATA_REGION_NUM = "DataRegionNum";
public static final String SCHEMA_REGION_NUM = "SchemaRegionNum";
+ public static final String DATA_REGION_NUM = "DataRegionNum";
// column names for show schema template statement
public static final String TEMPLATE_NAME = "TemplateName";
@@ -185,15 +186,27 @@ public class ColumnHeaderConstant {
new ColumnHeader(DATABASE, TSDataType.TEXT),
new ColumnHeader(COLUMN_TTL, TSDataType.INT64));
- public static final List<ColumnHeader> showStorageGroupColumnHeaders =
+ public static final List<ColumnHeader> showStorageGroupsColumnHeaders =
+ ImmutableList.of(
+ new ColumnHeader(DATABASE, TSDataType.TEXT),
+ new ColumnHeader(COLUMN_TTL, TSDataType.INT64),
+ new ColumnHeader(SCHEMA_REPLICATION_FACTOR, TSDataType.INT32),
+ new ColumnHeader(DATA_REPLICATION_FACTOR, TSDataType.INT32),
+ new ColumnHeader(TIME_PARTITION_INTERVAL, TSDataType.INT64));
+
+ public static final List<ColumnHeader> showStorageGroupsDetailColumnHeaders =
ImmutableList.of(
new ColumnHeader(DATABASE, TSDataType.TEXT),
new ColumnHeader(COLUMN_TTL, TSDataType.INT64),
new ColumnHeader(SCHEMA_REPLICATION_FACTOR, TSDataType.INT32),
new ColumnHeader(DATA_REPLICATION_FACTOR, TSDataType.INT32),
new ColumnHeader(TIME_PARTITION_INTERVAL, TSDataType.INT64),
- new ColumnHeader(SCHEMA_REGION_NUM, TSDataType.INT32),
- new ColumnHeader(DATA_REGION_NUM, TSDataType.INT32));
+ new ColumnHeader(SCHEMA_REGION_GROUP_NUM, TSDataType.INT32),
+ new ColumnHeader(MIN_SCHEMA_REGION_GROUP_NUM, TSDataType.INT32),
+ new ColumnHeader(MAX_SCHEMA_REGION_GROUP_NUM, TSDataType.INT32),
+ new ColumnHeader(DATA_REGION_GROUP_NUM, TSDataType.INT32),
+ new ColumnHeader(MIN_DATA_REGION_GROUP_NUM, TSDataType.INT32),
+ new ColumnHeader(MAX_DATA_REGION_GROUP_NUM, TSDataType.INT32));
public static final List<ColumnHeader> showChildPathsColumnHeaders =
ImmutableList.of(
diff --git a/server/src/main/java/org/apache/iotdb/db/mpp/common/header/DatasetHeaderFactory.java b/server/src/main/java/org/apache/iotdb/db/mpp/common/header/DatasetHeaderFactory.java
index f452e066df..7e8eb19b05 100644
--- a/server/src/main/java/org/apache/iotdb/db/mpp/common/header/DatasetHeaderFactory.java
+++ b/server/src/main/java/org/apache/iotdb/db/mpp/common/header/DatasetHeaderFactory.java
@@ -53,8 +53,10 @@ public class DatasetHeaderFactory {
return new DatasetHeader(ColumnHeaderConstant.showDevicesWithSgColumnHeaders, true);
}
- public static DatasetHeader getShowStorageGroupHeader() {
- return new DatasetHeader(ColumnHeaderConstant.showStorageGroupColumnHeaders, true);
+ public static DatasetHeader getShowStorageGroupHeader(boolean isDetailed) {
+ return isDetailed
+ ? new DatasetHeader(ColumnHeaderConstant.showStorageGroupsDetailColumnHeaders, true)
+ : new DatasetHeader(ColumnHeaderConstant.showStorageGroupsColumnHeaders, true);
}
public static DatasetHeader getShowTTLHeader() {
diff --git a/server/src/main/java/org/apache/iotdb/db/mpp/plan/analyze/AnalyzeVisitor.java b/server/src/main/java/org/apache/iotdb/db/mpp/plan/analyze/AnalyzeVisitor.java
index b09f3278e8..3ad3545145 100644
--- a/server/src/main/java/org/apache/iotdb/db/mpp/plan/analyze/AnalyzeVisitor.java
+++ b/server/src/main/java/org/apache/iotdb/db/mpp/plan/analyze/AnalyzeVisitor.java
@@ -2169,7 +2169,8 @@ public class AnalyzeVisitor extends StatementVisitor<Analysis, MPPQueryContext>
ShowStorageGroupStatement showStorageGroupStatement, MPPQueryContext context) {
Analysis analysis = new Analysis();
analysis.setStatement(showStorageGroupStatement);
- analysis.setRespDatasetHeader(DatasetHeaderFactory.getShowStorageGroupHeader());
+ analysis.setRespDatasetHeader(
+ DatasetHeaderFactory.getShowStorageGroupHeader(showStorageGroupStatement.isDetailed()));
return analysis;
}
diff --git a/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/executor/ClusterConfigTaskExecutor.java b/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/executor/ClusterConfigTaskExecutor.java
index 9a7bad6b5d..d9fbf026b8 100644
--- a/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/executor/ClusterConfigTaskExecutor.java
+++ b/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/executor/ClusterConfigTaskExecutor.java
@@ -95,7 +95,6 @@ import org.apache.iotdb.db.mpp.plan.execution.config.metadata.ShowContinuousQuer
import org.apache.iotdb.db.mpp.plan.execution.config.metadata.ShowDataNodesTask;
import org.apache.iotdb.db.mpp.plan.execution.config.metadata.ShowFunctionsTask;
import org.apache.iotdb.db.mpp.plan.execution.config.metadata.ShowRegionTask;
-import org.apache.iotdb.db.mpp.plan.execution.config.metadata.ShowStorageGroupTask;
import org.apache.iotdb.db.mpp.plan.execution.config.metadata.ShowTTLTask;
import org.apache.iotdb.db.mpp.plan.execution.config.metadata.ShowTriggersTask;
import org.apache.iotdb.db.mpp.plan.execution.config.metadata.ShowVariablesTask;
@@ -240,7 +239,7 @@ public class ClusterConfigTaskExecutor implements IConfigTaskExecutor {
// Send request to some API server
TShowStorageGroupResp resp = client.showStorageGroup(storageGroupPathPattern);
// build TSBlock
- ShowStorageGroupTask.buildTSBlock(resp.getStorageGroupInfoMap(), future);
+ showStorageGroupStatement.buildTSBlock(resp.getStorageGroupInfoMap(), future);
} catch (ClientManagerException | TException e) {
future.setException(e);
}
diff --git a/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/SetStorageGroupTask.java b/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/SetStorageGroupTask.java
index 5af256ff0f..c5b601a451 100644
--- a/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/SetStorageGroupTask.java
+++ b/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/SetStorageGroupTask.java
@@ -62,6 +62,13 @@ public class SetStorageGroupTask implements IConfigTask {
storageGroupSchema.setTimePartitionInterval(
setStorageGroupStatement.getTimePartitionInterval());
}
+ if (setStorageGroupStatement.getSchemaRegionGroupNum() != null) {
+ storageGroupSchema.setMinSchemaRegionGroupNum(
+ setStorageGroupStatement.getSchemaRegionGroupNum());
+ }
+ if (setStorageGroupStatement.getDataRegionGroupNum() != null) {
+ storageGroupSchema.setMinDataRegionGroupNum(setStorageGroupStatement.getDataRegionGroupNum());
+ }
return storageGroupSchema;
}
}
diff --git a/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/ShowStorageGroupTask.java b/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/ShowStorageGroupTask.java
index 372ee9d2f7..0cb0d4ddc4 100644
--- a/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/ShowStorageGroupTask.java
+++ b/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/ShowStorageGroupTask.java
@@ -19,26 +19,12 @@
package org.apache.iotdb.db.mpp.plan.execution.config.metadata;
-import org.apache.iotdb.confignode.rpc.thrift.TStorageGroupInfo;
-import org.apache.iotdb.db.mpp.common.header.ColumnHeader;
-import org.apache.iotdb.db.mpp.common.header.ColumnHeaderConstant;
-import org.apache.iotdb.db.mpp.common.header.DatasetHeader;
-import org.apache.iotdb.db.mpp.common.header.DatasetHeaderFactory;
import org.apache.iotdb.db.mpp.plan.execution.config.ConfigTaskResult;
import org.apache.iotdb.db.mpp.plan.execution.config.IConfigTask;
import org.apache.iotdb.db.mpp.plan.execution.config.executor.IConfigTaskExecutor;
import org.apache.iotdb.db.mpp.plan.statement.metadata.ShowStorageGroupStatement;
-import org.apache.iotdb.rpc.TSStatusCode;
-import org.apache.iotdb.tsfile.file.metadata.enums.TSDataType;
-import org.apache.iotdb.tsfile.read.common.block.TsBlockBuilder;
-import org.apache.iotdb.tsfile.utils.Binary;
import com.google.common.util.concurrent.ListenableFuture;
-import com.google.common.util.concurrent.SettableFuture;
-
-import java.util.List;
-import java.util.Map;
-import java.util.stream.Collectors;
public class ShowStorageGroupTask implements IConfigTask {
@@ -53,32 +39,4 @@ public class ShowStorageGroupTask implements IConfigTask {
throws InterruptedException {
return configTaskExecutor.showStorageGroup(showStorageGroupStatement);
}
-
- public static void buildTSBlock(
- Map<String, TStorageGroupInfo> storageGroupInfoMap, SettableFuture<ConfigTaskResult> future) {
- List<TSDataType> outputDataTypes =
- ColumnHeaderConstant.showStorageGroupColumnHeaders.stream()
- .map(ColumnHeader::getColumnType)
- .collect(Collectors.toList());
- TsBlockBuilder builder = new TsBlockBuilder(outputDataTypes);
- for (Map.Entry<String, TStorageGroupInfo> entry : storageGroupInfoMap.entrySet()) {
- String storageGroup = entry.getKey();
- TStorageGroupInfo storageGroupInfo = entry.getValue();
- builder.getTimeColumnBuilder().writeLong(0L);
- builder.getColumnBuilder(0).writeBinary(new Binary(storageGroup));
- if (Long.MAX_VALUE == storageGroupInfo.getTTL()) {
- builder.getColumnBuilder(1).appendNull();
- } else {
- builder.getColumnBuilder(1).writeLong(storageGroupInfo.getTTL());
- }
- builder.getColumnBuilder(2).writeInt(storageGroupInfo.getSchemaReplicationFactor());
- builder.getColumnBuilder(3).writeInt(storageGroupInfo.getDataReplicationFactor());
- builder.getColumnBuilder(4).writeLong(storageGroupInfo.getTimePartitionInterval());
- builder.getColumnBuilder(5).writeInt(storageGroupInfo.getSchemaRegionNum());
- builder.getColumnBuilder(6).writeInt(storageGroupInfo.getDataRegionNum());
- builder.declarePosition();
- }
- DatasetHeader datasetHeader = DatasetHeaderFactory.getShowStorageGroupHeader();
- future.set(new ConfigTaskResult(TSStatusCode.SUCCESS_STATUS, builder.build(), datasetHeader));
- }
}
diff --git a/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/ShowVariablesTask.java b/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/ShowVariablesTask.java
index 794d3aa84c..90e3ffd166 100644
--- a/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/ShowVariablesTask.java
+++ b/server/src/main/java/org/apache/iotdb/db/mpp/plan/execution/config/metadata/ShowVariablesTask.java
@@ -105,10 +105,6 @@ public class ShowVariablesTask implements IConfigTask {
builder,
new Binary(ColumnHeaderConstant.DATA_REGION_PER_PROCESSOR),
new Binary(String.valueOf(clusterParameters.getDataRegionPerProcessor())));
- buildTSBlock(
- builder,
- new Binary(ColumnHeaderConstant.LEAST_DATA_REGION_GROUP_NUM),
- new Binary(String.valueOf(clusterParameters.getLeastDataRegionGroupNum())));
buildTSBlock(
builder,
new Binary(ColumnHeaderConstant.SERIES_SLOT_NUM),
diff --git a/server/src/main/java/org/apache/iotdb/db/mpp/plan/parser/ASTVisitor.java b/server/src/main/java/org/apache/iotdb/db/mpp/plan/parser/ASTVisitor.java
index 919b943b8b..4be6e2b70b 100644
--- a/server/src/main/java/org/apache/iotdb/db/mpp/plan/parser/ASTVisitor.java
+++ b/server/src/main/java/org/apache/iotdb/db/mpp/plan/parser/ASTVisitor.java
@@ -589,11 +589,20 @@ public class ASTVisitor extends IoTDBSqlParserBaseVisitor<Statement> {
@Override
public Statement visitShowStorageGroup(IoTDBSqlParser.ShowStorageGroupContext ctx) {
+ ShowStorageGroupStatement showStorageGroupStatement;
+
+ // Parse prefixPath
if (ctx.prefixPath() != null) {
- return new ShowStorageGroupStatement(parsePrefixPath(ctx.prefixPath()));
+ showStorageGroupStatement = new ShowStorageGroupStatement(parsePrefixPath(ctx.prefixPath()));
} else {
- return new ShowStorageGroupStatement(new PartialPath(SqlConstant.getSingleRootArray()));
+ showStorageGroupStatement =
+ new ShowStorageGroupStatement(new PartialPath(SqlConstant.getSingleRootArray()));
}
+
+ // Is detailed
+ showStorageGroupStatement.setDetailed(ctx.DETAILS() != null);
+
+ return showStorageGroupStatement;
}
// Show Devices ========================================================================
@@ -1925,15 +1934,11 @@ public class ASTVisitor extends IoTDBSqlParserBaseVisitor<Statement> {
private void parseStorageGroupAttributesClause(
SetStorageGroupStatement setStorageGroupStatement,
IoTDBSqlParser.StorageGroupAttributesClauseContext ctx) {
- if (!ctx.storageGroupAttributeClause().isEmpty()) {
- throw new RuntimeException(
- "Currently not support set ttl, schemaReplication factor, dataReplication factor, time partition interval to specific database.");
- }
for (IoTDBSqlParser.StorageGroupAttributeClauseContext attribute :
ctx.storageGroupAttributeClause()) {
if (attribute.TTL() != null) {
long ttl = Long.parseLong(attribute.INTEGER_LITERAL().getText());
- setStorageGroupStatement.setTtl(ttl);
+ setStorageGroupStatement.setTTL(ttl);
} else if (attribute.SCHEMA_REPLICATION_FACTOR() != null) {
int schemaReplicationFactor = Integer.parseInt(attribute.INTEGER_LITERAL().getText());
setStorageGroupStatement.setSchemaReplicationFactor(schemaReplicationFactor);
@@ -1943,6 +1948,12 @@ public class ASTVisitor extends IoTDBSqlParserBaseVisitor<Statement> {
} else if (attribute.TIME_PARTITION_INTERVAL() != null) {
long timePartitionInterval = Long.parseLong(attribute.INTEGER_LITERAL().getText());
setStorageGroupStatement.setTimePartitionInterval(timePartitionInterval);
+ } else if (attribute.SCHEMA_REGION_GROUP_NUM() != null) {
+ int schemaRegionGroupNum = Integer.parseInt(attribute.INTEGER_LITERAL().getText());
+ setStorageGroupStatement.setSchemaRegionGroupNum(schemaRegionGroupNum);
+ } else if (attribute.DATA_REGION_GROUP_NUM() != null) {
+ int dataRegionGroupNum = Integer.parseInt(attribute.INTEGER_LITERAL().getText());
+ setStorageGroupStatement.setDataRegionGroupNum(dataRegionGroupNum);
}
}
}
diff --git a/server/src/main/java/org/apache/iotdb/db/mpp/plan/statement/metadata/SetStorageGroupStatement.java b/server/src/main/java/org/apache/iotdb/db/mpp/plan/statement/metadata/SetStorageGroupStatement.java
index 86092522b1..46c55337b5 100644
--- a/server/src/main/java/org/apache/iotdb/db/mpp/plan/statement/metadata/SetStorageGroupStatement.java
+++ b/server/src/main/java/org/apache/iotdb/db/mpp/plan/statement/metadata/SetStorageGroupStatement.java
@@ -31,10 +31,12 @@ import java.util.List;
public class SetStorageGroupStatement extends Statement implements IConfigStatement {
private PartialPath storageGroupPath;
- private Long ttl = null;
+ private Long TTL = null;
private Integer schemaReplicationFactor = null;
private Integer dataReplicationFactor = null;
private Long timePartitionInterval = null;
+ private Integer schemaRegionGroupNum = null;
+ private Integer dataRegionGroupNum = null;
public SetStorageGroupStatement() {
super();
@@ -45,57 +47,73 @@ public class SetStorageGroupStatement extends Statement implements IConfigStatem
return storageGroupPath;
}
- @Override
- public <R, C> R accept(StatementVisitor<R, C> visitor, C context) {
- return visitor.visitSetStorageGroup(this, context);
- }
-
public void setStorageGroupPath(PartialPath storageGroupPath) {
this.storageGroupPath = storageGroupPath;
}
- public void setTtl(Long ttl) {
- this.ttl = ttl;
+ public Long getTTL() {
+ return TTL;
+ }
+
+ public void setTTL(Long TTL) {
+ this.TTL = TTL;
+ }
+
+ public Integer getSchemaReplicationFactor() {
+ return schemaReplicationFactor;
}
public void setSchemaReplicationFactor(Integer schemaReplicationFactor) {
this.schemaReplicationFactor = schemaReplicationFactor;
}
+ public Integer getDataReplicationFactor() {
+ return dataReplicationFactor;
+ }
+
public void setDataReplicationFactor(Integer dataReplicationFactor) {
this.dataReplicationFactor = dataReplicationFactor;
}
+ public Long getTimePartitionInterval() {
+ return timePartitionInterval;
+ }
+
public void setTimePartitionInterval(Long timePartitionInterval) {
this.timePartitionInterval = timePartitionInterval;
}
- @Override
- public QueryType getQueryType() {
- return QueryType.WRITE;
+ public Integer getSchemaRegionGroupNum() {
+ return schemaRegionGroupNum;
}
- @Override
- public List<PartialPath> getPaths() {
- return storageGroupPath != null
- ? Collections.singletonList(storageGroupPath)
- : Collections.emptyList();
+ public void setSchemaRegionGroupNum(Integer schemaRegionGroupNum) {
+ this.schemaRegionGroupNum = schemaRegionGroupNum;
}
- public Long getTTL() {
- return ttl;
+ public Integer getDataRegionGroupNum() {
+ return dataRegionGroupNum;
}
- public Integer getSchemaReplicationFactor() {
- return schemaReplicationFactor;
+ public void setDataRegionGroupNum(Integer dataRegionGroupNum) {
+ this.dataRegionGroupNum = dataRegionGroupNum;
}
- public Integer getDataReplicationFactor() {
- return dataReplicationFactor;
+ @Override
+ public <R, C> R accept(StatementVisitor<R, C> visitor, C context) {
+ return visitor.visitSetStorageGroup(this, context);
}
- public Long getTimePartitionInterval() {
- return timePartitionInterval;
+ @Override
+ public QueryType getQueryType() {
+ return QueryType.WRITE;
+ }
+
+ @Override
+ public List<PartialPath> getPaths() {
+ return storageGroupPath != null
+ ? Collections.singletonList(storageGroupPath)
+ : Collections.emptyList();
}
@Override
@@ -104,13 +122,17 @@ public class SetStorageGroupStatement extends Statement implements IConfigStatem
+ "storageGroupPath="
+ storageGroupPath
+ ", ttl="
- + ttl
+ + TTL
+ ", schemaReplicationFactor="
+ schemaReplicationFactor
+ ", dataReplicationFactor="
+ dataReplicationFactor
+ ", timePartitionInterval="
+ timePartitionInterval
+ + ", schemaRegionGroupNum="
+ + schemaRegionGroupNum
+ + ", dataRegionGroupNum="
+ + dataRegionGroupNum
+ '}';
}
}
diff --git a/server/src/main/java/org/apache/iotdb/db/mpp/plan/statement/metadata/ShowStorageGroupStatement.java b/server/src/main/java/org/apache/iotdb/db/mpp/plan/statement/metadata/ShowStorageGroupStatement.java
index 97fbdd0a05..c20a5ece03 100644
--- a/server/src/main/java/org/apache/iotdb/db/mpp/plan/statement/metadata/ShowStorageGroupStatement.java
+++ b/server/src/main/java/org/apache/iotdb/db/mpp/plan/statement/metadata/ShowStorageGroupStatement.java
@@ -20,12 +20,26 @@
package org.apache.iotdb.db.mpp.plan.statement.metadata;
import org.apache.iotdb.commons.path.PartialPath;
+import org.apache.iotdb.confignode.rpc.thrift.TStorageGroupInfo;
+import org.apache.iotdb.db.mpp.common.header.ColumnHeader;
+import org.apache.iotdb.db.mpp.common.header.ColumnHeaderConstant;
+import org.apache.iotdb.db.mpp.common.header.DatasetHeader;
+import org.apache.iotdb.db.mpp.common.header.DatasetHeaderFactory;
import org.apache.iotdb.db.mpp.plan.analyze.QueryType;
+import org.apache.iotdb.db.mpp.plan.execution.config.ConfigTaskResult;
import org.apache.iotdb.db.mpp.plan.statement.IConfigStatement;
import org.apache.iotdb.db.mpp.plan.statement.StatementVisitor;
+import org.apache.iotdb.rpc.TSStatusCode;
+import org.apache.iotdb.tsfile.file.metadata.enums.TSDataType;
+import org.apache.iotdb.tsfile.read.common.block.TsBlockBuilder;
+import org.apache.iotdb.tsfile.utils.Binary;
+
+import com.google.common.util.concurrent.SettableFuture;
import java.util.Collections;
import java.util.List;
+import java.util.Map;
+import java.util.stream.Collectors;
/**
* SHOW DATABASES statement
@@ -37,16 +51,68 @@ import java.util.List;
public class ShowStorageGroupStatement extends ShowStatement implements IConfigStatement {
private final PartialPath pathPattern;
+ private boolean isDetailed;
public ShowStorageGroupStatement(PartialPath pathPattern) {
super();
this.pathPattern = pathPattern;
+ this.isDetailed = false;
}
public PartialPath getPathPattern() {
return pathPattern;
}
+ public boolean isDetailed() {
+ return isDetailed;
+ }
+
+ public void setDetailed(boolean detailed) {
+ isDetailed = detailed;
+ }
+
+ public void buildTSBlock(
+ Map<String, TStorageGroupInfo> storageGroupInfoMap, SettableFuture<ConfigTaskResult> future) {
+
+ List<TSDataType> outputDataTypes =
+ isDetailed
+ ? ColumnHeaderConstant.showStorageGroupsDetailColumnHeaders.stream()
+ .map(ColumnHeader::getColumnType)
+ .collect(Collectors.toList())
+ : ColumnHeaderConstant.showStorageGroupsColumnHeaders.stream()
+ .map(ColumnHeader::getColumnType)
+ .collect(Collectors.toList());
+
+ TsBlockBuilder builder = new TsBlockBuilder(outputDataTypes);
+ for (Map.Entry<String, TStorageGroupInfo> entry : storageGroupInfoMap.entrySet()) {
+ String storageGroup = entry.getKey();
+ TStorageGroupInfo storageGroupInfo = entry.getValue();
+ builder.getTimeColumnBuilder().writeLong(0L);
+ builder.getColumnBuilder(0).writeBinary(new Binary(storageGroup));
+
+ if (Long.MAX_VALUE == storageGroupInfo.getTTL()) {
+ builder.getColumnBuilder(1).appendNull();
+ } else {
+ builder.getColumnBuilder(1).writeLong(storageGroupInfo.getTTL());
+ }
+ builder.getColumnBuilder(2).writeInt(storageGroupInfo.getSchemaReplicationFactor());
+ builder.getColumnBuilder(3).writeInt(storageGroupInfo.getDataReplicationFactor());
+ builder.getColumnBuilder(4).writeLong(storageGroupInfo.getTimePartitionInterval());
+ if (isDetailed) {
+ builder.getColumnBuilder(5).writeInt(storageGroupInfo.getSchemaRegionNum());
+ builder.getColumnBuilder(6).writeInt(storageGroupInfo.getMinSchemaRegionNum());
+ builder.getColumnBuilder(7).writeInt(storageGroupInfo.getMaxSchemaRegionNum());
+ builder.getColumnBuilder(8).writeInt(storageGroupInfo.getDataRegionNum());
+ builder.getColumnBuilder(9).writeInt(storageGroupInfo.getMinDataRegionNum());
+ builder.getColumnBuilder(10).writeInt(storageGroupInfo.getMaxDataRegionNum());
+ }
+ builder.declarePosition();
+ }
+
+ DatasetHeader datasetHeader = DatasetHeaderFactory.getShowStorageGroupHeader(isDetailed);
+ future.set(new ConfigTaskResult(TSStatusCode.SUCCESS_STATUS, builder.build(), datasetHeader));
+ }
+
@Override
public <R, C> R accept(StatementVisitor<R, C> visitor, C context) {
return visitor.visitShowStorageGroup(this, context);
diff --git a/service-rpc/src/main/java/org/apache/iotdb/rpc/TSStatusCode.java b/service-rpc/src/main/java/org/apache/iotdb/rpc/TSStatusCode.java
index d7a4d0feb3..4c5519c19b 100644
--- a/service-rpc/src/main/java/org/apache/iotdb/rpc/TSStatusCode.java
+++ b/service-rpc/src/main/java/org/apache/iotdb/rpc/TSStatusCode.java
@@ -76,6 +76,7 @@ public enum TSStatusCode {
OVERSIZE_RECORD(522),
SCHEMA_FILE_REDO_LOG_BROKEN(523),
TEMPLATE_NOT_ACTIVATED(524),
+ DATABASE_CONFIG_ERROR(525),
// Storage Engine
SYSTEM_READ_ONLY(600),
diff --git a/thrift-confignode/src/main/thrift/confignode.thrift b/thrift-confignode/src/main/thrift/confignode.thrift
index b8dd2ee758..425402e3b2 100644
--- a/thrift-confignode/src/main/thrift/confignode.thrift
+++ b/thrift-confignode/src/main/thrift/confignode.thrift
@@ -190,8 +190,10 @@ struct TStorageGroupSchema {
3: optional i32 schemaReplicationFactor
4: optional i32 dataReplicationFactor
5: optional i64 timePartitionInterval
- 6: optional i32 maxSchemaRegionGroupNum
- 7: optional i32 maxDataRegionGroupNum
+ 6: optional i32 minSchemaRegionGroupNum
+ 7: optional i32 maxSchemaRegionGroupNum
+ 8: optional i32 minDataRegionGroupNum
+ 9: optional i32 maxDataRegionGroupNum
}
// Schema
@@ -339,10 +341,9 @@ struct TClusterParameters {
9: required string readConsistencyLevel
10: required double schemaRegionPerDataNode
11: required double dataRegionPerProcessor
- 12: required i32 leastDataRegionGroupNum
- 13: required i32 seriesPartitionSlotNum
- 14: required string seriesPartitionExecutorClass
- 15: required double diskSpaceWarningThreshold
+ 12: required i32 seriesPartitionSlotNum
+ 13: required string seriesPartitionExecutorClass
+ 14: required double diskSpaceWarningThreshold
}
struct TConfigNodeRegisterReq {
@@ -493,7 +494,11 @@ struct TStorageGroupInfo {
4: required i32 dataReplicationFactor
5: required i64 timePartitionInterval
6: required i32 schemaRegionNum
- 7: required i32 dataRegionNum
+ 7: required i32 minSchemaRegionNum
+ 8: required i32 maxSchemaRegionNum
+ 9: required i32 dataRegionNum
+ 10: required i32 minDataRegionNum
+ 11: required i32 maxDataRegionNum
}
struct TShowStorageGroupResp {