diff --git a/features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/exception/metadata/DuplicateSharingActualDataNodeException.java b/features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/exception/metadata/DuplicateShardingActualDataNodeException.java similarity index 86% rename from features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/exception/metadata/DuplicateSharingActualDataNodeException.java rename to features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/exception/metadata/DuplicateShardingActualDataNodeException.java index 9ae794b0c17044..99a78581fba816 100644 --- a/features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/exception/metadata/DuplicateSharingActualDataNodeException.java +++ b/features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/exception/metadata/DuplicateShardingActualDataNodeException.java @@ -23,11 +23,11 @@ /** * Duplicate sharding actual data node exception. */ -public final class DuplicateSharingActualDataNodeException extends ShardingSQLException { +public final class DuplicateShardingActualDataNodeException extends ShardingSQLException { private static final long serialVersionUID = 3503761639898230998L; - public DuplicateSharingActualDataNodeException(final String logicalTableName, final String dataSourceName, final String tableName) { + public DuplicateShardingActualDataNodeException(final String logicalTableName, final String dataSourceName, final String tableName) { super(XOpenSQLState.DUPLICATE, 12, "Same actual data node cannot be configured in multiple logic tables in same database, logical table '%s', actual data node '%s.%s'.", logicalTableName, dataSourceName, tableName); } diff --git a/features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/rule/ShardingRule.java b/features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/rule/ShardingRule.java index 1ded6e4ba4e417..d1ae60b32ef361 100644 --- a/features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/rule/ShardingRule.java +++ b/features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/rule/ShardingRule.java @@ -49,11 +49,10 @@ import org.apache.shardingsphere.sharding.api.config.strategy.sharding.StandardShardingStrategyConfiguration; import org.apache.shardingsphere.sharding.api.sharding.ShardingAutoTableAlgorithm; import org.apache.shardingsphere.sharding.cache.ShardingCache; -import org.apache.shardingsphere.sharding.exception.metadata.DuplicateSharingActualDataNodeException; -import org.apache.shardingsphere.sharding.exception.metadata.InvalidBindingTablesException; import org.apache.shardingsphere.sharding.exception.metadata.ShardingTableRuleNotFoundException; import org.apache.shardingsphere.sharding.rule.attribute.ShardingDataNodeRuleAttribute; import org.apache.shardingsphere.sharding.rule.attribute.ShardingTableNamesRuleAttribute; +import org.apache.shardingsphere.sharding.rule.checker.ShardingRuleChecker; import org.apache.shardingsphere.sharding.spi.ShardingAlgorithm; import org.apache.shardingsphere.sharding.spi.ShardingAuditAlgorithm; import org.apache.shardingsphere.sql.parser.sql.common.segment.dml.column.ColumnSegment; @@ -68,7 +67,6 @@ import java.util.Collection; import java.util.Collections; import java.util.HashSet; -import java.util.Iterator; import java.util.LinkedHashMap; import java.util.LinkedHashSet; import java.util.LinkedList; @@ -76,7 +74,6 @@ import java.util.Map; import java.util.Objects; import java.util.Optional; -import java.util.Set; import java.util.TreeSet; import java.util.function.Function; import java.util.stream.Collectors; @@ -87,8 +84,6 @@ @Getter public final class ShardingRule implements DatabaseRule { - private static final String ALGORITHM_EXPRESSION_KEY = "algorithm-expression"; - private final ShardingRuleConfiguration configuration; private final Collection dataSourceNames; @@ -117,15 +112,16 @@ public final class ShardingRule implements DatabaseRule { private final RuleAttributes attributes; + private final ShardingRuleChecker shardingRuleChecker = new ShardingRuleChecker(this); + public ShardingRule(final ShardingRuleConfiguration ruleConfig, final Map dataSources, final InstanceContext instanceContext) { configuration = ruleConfig; - this.dataSourceNames = getDataSourceNames(ruleConfig.getTables(), ruleConfig.getAutoTables(), dataSources.keySet()); + dataSourceNames = getDataSourceNames(ruleConfig.getTables(), ruleConfig.getAutoTables(), dataSources.keySet()); ruleConfig.getShardingAlgorithms().forEach((key, value) -> shardingAlgorithms.put(key, TypedSPILoader.getService(ShardingAlgorithm.class, value.getType(), value.getProps()))); ruleConfig.getKeyGenerators().forEach((key, value) -> keyGenerators.put(key, TypedSPILoader.getService(KeyGenerateAlgorithm.class, value.getType(), value.getProps()))); ruleConfig.getAuditors().forEach((key, value) -> auditors.put(key, TypedSPILoader.getService(ShardingAuditAlgorithm.class, value.getType(), value.getProps()))); shardingTables.putAll(createShardingTables(ruleConfig.getTables(), ruleConfig.getDefaultKeyGenerateStrategy())); shardingTables.putAll(createShardingAutoTables(ruleConfig.getAutoTables(), ruleConfig.getDefaultKeyGenerateStrategy())); - validateUniqueActualDataNodesInTableRules(); bindingTableRules.putAll(createBindingTableRules(ruleConfig.getBindingTableGroups())); defaultDatabaseShardingStrategyConfig = createDefaultDatabaseShardingStrategyConfiguration(ruleConfig); defaultTableShardingStrategyConfig = createDefaultTableShardingStrategyConfiguration(ruleConfig); @@ -134,25 +130,13 @@ public ShardingRule(final ShardingRuleConfiguration ruleConfig, final Map ((InstanceContextAware) each).setInstanceContext(instanceContext)); if (defaultKeyGenerateAlgorithm instanceof InstanceContextAware && -1 == instanceContext.getWorkerId()) { ((InstanceContextAware) defaultKeyGenerateAlgorithm).setInstanceContext(instanceContext); } shardingCache = null == ruleConfig.getShardingCache() ? null : new ShardingCache(ruleConfig.getShardingCache(), this); attributes = new RuleAttributes(new ShardingDataNodeRuleAttribute(shardingTables), new ShardingTableNamesRuleAttribute(shardingTables.values())); - } - - private void validateUniqueActualDataNodesInTableRules() { - Set uniqueActualDataNodes = new HashSet<>(shardingTables.size(), 1L); - shardingTables.forEach((key, value) -> { - DataNode sampleActualDataNode = value.getActualDataNodes().iterator().next(); - ShardingSpherePreconditions.checkState(!uniqueActualDataNodes.contains(sampleActualDataNode), - () -> new DuplicateSharingActualDataNodeException(key, sampleActualDataNode.getDataSourceName(), sampleActualDataNode.getTableName())); - uniqueActualDataNodes.add(sampleActualDataNode); - }); + shardingRuleChecker.check(ruleConfig); } private ShardingStrategyConfiguration createDefaultDatabaseShardingStrategyConfiguration(final ShardingRuleConfiguration ruleConfig) { @@ -248,71 +232,6 @@ private BindingTableRule createBindingTableRule(final String bindingTableGroup) return result; } - private boolean isValidBindingTableConfiguration(final Map shardingTables, final BindingTableCheckedConfiguration checkedConfig) { - for (ShardingTableReferenceRuleConfiguration each : checkedConfig.getBindingTableGroups()) { - Collection bindingTables = Splitter.on(",").trimResults().splitToList(each.getReference()); - if (bindingTables.size() <= 1) { - continue; - } - Iterator iterator = bindingTables.iterator(); - ShardingTable sampleShardingTable = getShardingTable(iterator.next(), shardingTables); - while (iterator.hasNext()) { - ShardingTable shardingTable = getShardingTable(iterator.next(), shardingTables); - if (!isValidActualDataSourceName(sampleShardingTable, shardingTable) || !isValidActualTableName(sampleShardingTable, shardingTable)) { - return false; - } - if (!isBindingShardingAlgorithm(sampleShardingTable, shardingTable, true, checkedConfig) || !isBindingShardingAlgorithm(sampleShardingTable, shardingTable, false, checkedConfig)) { - return false; - } - } - } - return true; - } - - private boolean isValidActualDataSourceName(final ShardingTable sampleShardingTable, final ShardingTable shardingTable) { - return sampleShardingTable.getActualDataSourceNames().equals(shardingTable.getActualDataSourceNames()); - } - - private boolean isValidActualTableName(final ShardingTable sampleShardingTable, final ShardingTable shardingTable) { - for (String each : sampleShardingTable.getActualDataSourceNames()) { - Collection sampleActualTableNames = - sampleShardingTable.getActualTableNames(each).stream().map(actualTableName -> actualTableName.replace(sampleShardingTable.getTableDataNode().getPrefix(), "")) - .collect(Collectors.toSet()); - Collection actualTableNames = - shardingTable.getActualTableNames(each).stream().map(optional -> optional.replace(shardingTable.getTableDataNode().getPrefix(), "")).collect(Collectors.toSet()); - if (!sampleActualTableNames.equals(actualTableNames)) { - return false; - } - } - return true; - } - - private boolean isBindingShardingAlgorithm(final ShardingTable sampleShardingTable, final ShardingTable shardingTable, final boolean databaseAlgorithm, - final BindingTableCheckedConfiguration checkedConfig) { - return getAlgorithmExpression(sampleShardingTable, databaseAlgorithm, checkedConfig).equals(getAlgorithmExpression(shardingTable, databaseAlgorithm, checkedConfig)); - } - - private Optional getAlgorithmExpression(final ShardingTable shardingTable, final boolean databaseAlgorithm, final BindingTableCheckedConfiguration checkedConfig) { - ShardingStrategyConfiguration shardingStrategyConfig = databaseAlgorithm - ? getDatabaseShardingStrategyConfiguration(shardingTable, checkedConfig.getDefaultDatabaseShardingStrategyConfig()) - : getTableShardingStrategyConfiguration(shardingTable, checkedConfig.getDefaultTableShardingStrategyConfig()); - ShardingAlgorithm shardingAlgorithm = checkedConfig.getShardingAlgorithms().get(shardingStrategyConfig.getShardingAlgorithmName()); - String dataNodePrefix = databaseAlgorithm ? shardingTable.getDataSourceDataNode().getPrefix() : shardingTable.getTableDataNode().getPrefix(); - String shardingColumn = getShardingColumn(shardingStrategyConfig, checkedConfig.getDefaultShardingColumn()); - return null == shardingAlgorithm ? Optional.empty() : shardingAlgorithm.getAlgorithmStructure(dataNodePrefix, shardingColumn); - } - - private String getShardingColumn(final ShardingStrategyConfiguration shardingStrategyConfig, final String defaultShardingColumn) { - String shardingColumn = defaultShardingColumn; - if (shardingStrategyConfig instanceof ComplexShardingStrategyConfiguration) { - shardingColumn = ((ComplexShardingStrategyConfiguration) shardingStrategyConfig).getShardingColumns(); - } - if (shardingStrategyConfig instanceof StandardShardingStrategyConfiguration) { - shardingColumn = ((StandardShardingStrategyConfiguration) shardingStrategyConfig).getShardingColumn(); - } - return null == shardingColumn ? "" : shardingColumn; - } - /** * Get database sharding strategy configuration. * @@ -387,19 +306,7 @@ public Optional findShardingTableByActualTable(final String actua * @throws ShardingTableRuleNotFoundException sharding table rule not found exception */ public ShardingTable getShardingTable(final String logicTableName) { - Optional shardingTable = findShardingTable(logicTableName); - if (shardingTable.isPresent()) { - return shardingTable.get(); - } - throw new ShardingTableRuleNotFoundException(Collections.singleton(logicTableName)); - } - - private ShardingTable getShardingTable(final String logicTableName, final Map shardingTables) { - ShardingTable result = shardingTables.get(logicTableName); - if (null != result) { - return result; - } - throw new ShardingTableRuleNotFoundException(Collections.singleton(logicTableName)); + return findShardingTable(logicTableName).orElseThrow(() -> new ShardingTableRuleNotFoundException(Collections.singleton(logicTableName))); } /** @@ -437,8 +344,8 @@ public boolean isAllBindingTables(final ShardingSphereDatabase database, final S return false; } String defaultSchemaName = new DatabaseTypeRegistry(sqlStatementContext.getDatabaseType()).getDefaultSchemaName(database.getName()); - ShardingSphereSchema schema = sqlStatementContext.getTablesContext().getSchemaName().map(database::getSchema).orElseGet(() -> database.getSchema(defaultSchemaName)); SelectStatementContext select = (SelectStatementContext) sqlStatementContext; + ShardingSphereSchema schema = select.getTablesContext().getSchemaName().map(database::getSchema).orElseGet(() -> database.getSchema(defaultSchemaName)); return isJoinConditionContainsShardingColumns(schema, select, logicTableNames, select.getWhereSegments()); } @@ -555,11 +462,11 @@ private Optional findShardingColumn(final ShardingStrategyConfiguration } /** - * Judge whether given logic table column is generate key column or not. + * Judge whether given logic table column is key generated column or not. * * @param columnName column name * @param tableName table name - * @return whether given logic table column is generate key column or not + * @return whether given logic table column is key generated column or not */ public boolean isGenerateKeyColumn(final String columnName, final String tableName) { return Optional.ofNullable(shardingTables.get(tableName)).filter(each -> isGenerateKeyColumn(each, columnName)).isPresent(); @@ -593,7 +500,7 @@ public Collection> generateKeys(final AlgorithmSQLContex /** * Judge whether support auto increment or not. - * + * * @param logicTableName logic table name * @return whether support auto increment or not */ diff --git a/features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/rule/checker/ShardingRuleChecker.java b/features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/rule/checker/ShardingRuleChecker.java new file mode 100644 index 00000000000000..4713cd900a6c00 --- /dev/null +++ b/features/sharding/core/src/main/java/org/apache/shardingsphere/sharding/rule/checker/ShardingRuleChecker.java @@ -0,0 +1,198 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.shardingsphere.sharding.rule.checker; + +import com.google.common.base.Splitter; +import lombok.RequiredArgsConstructor; +import org.apache.shardingsphere.infra.algorithm.core.exception.AlgorithmInitializationException; +import org.apache.shardingsphere.infra.datanode.DataNode; +import org.apache.shardingsphere.infra.datanode.DataNodeInfo; +import org.apache.shardingsphere.infra.exception.core.ShardingSpherePreconditions; +import org.apache.shardingsphere.sharding.algorithm.sharding.inline.InlineShardingAlgorithm; +import org.apache.shardingsphere.sharding.api.config.ShardingRuleConfiguration; +import org.apache.shardingsphere.sharding.api.config.rule.ShardingTableReferenceRuleConfiguration; +import org.apache.shardingsphere.sharding.api.config.strategy.sharding.ComplexShardingStrategyConfiguration; +import org.apache.shardingsphere.sharding.api.config.strategy.sharding.ShardingStrategyConfiguration; +import org.apache.shardingsphere.sharding.api.config.strategy.sharding.StandardShardingStrategyConfiguration; +import org.apache.shardingsphere.sharding.api.sharding.standard.PreciseShardingValue; +import org.apache.shardingsphere.sharding.exception.metadata.DuplicateShardingActualDataNodeException; +import org.apache.shardingsphere.sharding.exception.metadata.InvalidBindingTablesException; +import org.apache.shardingsphere.sharding.exception.metadata.ShardingTableRuleNotFoundException; +import org.apache.shardingsphere.sharding.rule.BindingTableCheckedConfiguration; +import org.apache.shardingsphere.sharding.rule.ShardingRule; +import org.apache.shardingsphere.sharding.rule.ShardingTable; +import org.apache.shardingsphere.sharding.spi.ShardingAlgorithm; + +import java.util.Collection; +import java.util.Collections; +import java.util.HashSet; +import java.util.Iterator; +import java.util.Map; +import java.util.Optional; +import java.util.stream.Collectors; + +@RequiredArgsConstructor +public class ShardingRuleChecker { + + private final ShardingRule shardingRule; + + /** + * Check sharding rule. + * + * @param ruleConfig sharding rule configuration + */ + public void check(final ShardingRuleConfiguration ruleConfig) { + checkUniqueActualDataNodesInTableRules(); + checkBindingTableConfiguration(ruleConfig); + checkInlineShardingAlgorithmsInTableRules(); + } + + private void checkUniqueActualDataNodesInTableRules() { + Collection uniqueActualDataNodes = new HashSet<>(shardingRule.getShardingTables().size(), 1F); + shardingRule.getShardingTables().forEach((key, value) -> checkUniqueActualDataNodes(uniqueActualDataNodes, key, value.getActualDataNodes().iterator().next())); + } + + private void checkUniqueActualDataNodes(final Collection uniqueActualDataNodes, final String logicTable, final DataNode sampleActualDataNode) { + ShardingSpherePreconditions.checkNotContains(uniqueActualDataNodes, sampleActualDataNode, + () -> new DuplicateShardingActualDataNodeException(logicTable, sampleActualDataNode.getDataSourceName(), sampleActualDataNode.getTableName())); + uniqueActualDataNodes.add(sampleActualDataNode); + } + + private void checkBindingTableConfiguration(final ShardingRuleConfiguration ruleConfig) { + ShardingSpherePreconditions.checkState( + isValidBindingTableConfiguration(shardingRule.getShardingTables(), + new BindingTableCheckedConfiguration(shardingRule.getDataSourceNames(), shardingRule.getShardingAlgorithms(), ruleConfig.getBindingTableGroups(), + shardingRule.getDefaultDatabaseShardingStrategyConfig(), shardingRule.getDefaultTableShardingStrategyConfig(), shardingRule.getDefaultShardingColumn())), + InvalidBindingTablesException::new); + } + + private boolean isValidBindingTableConfiguration(final Map shardingTables, final BindingTableCheckedConfiguration checkedConfig) { + for (ShardingTableReferenceRuleConfiguration each : checkedConfig.getBindingTableGroups()) { + Collection bindingTables = Splitter.on(",").trimResults().splitToList(each.getReference()); + if (bindingTables.size() <= 1) { + continue; + } + Iterator iterator = bindingTables.iterator(); + ShardingTable sampleShardingTable = getShardingTable(iterator.next(), shardingTables); + while (iterator.hasNext()) { + ShardingTable shardingTable = getShardingTable(iterator.next(), shardingTables); + if (!isValidActualDataSourceName(sampleShardingTable, shardingTable) || !isValidActualTableName(sampleShardingTable, shardingTable)) { + return false; + } + if (!isBindingShardingAlgorithm(sampleShardingTable, shardingTable, true, checkedConfig) || !isBindingShardingAlgorithm(sampleShardingTable, shardingTable, false, checkedConfig)) { + return false; + } + } + } + return true; + } + + private ShardingTable getShardingTable(final String logicTableName, final Map shardingTables) { + ShardingTable result = shardingTables.get(logicTableName); + ShardingSpherePreconditions.checkNotNull(result, () -> new ShardingTableRuleNotFoundException(Collections.singleton(logicTableName))); + return result; + } + + private boolean isValidActualDataSourceName(final ShardingTable sampleShardingTable, final ShardingTable shardingTable) { + return sampleShardingTable.getActualDataSourceNames().equals(shardingTable.getActualDataSourceNames()); + } + + private boolean isValidActualTableName(final ShardingTable sampleShardingTable, final ShardingTable shardingTable) { + for (String each : sampleShardingTable.getActualDataSourceNames()) { + Collection sampleActualTableNames = sampleShardingTable.getActualTableNames(each).stream() + .map(actualTableName -> actualTableName.replace(sampleShardingTable.getTableDataNode().getPrefix(), "")).collect(Collectors.toSet()); + Collection actualTableNames = + shardingTable.getActualTableNames(each).stream().map(optional -> optional.replace(shardingTable.getTableDataNode().getPrefix(), "")).collect(Collectors.toSet()); + if (!sampleActualTableNames.equals(actualTableNames)) { + return false; + } + } + return true; + } + + private boolean isBindingShardingAlgorithm(final ShardingTable sampleShardingTable, final ShardingTable shardingTable, final boolean databaseAlgorithm, + final BindingTableCheckedConfiguration checkedConfig) { + return getAlgorithmExpression(sampleShardingTable, databaseAlgorithm, checkedConfig).equals(getAlgorithmExpression(shardingTable, databaseAlgorithm, checkedConfig)); + } + + private Optional getAlgorithmExpression(final ShardingTable shardingTable, final boolean databaseAlgorithm, final BindingTableCheckedConfiguration checkedConfig) { + ShardingStrategyConfiguration shardingStrategyConfig = databaseAlgorithm + ? shardingRule.getDatabaseShardingStrategyConfiguration(shardingTable) + : shardingRule.getTableShardingStrategyConfiguration(shardingTable); + ShardingAlgorithm shardingAlgorithm = checkedConfig.getShardingAlgorithms().get(shardingStrategyConfig.getShardingAlgorithmName()); + String dataNodePrefix = databaseAlgorithm ? shardingTable.getDataSourceDataNode().getPrefix() : shardingTable.getTableDataNode().getPrefix(); + String shardingColumn = getShardingColumn(shardingStrategyConfig, shardingRule.getDefaultShardingColumn()); + return null == shardingAlgorithm ? Optional.empty() : shardingAlgorithm.getAlgorithmStructure(dataNodePrefix, shardingColumn); + } + + private String getShardingColumn(final ShardingStrategyConfiguration shardingStrategyConfig, final String defaultShardingColumn) { + String shardingColumn = defaultShardingColumn; + if (shardingStrategyConfig instanceof ComplexShardingStrategyConfiguration) { + shardingColumn = ((ComplexShardingStrategyConfiguration) shardingStrategyConfig).getShardingColumns(); + } + if (shardingStrategyConfig instanceof StandardShardingStrategyConfiguration) { + shardingColumn = ((StandardShardingStrategyConfiguration) shardingStrategyConfig).getShardingColumn(); + } + return null == shardingColumn ? "" : shardingColumn; + } + + private void checkInlineShardingAlgorithmsInTableRules() { + shardingRule.getShardingTables().forEach((key, value) -> { + validateInlineShardingAlgorithm(value, shardingRule.getTableShardingStrategyConfiguration(value), value.getTableDataNode()); + validateInlineShardingAlgorithm(value, shardingRule.getDatabaseShardingStrategyConfiguration(value), value.getDataSourceDataNode()); + }); + } + + private void validateInlineShardingAlgorithm(final ShardingTable shardingTable, final ShardingStrategyConfiguration shardingStrategy, final DataNodeInfo dataNodeInfo) { + if (null == shardingStrategy) { + return; + } + ShardingAlgorithm shardingAlgorithm = shardingRule.getShardingAlgorithms().get(shardingStrategy.getShardingAlgorithmName()); + if (shardingAlgorithm instanceof InlineShardingAlgorithm) { + String shardingColumn = null == ((StandardShardingStrategyConfiguration) shardingStrategy).getShardingColumn() ? shardingRule.getDefaultShardingColumn() + : ((StandardShardingStrategyConfiguration) shardingStrategy).getShardingColumn(); + String result = null; + try { + result = ((InlineShardingAlgorithm) shardingAlgorithm).doSharding(Collections.emptySet(), new PreciseShardingValue<>(shardingTable.getLogicTable(), shardingColumn, dataNodeInfo, 1)); + // CHECKSTYLE:OFF + } catch (final Exception ignored) { + // CHECKSTYLE:ON + } + ShardingSpherePreconditions.checkState(null == result || result.startsWith(dataNodeInfo.getPrefix()), + () -> new AlgorithmInitializationException(shardingAlgorithm, "`%s` sharding algorithm configuration of `%s` does not match the actual data nodes", + shardingStrategy.getShardingAlgorithmName(), shardingTable.getLogicTable())); + } + } + + /** + * Check to be added data nodes. + * + * @param toBeAddedDataNodes to be added data nodes + * @param isAlteration is alteration + */ + public void checkToBeAddedDataNodes(final Map> toBeAddedDataNodes, final boolean isAlteration) { + Collection uniqueActualDataNodes = new HashSet<>(shardingRule.getShardingTables().size() + toBeAddedDataNodes.size(), 1F); + shardingRule.getShardingTables().forEach((key, value) -> { + if (isAlteration && toBeAddedDataNodes.containsKey(key)) { + return; + } + checkUniqueActualDataNodes(uniqueActualDataNodes, key, value.getActualDataNodes().iterator().next()); + }); + toBeAddedDataNodes.forEach((key, value) -> checkUniqueActualDataNodes(uniqueActualDataNodes, key, value.iterator().next())); + } +} diff --git a/features/sharding/core/src/test/java/org/apache/shardingsphere/sharding/cache/checker/ShardingRouteCacheableCheckerTest.java b/features/sharding/core/src/test/java/org/apache/shardingsphere/sharding/cache/checker/ShardingRouteCacheableCheckerTest.java index 1807c5b28b5860..8a8dd05f4c37f2 100644 --- a/features/sharding/core/src/test/java/org/apache/shardingsphere/sharding/cache/checker/ShardingRouteCacheableCheckerTest.java +++ b/features/sharding/core/src/test/java/org/apache/shardingsphere/sharding/cache/checker/ShardingRouteCacheableCheckerTest.java @@ -92,6 +92,8 @@ private ShardingRule createShardingRule() { ruleConfig.getBindingTableGroups().add(new ShardingTableReferenceRuleConfiguration("foo", "t_order,t_order_item")); ruleConfig.getShardingAlgorithms().put("mod", new AlgorithmConfiguration("MOD", PropertiesBuilder.build(new Property("sharding-count", "2")))); ruleConfig.getShardingAlgorithms().put("inline", new AlgorithmConfiguration("INLINE", PropertiesBuilder.build(new Property("algorithm-expression", "ds_${id % 2}")))); + ruleConfig.getShardingAlgorithms().put("table-inline", + new AlgorithmConfiguration("INLINE", PropertiesBuilder.build(new Property("algorithm-expression", "t_non_cacheable_table_sharding_${id % 2}")))); ruleConfig.setDefaultDatabaseShardingStrategy(new StandardShardingStrategyConfiguration("warehouse_id", "inline")); ShardingAutoTableRuleConfiguration warehouse = new ShardingAutoTableRuleConfiguration("t_warehouse", "ds_${0..1}"); warehouse.setShardingStrategy(new StandardShardingStrategyConfiguration("id", "mod")); @@ -102,7 +104,7 @@ private ShardingRule createShardingRule() { nonCacheableDatabaseSharding.setDatabaseShardingStrategy(new StandardShardingStrategyConfiguration("id", "inline")); ruleConfig.getTables().add(nonCacheableDatabaseSharding); ShardingTableRuleConfiguration nonCacheableTableSharding = new ShardingTableRuleConfiguration("t_non_cacheable_table_sharding", "ds_0.t_non_cacheable_table_sharding_${0..1}"); - nonCacheableTableSharding.setTableShardingStrategy(new StandardShardingStrategyConfiguration("id", "inline")); + nonCacheableTableSharding.setTableShardingStrategy(new StandardShardingStrategyConfiguration("id", "table-inline")); ruleConfig.getTables().add(nonCacheableTableSharding); ruleConfig.setShardingCache(new ShardingCacheConfiguration(100, new ShardingCacheOptionsConfiguration(true, 0, 0))); return new ShardingRule(ruleConfig, Maps.of("ds_0", new MockedDataSource(), "ds_1", new MockedDataSource()), diff --git a/features/sharding/core/src/test/java/org/apache/shardingsphere/sharding/rule/ShardingRuleTest.java b/features/sharding/core/src/test/java/org/apache/shardingsphere/sharding/rule/ShardingRuleTest.java index 32f22b4bad1c43..55fc20d048a748 100644 --- a/features/sharding/core/src/test/java/org/apache/shardingsphere/sharding/rule/ShardingRuleTest.java +++ b/features/sharding/core/src/test/java/org/apache/shardingsphere/sharding/rule/ShardingRuleTest.java @@ -46,7 +46,7 @@ import org.apache.shardingsphere.sharding.api.config.strategy.sharding.NoneShardingStrategyConfiguration; import org.apache.shardingsphere.sharding.api.config.strategy.sharding.ShardingStrategyConfiguration; import org.apache.shardingsphere.sharding.api.config.strategy.sharding.StandardShardingStrategyConfiguration; -import org.apache.shardingsphere.sharding.exception.metadata.DuplicateSharingActualDataNodeException; +import org.apache.shardingsphere.sharding.exception.metadata.DuplicateShardingActualDataNodeException; import org.apache.shardingsphere.sharding.exception.metadata.InvalidBindingTablesException; import org.apache.shardingsphere.sharding.exception.metadata.ShardingTableRuleNotFoundException; import org.apache.shardingsphere.sql.parser.sql.common.segment.dml.column.ColumnSegment; @@ -171,7 +171,45 @@ void assertNewShardingRuleWithDuplicateActualDataNodeInTableRules() { duplicateTableRuleConfig.setTableShardingStrategy(new StandardShardingStrategyConfiguration("order_id", "MOD")); ruleConfig.getTables().add(duplicateTableRuleConfig); ruleConfig.getShardingAlgorithms().put("INLINE", new AlgorithmConfiguration("INLINE", PropertiesBuilder.build(new Property("algorithm-expression", "t_order_%{order_id % 2}")))); - assertThrows(DuplicateSharingActualDataNodeException.class, () -> new ShardingRule(ruleConfig, Collections.emptyMap(), mock(InstanceContext.class))); + assertThrows(DuplicateShardingActualDataNodeException.class, () -> new ShardingRule(ruleConfig, Collections.emptyMap(), mock(InstanceContext.class))); + } + + @Test + void assertValidateWrongInlineShardingAlgorithmsInTableRules() { + ShardingRuleConfiguration ruleConfig = new ShardingRuleConfiguration(); + ShardingTableRuleConfiguration tableRuleConfig = new ShardingTableRuleConfiguration("t_order", "ds_${0..1}.t_order_tmp_${0..1}"); + tableRuleConfig.setTableShardingStrategy(new StandardShardingStrategyConfiguration("order_id", "order_id_inline")); + ruleConfig.getTables().add(tableRuleConfig); + ruleConfig.getShardingAlgorithms().put("order_id_inline", new AlgorithmConfiguration("INLINE", PropertiesBuilder.build(new Property("algorithm-expression", "t_order_${order_id % 2}")))); + assertThrows(AlgorithmInitializationException.class, () -> new ShardingRule(ruleConfig, Collections.emptyMap(), mock(InstanceContext.class))); + } + + @Test + void assertValidateInlineShardingAlgorithmsInTableRules() { + ShardingRuleConfiguration ruleConfig = new ShardingRuleConfiguration(); + ShardingTableRuleConfiguration tableRuleConfig = new ShardingTableRuleConfiguration("t_order", "ds_${0..1}.t_order_${0..1}"); + tableRuleConfig.setTableShardingStrategy(new StandardShardingStrategyConfiguration("order_id", "order_id_inline")); + tableRuleConfig.setDatabaseShardingStrategy(new StandardShardingStrategyConfiguration("user_id", "user_id_inline")); + ruleConfig.getTables().add(tableRuleConfig); + ShardingTableRuleConfiguration versionTableRuleConfig = new ShardingTableRuleConfiguration("t_order", "ds_${0..1}.v2_t_order_${0..1}"); + versionTableRuleConfig.setTableShardingStrategy(new StandardShardingStrategyConfiguration("order_id", "order_id_inline")); + versionTableRuleConfig.setDatabaseShardingStrategy(new StandardShardingStrategyConfiguration("user_id", "user_id_inline")); + ruleConfig.getTables().add(versionTableRuleConfig); + ruleConfig.getShardingAlgorithms().put("order_id_inline", new AlgorithmConfiguration("INLINE", PropertiesBuilder.build(new Property("algorithm-expression", "t_order_${order_id % 2}")))); + ruleConfig.getShardingAlgorithms().put("user_id_inline", new AlgorithmConfiguration("INLINE", PropertiesBuilder.build(new Property("algorithm-expression", "ds_${user_id % 2}")))); + new ShardingRule(ruleConfig, Collections.emptyMap(), mock(InstanceContext.class, RETURNS_DEEP_STUBS)); + } + + @Test + void assertValidateInlineShardingAlgorithmsIgnoreExceptionInTableRules() { + ShardingRuleConfiguration ruleConfig = new ShardingRuleConfiguration(); + ShardingTableRuleConfiguration tableRuleConfig = + new ShardingTableRuleConfiguration("t_order_interval", "ds_${0..1}.t_order_interval_${(2018..2050)}${(01..12).collect{m ->m.toString().padLeft(2, '0')}}"); + tableRuleConfig.setTableShardingStrategy(new StandardShardingStrategyConfiguration("order_id", "t_order_interval_inline")); + ruleConfig.getTables().add(tableRuleConfig); + ruleConfig.getShardingAlgorithms().put("t_order_interval_inline", + new AlgorithmConfiguration("INLINE", PropertiesBuilder.build(new Property("algorithm-expression", "t_order_interval_${order_id.substring(0, 6)}")))); + new ShardingRule(ruleConfig, Collections.emptyMap(), mock(InstanceContext.class, RETURNS_DEEP_STUBS)); } @Test diff --git a/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/checker/ShardingTableRuleStatementChecker.java b/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/checker/ShardingTableRuleStatementChecker.java index 7d46e04056abbe..3e9534db76b8bf 100644 --- a/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/checker/ShardingTableRuleStatementChecker.java +++ b/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/checker/ShardingTableRuleStatementChecker.java @@ -27,10 +27,10 @@ import org.apache.shardingsphere.infra.algorithm.keygen.core.KeyGenerateAlgorithm; import org.apache.shardingsphere.infra.datanode.DataNode; import org.apache.shardingsphere.infra.exception.core.ShardingSpherePreconditions; +import org.apache.shardingsphere.infra.exception.kernel.metadata.resource.storageunit.MissingRequiredStorageUnitsException; import org.apache.shardingsphere.infra.exception.kernel.metadata.rule.DuplicateRuleException; import org.apache.shardingsphere.infra.exception.kernel.metadata.rule.InvalidRuleConfigurationException; import org.apache.shardingsphere.infra.exception.kernel.metadata.rule.MissingRequiredRuleException; -import org.apache.shardingsphere.infra.exception.kernel.metadata.resource.storageunit.MissingRequiredStorageUnitsException; import org.apache.shardingsphere.infra.expr.core.InlineExpressionParserFactory; import org.apache.shardingsphere.infra.metadata.database.ShardingSphereDatabase; import org.apache.shardingsphere.infra.rule.attribute.datasource.DataSourceMapperRuleAttribute; @@ -55,6 +55,7 @@ import org.apache.shardingsphere.sharding.distsql.segment.table.AutoTableRuleSegment; import org.apache.shardingsphere.sharding.distsql.segment.table.TableRuleSegment; import org.apache.shardingsphere.sharding.exception.algorithm.ShardingAlgorithmClassImplementationException; +import org.apache.shardingsphere.sharding.exception.metadata.DuplicateShardingActualDataNodeException; import org.apache.shardingsphere.sharding.exception.metadata.ShardingTableRuleNotFoundException; import org.apache.shardingsphere.sharding.exception.strategy.InvalidShardingStrategyConfigurationException; import org.apache.shardingsphere.sharding.rule.BindingTableCheckedConfiguration; @@ -87,7 +88,7 @@ public final class ShardingTableRuleStatementChecker { private static final String DELIMITER = "."; /** - * Check create sharing table rule statement. + * Check create sharding table rule statement. * * @param database database * @param rules rules @@ -100,7 +101,7 @@ public static void checkCreation(final ShardingSphereDatabase database, final Co } /** - * Check alter sharing table rule statement. + * Check alter sharding table rule statement. * * @param database database * @param rules rules @@ -191,7 +192,7 @@ private static Collection getDataSourceNames(final ShardingTableRuleConf } private static Collection getDataSourceNames(final Collection actualDataNodes) { - Collection result = new LinkedHashSet<>(); + Collection result = new LinkedHashSet<>(actualDataNodes.size(), 1F); for (String each : actualDataNodes) { result.add(isValidDataNode(each) ? new DataNode(each).getDataSourceName() : each); } @@ -314,7 +315,7 @@ private static void checkDataSources(final String databaseName, final Collection Collection notExistedDataSources = database.getResourceMetaData().getNotExistedDataSources(requiredDataSource); Collection logicDataSources = getLogicDataSources(database); notExistedDataSources.removeIf(logicDataSources::contains); - ShardingSpherePreconditions.checkState(notExistedDataSources.isEmpty(), () -> new MissingRequiredStorageUnitsException(databaseName, notExistedDataSources)); + ShardingSpherePreconditions.checkMustEmpty(notExistedDataSources, () -> new MissingRequiredStorageUnitsException(databaseName, notExistedDataSources)); } private static Collection getRequiredDataSources(final ShardingRuleConfiguration config) { @@ -347,22 +348,22 @@ private static void checkTables(final String databaseName, final Collection requiredTables = rules.stream().map(AbstractTableRuleSegment::getLogicTable).collect(Collectors.toList()); Collection duplicatedRuleNames = getDuplicatedRuleNames(requiredTables); - ShardingSpherePreconditions.checkState(duplicatedRuleNames.isEmpty(), () -> new DuplicateRuleException("sharding", databaseName, duplicatedRuleNames)); + ShardingSpherePreconditions.checkMustEmpty(duplicatedRuleNames, () -> new DuplicateRuleException("sharding", databaseName, duplicatedRuleNames)); Collection currentShardingTables = null == currentRuleConfig ? Collections.emptyList() : getCurrentShardingTables(currentRuleConfig); if (isCreate) { if (!ifNotExists) { duplicatedRuleNames.addAll(getDuplicatedRuleNames(requiredTables, currentShardingTables)); - ShardingSpherePreconditions.checkState(duplicatedRuleNames.isEmpty(), () -> new DuplicateRuleException("sharding", databaseName, duplicatedRuleNames)); + ShardingSpherePreconditions.checkMustEmpty(duplicatedRuleNames, () -> new DuplicateRuleException("sharding", databaseName, duplicatedRuleNames)); } } else { Collection notExistedRules = getNotExistedRules(requiredTables, currentShardingTables); - ShardingSpherePreconditions.checkState(notExistedRules.isEmpty(), () -> new MissingRequiredRuleException("sharding", databaseName, notExistedRules)); + ShardingSpherePreconditions.checkMustEmpty(notExistedRules, () -> new MissingRequiredRuleException("sharding", databaseName, notExistedRules)); } } private static Collection getDuplicatedRuleNames(final Collection collection) { Collection duplicatedNames = collection.stream().collect(Collectors.groupingBy(String::toLowerCase, Collectors.counting())).entrySet().stream() - .filter(each -> each.getValue() > 1).map(Entry::getKey).collect(Collectors.toCollection(CaseInsensitiveSet::new)); + .filter(each -> each.getValue() > 1L).map(Entry::getKey).collect(Collectors.toCollection(CaseInsensitiveSet::new)); return collection.stream().filter(duplicatedNames::contains).collect(Collectors.toSet()); } @@ -469,7 +470,7 @@ private static Collection getRequiredTables(final Collection getTableNames(final Collection actualDataNodes) { - Collection result = new HashSet<>(); + Collection result = new HashSet<>(actualDataNodes.size(), 1F); for (String each : actualDataNodes) { result.add(isValidDataNode(each) ? new DataNode(each).getTableName() : each); } @@ -525,4 +526,19 @@ private static Collection getAlteredLogicalTableNames(final ShardingRule result.addAll(toBeAlteredRuleConfig.getAutoTables().stream().map(ShardingAutoTableRuleConfiguration::getLogicTable).collect(Collectors.toList())); return result; } + + /** + * Check to be added data nodes. + * + * @param toBeAddedDataNodes to be added data nodes + */ + public static void checkToBeAddedDataNodes(final Map> toBeAddedDataNodes) { + Collection uniqueActualDataNodes = new HashSet<>(toBeAddedDataNodes.size(), 1F); + toBeAddedDataNodes.forEach((key, value) -> { + DataNode sampleActualDataNode = value.iterator().next(); + ShardingSpherePreconditions.checkNotContains(uniqueActualDataNodes, sampleActualDataNode, + () -> new DuplicateShardingActualDataNodeException(key, sampleActualDataNode.getDataSourceName(), sampleActualDataNode.getTableName())); + uniqueActualDataNodes.add(sampleActualDataNode); + }); + } } diff --git a/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/converter/ShardingTableRuleStatementConverter.java b/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/converter/ShardingTableRuleStatementConverter.java index 89822c253a6fc7..3a5502076a356d 100644 --- a/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/converter/ShardingTableRuleStatementConverter.java +++ b/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/converter/ShardingTableRuleStatementConverter.java @@ -21,6 +21,12 @@ import lombok.NoArgsConstructor; import org.apache.shardingsphere.distsql.segment.AlgorithmSegment; import org.apache.shardingsphere.infra.algorithm.core.config.AlgorithmConfiguration; +import org.apache.shardingsphere.infra.algorithm.core.exception.AlgorithmInitializationException; +import org.apache.shardingsphere.infra.datanode.DataNode; +import org.apache.shardingsphere.infra.datanode.DataNodeUtils; +import org.apache.shardingsphere.infra.exception.core.ShardingSpherePreconditions; +import org.apache.shardingsphere.infra.expr.core.InlineExpressionParserFactory; +import org.apache.shardingsphere.infra.spi.type.typed.TypedSPILoader; import org.apache.shardingsphere.sharding.api.config.ShardingRuleConfiguration; import org.apache.shardingsphere.sharding.api.config.rule.ShardingAutoTableRuleConfiguration; import org.apache.shardingsphere.sharding.api.config.rule.ShardingTableRuleConfiguration; @@ -28,6 +34,7 @@ import org.apache.shardingsphere.sharding.api.config.strategy.keygen.KeyGenerateStrategyConfiguration; import org.apache.shardingsphere.sharding.api.config.strategy.sharding.NoneShardingStrategyConfiguration; import org.apache.shardingsphere.sharding.api.config.strategy.sharding.ShardingStrategyConfiguration; +import org.apache.shardingsphere.sharding.api.sharding.ShardingAutoTableAlgorithm; import org.apache.shardingsphere.sharding.distsql.handler.enums.ShardingStrategyLevelType; import org.apache.shardingsphere.sharding.distsql.handler.enums.ShardingStrategyType; import org.apache.shardingsphere.sharding.distsql.segment.strategy.AuditStrategySegment; @@ -37,9 +44,12 @@ import org.apache.shardingsphere.sharding.distsql.segment.table.AbstractTableRuleSegment; import org.apache.shardingsphere.sharding.distsql.segment.table.AutoTableRuleSegment; import org.apache.shardingsphere.sharding.distsql.segment.table.TableRuleSegment; +import org.apache.shardingsphere.sharding.spi.ShardingAlgorithm; import java.util.Collection; import java.util.HashMap; +import java.util.LinkedList; +import java.util.List; import java.util.Map; import java.util.Optional; import java.util.stream.Collectors; @@ -195,4 +205,41 @@ private static String getTableShardingAlgorithmName(final String tableName, fina private static String getKeyGeneratorName(final String tableName, final String algorithmType) { return String.format("%s_%s", tableName, algorithmType).toLowerCase(); } + + /** + * Convert rule segments to data nodes. + * + * @param segments sharding table rule segments + * @return data nodes map + */ + public static Map> convertDataNodes(final Collection segments) { + Map> result = new HashMap<>(segments.size(), 1F); + for (AbstractTableRuleSegment each : segments) { + if (each instanceof TableRuleSegment) { + result.put(each.getLogicTable(), getActualDataNodes((TableRuleSegment) each)); + continue; + } + result.put(each.getLogicTable(), getActualDataNodes((AutoTableRuleSegment) each)); + } + return result; + } + + private static Collection getActualDataNodes(final TableRuleSegment ruleSegment) { + Collection result = new LinkedList<>(); + for (String each : ruleSegment.getDataSourceNodes()) { + List dataNodes = InlineExpressionParserFactory.newInstance(each).splitAndEvaluate(); + result.addAll(dataNodes.stream().map(DataNode::new).collect(Collectors.toList())); + } + return result; + } + + private static Collection getActualDataNodes(final AutoTableRuleSegment ruleSegment) { + ShardingAlgorithm shardingAlgorithm = + TypedSPILoader.getService(ShardingAlgorithm.class, ruleSegment.getShardingAlgorithmSegment().getName(), ruleSegment.getShardingAlgorithmSegment().getProps()); + ShardingSpherePreconditions.checkState(shardingAlgorithm instanceof ShardingAutoTableAlgorithm, + () -> new AlgorithmInitializationException(shardingAlgorithm, "Auto sharding algorithm is required for table '%s'", ruleSegment.getLogicTable())); + List dataNodes = DataNodeUtils.getFormatDataNodes(((ShardingAutoTableAlgorithm) shardingAlgorithm).getAutoTablesAmount(), + ruleSegment.getLogicTable(), ruleSegment.getDataSourceNodes()); + return dataNodes.stream().map(DataNode::new).collect(Collectors.toList()); + } } diff --git a/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/update/AlterShardingTableRuleExecutor.java b/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/update/AlterShardingTableRuleExecutor.java index 3263560a1a8131..4fef1f24a04d5b 100644 --- a/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/update/AlterShardingTableRuleExecutor.java +++ b/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/update/AlterShardingTableRuleExecutor.java @@ -46,6 +46,11 @@ public final class AlterShardingTableRuleExecutor implements DatabaseRuleAlterEx @Override public void checkBeforeUpdate(final AlterShardingTableRuleStatement sqlStatement) { ShardingTableRuleStatementChecker.checkAlteration(database, sqlStatement.getRules(), rule.getConfiguration()); + checkUniqueActualDataNodes(sqlStatement); + } + + private void checkUniqueActualDataNodes(final AlterShardingTableRuleStatement sqlStatement) { + rule.getShardingRuleChecker().checkToBeAddedDataNodes(ShardingTableRuleStatementConverter.convertDataNodes(sqlStatement.getRules()), true); } @Override diff --git a/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/update/CreateShardingTableRuleExecutor.java b/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/update/CreateShardingTableRuleExecutor.java index 7dbcdf6ceb8870..d842b3bd78b06a 100644 --- a/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/update/CreateShardingTableRuleExecutor.java +++ b/features/sharding/distsql/handler/src/main/java/org/apache/shardingsphere/sharding/distsql/handler/update/CreateShardingTableRuleExecutor.java @@ -17,6 +17,7 @@ package org.apache.shardingsphere.sharding.distsql.handler.update; +import com.cedarsoftware.util.CaseInsensitiveSet; import lombok.Setter; import org.apache.shardingsphere.distsql.handler.engine.update.rdl.rule.spi.database.DatabaseRuleCreateExecutor; import org.apache.shardingsphere.infra.metadata.database.ShardingSphereDatabase; @@ -31,7 +32,6 @@ import java.util.Collection; import java.util.Collections; -import java.util.LinkedList; import java.util.stream.Collectors; /** @@ -47,16 +47,24 @@ public final class CreateShardingTableRuleExecutor implements DatabaseRuleCreate @Override public void checkBeforeUpdate(final CreateShardingTableRuleStatement sqlStatement) { ShardingTableRuleStatementChecker.checkCreation(database, sqlStatement.getRules(), sqlStatement.isIfNotExists(), null == rule ? null : rule.getConfiguration()); + checkUniqueActualDataNodes(sqlStatement); } - @Override - public ShardingRuleConfiguration buildToBeCreatedRuleConfiguration(final CreateShardingTableRuleStatement sqlStatement) { - Collection segments = sqlStatement.getRules(); + private void checkUniqueActualDataNodes(final CreateShardingTableRuleStatement sqlStatement) { + if (null == rule) { + ShardingTableRuleStatementChecker.checkToBeAddedDataNodes(ShardingTableRuleStatementConverter.convertDataNodes(sqlStatement.getRules())); + return; + } if (sqlStatement.isIfNotExists()) { Collection duplicatedRuleNames = getDuplicatedRuleNames(sqlStatement); - segments.removeIf(each -> duplicatedRuleNames.contains(each.getLogicTable())); + sqlStatement.getRules().removeIf(each -> duplicatedRuleNames.contains(each.getLogicTable())); } - return ShardingTableRuleStatementConverter.convert(segments); + rule.getShardingRuleChecker().checkToBeAddedDataNodes(ShardingTableRuleStatementConverter.convertDataNodes(sqlStatement.getRules()), false); + } + + @Override + public ShardingRuleConfiguration buildToBeCreatedRuleConfiguration(final CreateShardingTableRuleStatement sqlStatement) { + return ShardingTableRuleStatementConverter.convert(sqlStatement.getRules()); } private Collection getDuplicatedRuleNames(final CreateShardingTableRuleStatement sqlStatement) { @@ -65,7 +73,7 @@ private Collection getDuplicatedRuleNames(final CreateShardingTableRuleS } private Collection getCurrentShardingTables() { - Collection result = new LinkedList<>(); + Collection result = new CaseInsensitiveSet<>(); result.addAll(rule.getConfiguration().getTables().stream().map(ShardingTableRuleConfiguration::getLogicTable).collect(Collectors.toSet())); result.addAll(rule.getConfiguration().getAutoTables().stream().map(ShardingAutoTableRuleConfiguration::getLogicTable).collect(Collectors.toSet())); return result; diff --git a/features/sharding/distsql/handler/src/test/java/org/apache/shardingsphere/sharding/distsql/update/AlterShardingTableRuleExecutorTest.java b/features/sharding/distsql/handler/src/test/java/org/apache/shardingsphere/sharding/distsql/update/AlterShardingTableRuleExecutorTest.java index 47714e66b55fc0..44d2bb2800ef4b 100644 --- a/features/sharding/distsql/handler/src/test/java/org/apache/shardingsphere/sharding/distsql/update/AlterShardingTableRuleExecutorTest.java +++ b/features/sharding/distsql/handler/src/test/java/org/apache/shardingsphere/sharding/distsql/update/AlterShardingTableRuleExecutorTest.java @@ -17,7 +17,9 @@ package org.apache.shardingsphere.sharding.distsql.update; +import lombok.SneakyThrows; import org.apache.shardingsphere.distsql.segment.AlgorithmSegment; +import org.apache.shardingsphere.distsql.statement.DistSQLStatement; import org.apache.shardingsphere.infra.algorithm.core.config.AlgorithmConfiguration; import org.apache.shardingsphere.infra.metadata.database.ShardingSphereDatabase; import org.apache.shardingsphere.infra.metadata.database.resource.ResourceMetaData; @@ -28,12 +30,18 @@ import org.apache.shardingsphere.sharding.api.config.strategy.keygen.KeyGenerateStrategyConfiguration; import org.apache.shardingsphere.sharding.api.config.strategy.sharding.StandardShardingStrategyConfiguration; import org.apache.shardingsphere.sharding.distsql.handler.update.AlterShardingTableRuleExecutor; +import org.apache.shardingsphere.sharding.distsql.parser.facade.ShardingDistSQLParserFacade; import org.apache.shardingsphere.sharding.distsql.segment.strategy.KeyGenerateStrategySegment; import org.apache.shardingsphere.sharding.distsql.segment.strategy.ShardingStrategySegment; import org.apache.shardingsphere.sharding.distsql.segment.table.AutoTableRuleSegment; import org.apache.shardingsphere.sharding.distsql.segment.table.TableRuleSegment; import org.apache.shardingsphere.sharding.distsql.statement.AlterShardingTableRuleStatement; +import org.apache.shardingsphere.sharding.exception.metadata.DuplicateShardingActualDataNodeException; import org.apache.shardingsphere.sharding.rule.ShardingRule; +import org.apache.shardingsphere.sharding.rule.checker.ShardingRuleChecker; +import org.apache.shardingsphere.sql.parser.api.visitor.SQLVisitor; +import org.apache.shardingsphere.sql.parser.core.ParseASTNode; +import org.apache.shardingsphere.sql.parser.core.SQLParserFactory; import org.apache.shardingsphere.test.fixture.jdbc.MockedDataSource; import org.apache.shardingsphere.test.util.PropertiesBuilder; import org.apache.shardingsphere.test.util.PropertiesBuilder.Property; @@ -50,6 +58,7 @@ import static org.hamcrest.CoreMatchers.instanceOf; import static org.hamcrest.CoreMatchers.is; import static org.hamcrest.MatcherAssert.assertThat; +import static org.junit.jupiter.api.Assertions.assertThrows; import static org.mockito.Mockito.RETURNS_DEEP_STUBS; import static org.mockito.Mockito.mock; import static org.mockito.Mockito.when; @@ -69,14 +78,29 @@ void setUp() { when(database.getResourceMetaData()).thenReturn(resourceMetaData); when(database.getRuleMetaData()).thenReturn(new RuleMetaData(Collections.emptyList())); executor.setDatabase(database); + ShardingRule rule = mock(ShardingRule.class); + when(rule.getConfiguration()).thenReturn(currentRuleConfig); + ShardingRuleChecker checker = new ShardingRuleChecker(rule); + when(rule.getShardingRuleChecker()).thenReturn(checker); + executor.setRule(rule); + } + + @Test + void assertCheckWithDuplicateDataNodes() { + String sql = "ALTER SHARDING TABLE RULE t_order(" + + "DATANODES('ds_${0..1}.t_order')," + + "DATABASE_STRATEGY(TYPE='standard',SHARDING_COLUMN=user_id,SHARDING_ALGORITHM(TYPE(NAME='inline',PROPERTIES('algorithm-expression'='ds_${user_id % 2}'))))" + + "), t_order_item(" + + "DATANODES('ds_${0..1}.t_order')," + + "DATABASE_STRATEGY(TYPE='standard',SHARDING_COLUMN=user_id,SHARDING_ALGORITHM(TYPE(NAME='inline',PROPERTIES('algorithm-expression'='ds_${user_id % 2}'))))" + + ");"; + AlterShardingTableRuleStatement sqlStatement = (AlterShardingTableRuleStatement) getDistSQLStatement(sql); + assertThrows(DuplicateShardingActualDataNodeException.class, () -> executor.checkBeforeUpdate(sqlStatement)); } @Test void assertUpdate() { AlterShardingTableRuleStatement sqlStatement = new AlterShardingTableRuleStatement(Arrays.asList(createCompleteAutoTableRule("t_order_item"), createCompleteTableRule("t_order"))); - ShardingRule rule = mock(ShardingRule.class); - when(rule.getConfiguration()).thenReturn(currentRuleConfig); - executor.setRule(rule); executor.checkBeforeUpdate(sqlStatement); ShardingRuleConfiguration toBeAlteredRuleConfig = executor.buildToBeAlteredRuleConfiguration(sqlStatement); assertThat(toBeAlteredRuleConfig.getTables().size(), is(1)); @@ -101,9 +125,6 @@ void assertUpdate() { @Test void assertUpdateWithDifferentCase() { AlterShardingTableRuleStatement sqlStatement = new AlterShardingTableRuleStatement(Arrays.asList(createCompleteAutoTableRule("T_ORDER_ITEM"), createCompleteTableRule("T_ORDER"))); - ShardingRule rule = mock(ShardingRule.class); - when(rule.getConfiguration()).thenReturn(currentRuleConfig); - executor.setRule(rule); executor.checkBeforeUpdate(sqlStatement); ShardingRuleConfiguration toBeAlteredRuleConfig = executor.buildToBeAlteredRuleConfiguration(sqlStatement); assertThat(toBeAlteredRuleConfig.getTables().size(), is(1)); @@ -128,9 +149,6 @@ void assertUpdateWithDifferentCase() { @Test void assertUpdateTableType() { AlterShardingTableRuleStatement sqlStatement = new AlterShardingTableRuleStatement(Arrays.asList(createCompleteAutoTableRule("t_order"), createCompleteTableRule("t_order_item"))); - ShardingRule rule = mock(ShardingRule.class); - when(rule.getConfiguration()).thenReturn(currentRuleConfig); - executor.setRule(rule); executor.checkBeforeUpdate(sqlStatement); ShardingRuleConfiguration toBeAlteredRuleConfig = executor.buildToBeAlteredRuleConfiguration(sqlStatement); assertThat(toBeAlteredRuleConfig.getTables().size(), is(1)); @@ -197,4 +215,13 @@ private Map createDataSource() { result.put("ds_1", new MockedDataSource()); return result; } + + @SneakyThrows(ReflectiveOperationException.class) + @SuppressWarnings("rawtypes") + private DistSQLStatement getDistSQLStatement(final String sql) { + ShardingDistSQLParserFacade facade = new ShardingDistSQLParserFacade(); + ParseASTNode parseASTNode = (ParseASTNode) SQLParserFactory.newInstance(sql, facade.getLexerClass(), facade.getParserClass()).parse(); + SQLVisitor visitor = facade.getVisitorClass().getDeclaredConstructor().newInstance(); + return (DistSQLStatement) visitor.visit(parseASTNode.getRootNode()); + } } diff --git a/features/sharding/distsql/handler/src/test/java/org/apache/shardingsphere/sharding/distsql/update/CreateShardingTableRuleExecutorTest.java b/features/sharding/distsql/handler/src/test/java/org/apache/shardingsphere/sharding/distsql/update/CreateShardingTableRuleExecutorTest.java index 4479ae23d634e7..2bba05b577f396 100644 --- a/features/sharding/distsql/handler/src/test/java/org/apache/shardingsphere/sharding/distsql/update/CreateShardingTableRuleExecutorTest.java +++ b/features/sharding/distsql/handler/src/test/java/org/apache/shardingsphere/sharding/distsql/update/CreateShardingTableRuleExecutorTest.java @@ -41,8 +41,10 @@ import org.apache.shardingsphere.sharding.distsql.segment.table.AutoTableRuleSegment; import org.apache.shardingsphere.sharding.distsql.segment.table.TableRuleSegment; import org.apache.shardingsphere.sharding.distsql.statement.CreateShardingTableRuleStatement; +import org.apache.shardingsphere.sharding.exception.metadata.DuplicateShardingActualDataNodeException; import org.apache.shardingsphere.sharding.exception.strategy.InvalidShardingStrategyConfigurationException; import org.apache.shardingsphere.sharding.rule.ShardingRule; +import org.apache.shardingsphere.sharding.rule.checker.ShardingRuleChecker; import org.apache.shardingsphere.sql.parser.api.visitor.SQLVisitor; import org.apache.shardingsphere.sql.parser.core.ParseASTNode; import org.apache.shardingsphere.sql.parser.core.SQLParserFactory; @@ -92,10 +94,12 @@ void setUp() { @Test void assertBuildToBeCreatedRuleConfiguration() { - CreateShardingTableRuleStatement sqlStatement = new CreateShardingTableRuleStatement(false, Arrays.asList(createCompleteAutoTableRule(), createCompleteTableRule())); ShardingRule rule = mock(ShardingRule.class); when(rule.getConfiguration()).thenReturn(currentRuleConfig); + ShardingRuleChecker checker = new ShardingRuleChecker(rule); + when(rule.getShardingRuleChecker()).thenReturn(checker); executor.setRule(rule); + CreateShardingTableRuleStatement sqlStatement = new CreateShardingTableRuleStatement(false, Arrays.asList(createCompleteAutoTableRule(), createCompleteTableRule())); executor.checkBeforeUpdate(sqlStatement); ShardingRuleConfiguration actual = executor.buildToBeCreatedRuleConfiguration(sqlStatement); assertThat(actual.getTables().size(), is(1)); @@ -185,15 +189,30 @@ void assertCheckCreateShardingStatementWithNoneTableStrategyThrows() { assertThrows(InvalidShardingStrategyConfigurationException.class, () -> executor.checkBeforeUpdate(distSQLStatement)); } + @Test + void assertCheckWithDuplicateDataNodes() { + String sql = "CREATE SHARDING TABLE RULE t_order(" + + "DATANODES('ds_${0..1}.t_order')," + + "DATABASE_STRATEGY(TYPE='standard',SHARDING_COLUMN=user_id,SHARDING_ALGORITHM(TYPE(NAME='inline',PROPERTIES('algorithm-expression'='ds_${user_id % 2}'))))" + + "), t_order_item(" + + "DATANODES('ds_${0..1}.t_order')," + + "DATABASE_STRATEGY(TYPE='standard',SHARDING_COLUMN=user_id,SHARDING_ALGORITHM(TYPE(NAME='inline',PROPERTIES('algorithm-expression'='ds_${user_id % 2}'))))" + + ");"; + CreateShardingTableRuleStatement sqlStatement = (CreateShardingTableRuleStatement) getDistSQLStatement(sql); + assertThrows(DuplicateShardingActualDataNodeException.class, () -> executor.checkBeforeUpdate(sqlStatement)); + } + @Test void assertUpdateWithIfNotExistsStatement() { + ShardingRule rule = mock(ShardingRule.class); + when(rule.getConfiguration()).thenReturn(currentRuleConfig); + ShardingRuleChecker checker = new ShardingRuleChecker(rule); + when(rule.getShardingRuleChecker()).thenReturn(checker); + executor.setRule(rule); Collection segments = new LinkedList<>(); segments.add(createCompleteAutoTableRule()); segments.add(createCompleteTableRule()); CreateShardingTableRuleStatement statementWithIfNotExists = new CreateShardingTableRuleStatement(true, segments); - ShardingRule rule = mock(ShardingRule.class); - when(rule.getConfiguration()).thenReturn(currentRuleConfig); - executor.setRule(rule); executor.checkBeforeUpdate(statementWithIfNotExists); ShardingRuleConfiguration actual = executor.buildToBeCreatedRuleConfiguration(statementWithIfNotExists); assertThat(actual.getTables().size(), is(1)); diff --git a/infra/exception/core/src/main/java/org/apache/shardingsphere/infra/exception/core/ShardingSpherePreconditions.java b/infra/exception/core/src/main/java/org/apache/shardingsphere/infra/exception/core/ShardingSpherePreconditions.java index 633dd047bdc372..bdec1cd19d170d 100644 --- a/infra/exception/core/src/main/java/org/apache/shardingsphere/infra/exception/core/ShardingSpherePreconditions.java +++ b/infra/exception/core/src/main/java/org/apache/shardingsphere/infra/exception/core/ShardingSpherePreconditions.java @@ -17,9 +17,12 @@ package org.apache.shardingsphere.infra.exception.core; +import com.google.common.base.Strings; import lombok.AccessLevel; import lombok.NoArgsConstructor; +import java.util.Collection; +import java.util.Map; import java.util.function.Supplier; /** @@ -55,4 +58,119 @@ public static void checkNotNull(final Object reference, fi throw exceptionSupplierIfUnexpected.get(); } } + + /** + * Ensures that a string passed as a parameter to the calling method is not empty. + * + * @param type of exception + * @param value string to be checked + * @param exceptionSupplierIfUnexpected exception from this supplier will be thrown if expression is unexpected + * @throws T exception to be thrown + */ + public static void checkNotEmpty(final String value, final Supplier exceptionSupplierIfUnexpected) throws T { + if (Strings.isNullOrEmpty(value)) { + throw exceptionSupplierIfUnexpected.get(); + } + } + + /** + * Ensures that a collection passed as a parameter to the calling method is not empty. + * + * @param type of exception + * @param values collection to be checked + * @param exceptionSupplierIfUnexpected exception from this supplier will be thrown if expression is unexpected + * @throws T exception to be thrown + */ + public static void checkNotEmpty(final Collection values, final Supplier exceptionSupplierIfUnexpected) throws T { + if (values.isEmpty()) { + throw exceptionSupplierIfUnexpected.get(); + } + } + + /** + * Ensures that a collection passed as a parameter to the calling method is not empty. + * + * @param type of exception + * @param map map to be checked + * @param exceptionSupplierIfUnexpected exception from this supplier will be thrown if expression is unexpected + * @throws T exception to be thrown + */ + public static void checkNotEmpty(final Map map, final Supplier exceptionSupplierIfUnexpected) throws T { + if (map.isEmpty()) { + throw exceptionSupplierIfUnexpected.get(); + } + } + + /** + * Ensures that a collection passed as a parameter to the calling method must empty. + * + * @param type of exception + * @param values collection to be checked + * @param exceptionSupplierIfUnexpected exception from this supplier will be thrown if expression is unexpected + * @throws T exception to be thrown + */ + public static void checkMustEmpty(final Collection values, final Supplier exceptionSupplierIfUnexpected) throws T { + if (!values.isEmpty()) { + throw exceptionSupplierIfUnexpected.get(); + } + } + + /** + * Ensures that a map passed as a parameter to the calling method must empty. + * + * @param type of exception + * @param map map to be checked + * @param exceptionSupplierIfUnexpected exception from this supplier will be thrown if expression is unexpected + * @throws T exception to be thrown + */ + public static void checkMustEmpty(final Map map, final Supplier exceptionSupplierIfUnexpected) throws T { + if (!map.isEmpty()) { + throw exceptionSupplierIfUnexpected.get(); + } + } + + /** + * Ensures that a collection passed as a parameter to the calling method must contain element. + * + * @param type of exception + * @param values values to be checked + * @param element element to be checked + * @param exceptionSupplierIfUnexpected exception from this supplier will be thrown if expression is unexpected + * @throws T exception to be thrown + */ + public static void checkContains(final Collection values, final Object element, final Supplier exceptionSupplierIfUnexpected) throws T { + if (!values.contains(element)) { + throw exceptionSupplierIfUnexpected.get(); + } + } + + /** + * Ensures that a collection passed as a parameter to the calling method must not contain element. + * + * @param type of exception + * @param values values to be checked + * @param element element to be checked + * @param exceptionSupplierIfUnexpected exception from this supplier will be thrown if expression is unexpected + * @throws T exception to be thrown + */ + public static void checkNotContains(final Collection values, final Object element, final Supplier exceptionSupplierIfUnexpected) throws T { + if (values.contains(element)) { + throw exceptionSupplierIfUnexpected.get(); + } + } + + /** + * Ensures that a map passed as a parameter to the calling method must contain key. + * + * @param type of exception + * @param map map to be checked + * @param key key to be checked + * @param exceptionSupplierIfUnexpected exception from this supplier will be thrown if expression is unexpected + * @throws T exception to be thrown + */ + public static void checkContainsKey(final Map map, final Object key, final Supplier exceptionSupplierIfUnexpected) throws T { + if (!map.containsKey(key)) { + throw exceptionSupplierIfUnexpected.get(); + } + } }