You are viewing a plain text version of this content. The canonical link for it is here.
Posted to notifications@shardingsphere.apache.org by pa...@apache.org on 2021/09/24 04:47:01 UTC

[shardingsphere] branch master updated: fix select subquery wrong result (#12658)

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

panjuan pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/shardingsphere.git


The following commit(s) were added to refs/heads/master by this push:
     new ffd8576  fix select subquery wrong result (#12658)
ffd8576 is described below

commit ffd857620152f33ee2935d06c90ec3cc1bbb3578
Author: Zhengqiang Duan <du...@apache.org>
AuthorDate: Fri Sep 24 12:46:24 2021 +0800

    fix select subquery wrong result (#12658)
    
    * fix select subquery wrong result
    
    * fix unit test
    
    * fix pagination route logic
    
    * fix unit test
    
    * fix unit test
    
    * add federate route test
    
    * optimize route logic
    
    * add todo
    
    * add todo
---
 .../route/engine/condition/ShardingCondition.java  |   4 +
 .../route/engine/condition/ShardingConditions.java |  48 +++++++-
 .../impl/WhereClauseShardingConditionEngine.java   |   9 +-
 .../engine/type/ShardingRouteEngineFactory.java    |  22 +++-
 .../federated/ShardingFederatedRoutingEngine.java  |  29 ++---
 .../ShardingStatementValidatorFactory.java         |   2 +-
 .../dml/impl/ShardingInsertStatementValidator.java |   2 +-
 .../dml/impl/ShardingSelectStatementValidator.java |   8 --
 .../type/ShardingRouteEngineFactoryTest.java       |  20 ++++
 .../ShardingFederatedRoutingEngineTest.java        | 122 +++++++++++++--------
 .../statement/dml/SelectStatementContext.java      |  14 ++-
 .../resources/scenario/sharding/case/select.xml    |   8 +-
 12 files changed, 195 insertions(+), 93 deletions(-)

diff --git a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/ShardingCondition.java b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/ShardingCondition.java
index af0de20..55dc4bc 100644
--- a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/ShardingCondition.java
+++ b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/ShardingCondition.java
@@ -18,6 +18,7 @@
 package org.apache.shardingsphere.sharding.route.engine.condition;
 
 import lombok.Getter;
+import lombok.Setter;
 import lombok.ToString;
 import org.apache.shardingsphere.sharding.route.engine.condition.value.ShardingConditionValue;
 
@@ -32,4 +33,7 @@ import java.util.List;
 public class ShardingCondition {
     
     private final List<ShardingConditionValue> values = new LinkedList<>();
+    
+    @Setter
+    private int startIndex;
 }
diff --git a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/ShardingConditions.java b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/ShardingConditions.java
index af3e6fe..6e6dab5 100644
--- a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/ShardingConditions.java
+++ b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/ShardingConditions.java
@@ -18,7 +18,6 @@
 package org.apache.shardingsphere.sharding.route.engine.condition;
 
 import lombok.Getter;
-import lombok.RequiredArgsConstructor;
 import lombok.ToString;
 import org.apache.shardingsphere.infra.binder.statement.SQLStatementContext;
 import org.apache.shardingsphere.infra.binder.statement.dml.InsertStatementContext;
@@ -31,16 +30,21 @@ import org.apache.shardingsphere.sharding.route.engine.condition.value.ShardingC
 import org.apache.shardingsphere.sharding.rule.BindingTableRule;
 import org.apache.shardingsphere.sharding.rule.ShardingRule;
 import org.apache.shardingsphere.sharding.rule.TableRule;
+import org.apache.shardingsphere.sql.parser.sql.common.segment.dml.expr.subquery.SubquerySegment;
+import org.apache.shardingsphere.sql.parser.sql.common.statement.dml.SelectStatement;
 import org.apache.shardingsphere.sql.parser.sql.common.util.SafeNumberOperationUtil;
 
 import java.util.ArrayList;
+import java.util.Collection;
+import java.util.LinkedList;
 import java.util.List;
+import java.util.Map;
 import java.util.Optional;
+import java.util.stream.Collectors;
 
 /**
  * Sharding conditions.
  */
-@RequiredArgsConstructor
 @Getter
 @ToString
 public final class ShardingConditions {
@@ -51,6 +55,15 @@ public final class ShardingConditions {
     
     private final ShardingRule rule;
     
+    private final boolean subqueryContainsShardingCondition;
+    
+    public ShardingConditions(final List<ShardingCondition> conditions, final SQLStatementContext<?> sqlStatementContext, final ShardingRule rule) {
+        this.conditions = conditions;
+        this.sqlStatementContext = sqlStatementContext;
+        this.rule = rule;
+        subqueryContainsShardingCondition = isSubqueryContainsShardingCondition(conditions, sqlStatementContext);
+    }
+    
     /**
      * Judge sharding conditions is always false or not.
      *
@@ -98,11 +111,38 @@ public final class ShardingConditions {
      */
     public boolean isNeedMerge() {
         boolean selectContainsSubquery = sqlStatementContext instanceof SelectStatementContext && ((SelectStatementContext) sqlStatementContext).isContainsSubquery();
-        boolean insertSelectContainsSubquery = sqlStatementContext instanceof InsertStatementContext && null != ((InsertStatementContext) sqlStatementContext).getInsertSelectContext()
+        boolean insertSelectContainsSubquery = sqlStatementContext instanceof InsertStatementContext && null != ((InsertStatementContext) sqlStatementContext).getInsertSelectContext() 
                 && ((InsertStatementContext) sqlStatementContext).getInsertSelectContext().getSelectStatementContext().isContainsSubquery();
         return (selectContainsSubquery || insertSelectContainsSubquery) && !rule.getShardingLogicTableNames(sqlStatementContext.getTablesContext().getTableNames()).isEmpty();
     }
     
+    private boolean isSubqueryContainsShardingCondition(final List<ShardingCondition> conditions, final SQLStatementContext<?> sqlStatementContext) {
+        Collection<SelectStatement> selectStatements = getSelectStatements(sqlStatementContext);
+        if (selectStatements.size() > 1) {
+            Map<Integer, List<ShardingCondition>> startIndexShardingConditions = conditions.stream().collect(Collectors.groupingBy(ShardingCondition::getStartIndex));
+            for (SelectStatement each : selectStatements) {
+                if (!each.getWhere().isPresent() || !startIndexShardingConditions.containsKey(each.getWhere().get().getExpr().getStartIndex())) {
+                    return false;
+                }
+            }
+        }
+        return true;
+    }
+    
+    private Collection<SelectStatement> getSelectStatements(final SQLStatementContext<?> sqlStatementContext) {
+        Collection<SelectStatement> result = new LinkedList<>();
+        if (sqlStatementContext instanceof SelectStatementContext) {
+            result.add(((SelectStatementContext) sqlStatementContext).getSqlStatement());
+            result.addAll(((SelectStatementContext) sqlStatementContext).getSubquerySegments().stream().map(SubquerySegment::getSelect).collect(Collectors.toList()));
+        }
+        if (sqlStatementContext instanceof InsertStatementContext && null != ((InsertStatementContext) sqlStatementContext).getInsertSelectContext()) {
+            SelectStatementContext selectStatementContext = ((InsertStatementContext) sqlStatementContext).getInsertSelectContext().getSelectStatementContext();
+            result.add(selectStatementContext.getSqlStatement());
+            result.addAll(selectStatementContext.getSubquerySegments().stream().map(SubquerySegment::getSelect).collect(Collectors.toList()));
+        }
+        return result;
+    }
+    
     /**
      * Judge whether all sharding conditions are same or not.
      *
@@ -116,7 +156,7 @@ public final class ShardingConditions {
                 return false;
             }
         }
-        return conditions.size() <= 1;
+        return subqueryContainsShardingCondition && conditions.size() == 1;
     }
     
     private boolean isSameShardingCondition(final ShardingRule shardingRule, final ShardingCondition shardingCondition1, final ShardingCondition shardingCondition2) {
diff --git a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/engine/impl/WhereClauseShardingConditionEngine.java b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/engine/impl/WhereClauseShardingConditionEngine.java
index 7491d22..f0831d8 100644
--- a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/engine/impl/WhereClauseShardingConditionEngine.java
+++ b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/condition/engine/impl/WhereClauseShardingConditionEngine.java
@@ -77,8 +77,8 @@ public final class WhereClauseShardingConditionEngine implements ShardingConditi
         return result;
     }
     
-    private Collection<ShardingCondition> createShardingConditions(final SQLStatementContext<?> sqlStatementContext, final ExpressionSegment expressionSegment, final List<Object> parameters) {
-        Collection<AndPredicate> andPredicates = ExpressionExtractUtil.getAndPredicates(expressionSegment);
+    private Collection<ShardingCondition> createShardingConditions(final SQLStatementContext<?> sqlStatementContext, final ExpressionSegment expression, final List<Object> parameters) {
+        Collection<AndPredicate> andPredicates = ExpressionExtractUtil.getAndPredicates(expression);
         Map<String, String> columnTableNames = getColumnTableNames(sqlStatementContext, andPredicates);
         Collection<ShardingCondition> result = new LinkedList<>();
         for (AndPredicate each : andPredicates) {
@@ -86,7 +86,10 @@ public final class WhereClauseShardingConditionEngine implements ShardingConditi
             if (shardingConditionValues.isEmpty()) {
                 return Collections.emptyList();
             }
-            result.add(createShardingCondition(shardingConditionValues));
+            ShardingCondition shardingCondition = createShardingCondition(shardingConditionValues);
+            // TODO remove startIndex when federation has perfect support for subquery
+            shardingCondition.setStartIndex(expression.getStartIndex());
+            result.add(shardingCondition);
         }
         return result;
     }
diff --git a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/type/ShardingRouteEngineFactory.java b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/type/ShardingRouteEngineFactory.java
index b95dd07..537e752 100644
--- a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/type/ShardingRouteEngineFactory.java
+++ b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/type/ShardingRouteEngineFactory.java
@@ -179,18 +179,29 @@ public final class ShardingRouteEngineFactory {
                                                                          final ShardingConditions shardingConditions, final Collection<String> tableNames, final ConfigurationProperties props) {
         if (isShardingStandardQuery(tableNames, shardingRule)) {
             ShardingStandardRoutingEngine result = new ShardingStandardRoutingEngine(getLogicTableName(shardingConditions, tableNames), shardingConditions, props);
-            boolean needExecuteByCalcite = sqlStatementContext instanceof SelectStatementContext && ((SelectStatementContext) sqlStatementContext).isNeedExecuteByCalcite();
+            boolean needExecuteByCalcite = isNeedExecuteByCalcite(sqlStatementContext, shardingConditions);
             if (!needExecuteByCalcite || result.route(shardingRule).isSingleRouting()) {
                 return result;
             }
         }
-        if (isShardingFederatedQuery(sqlStatementContext, tableNames, shardingRule)) {
-            return new ShardingFederatedRoutingEngine(tableNames, shardingConditions, props);
+        if (isShardingFederatedQuery(sqlStatementContext, tableNames, shardingRule, shardingConditions)) {
+            return new ShardingFederatedRoutingEngine(tableNames);
         }
         // TODO config for cartesian set
         return new ShardingComplexRoutingEngine(tableNames, shardingConditions, props);
     }
     
+    private static boolean isNeedExecuteByCalcite(final SQLStatementContext<?> sqlStatementContext, final ShardingConditions shardingConditions) {
+        if (!(sqlStatementContext instanceof SelectStatementContext)) {
+            return false;
+        }
+        SelectStatementContext selectStatementContext = (SelectStatementContext) sqlStatementContext;
+        if (selectStatementContext.getPaginationContext().isHasPagination()) {
+            return false;
+        }
+        return selectStatementContext.isNeedExecuteByCalcite() || (shardingConditions.isNeedMerge() && !shardingConditions.isSameShardingCondition());
+    }
+    
     private static String getLogicTableName(final ShardingConditions shardingConditions, final Collection<String> tableNames) {
         return shardingConditions.getConditions().stream().flatMap(each -> each.getValues().stream())
                 .map(ShardingConditionValue::getTableName).findFirst().orElseGet(() -> tableNames.iterator().next());
@@ -200,12 +211,13 @@ public final class ShardingRouteEngineFactory {
         return 1 == tableNames.size() && shardingRule.isAllShardingTables(tableNames) || shardingRule.isAllBindingTables(tableNames);
     }
     
-    private static boolean isShardingFederatedQuery(final SQLStatementContext<?> sqlStatementContext, final Collection<String> tableNames, final ShardingRule shardingRule) {
+    private static boolean isShardingFederatedQuery(final SQLStatementContext<?> sqlStatementContext, final Collection<String> tableNames, 
+                                                    final ShardingRule shardingRule, final ShardingConditions shardingConditions) {
         if (!(sqlStatementContext instanceof SelectStatementContext)) {
             return false;
         }
         SelectStatementContext select = (SelectStatementContext) sqlStatementContext;
-        if (select.isNeedExecuteByCalcite()) {
+        if (isNeedExecuteByCalcite(sqlStatementContext, shardingConditions)) {
             return true;
         }
         if ((!select.isContainsJoinQuery() && !select.isContainsSubquery()) || shardingRule.isAllTablesInSameDataSource(tableNames)) {
diff --git a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/type/federated/ShardingFederatedRoutingEngine.java b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/type/federated/ShardingFederatedRoutingEngine.java
index 44685ff..8869089 100644
--- a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/type/federated/ShardingFederatedRoutingEngine.java
+++ b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/type/federated/ShardingFederatedRoutingEngine.java
@@ -18,14 +18,12 @@
 package org.apache.shardingsphere.sharding.route.engine.type.federated;
 
 import lombok.RequiredArgsConstructor;
-import org.apache.shardingsphere.infra.config.properties.ConfigurationProperties;
+import org.apache.shardingsphere.infra.datanode.DataNode;
 import org.apache.shardingsphere.infra.route.context.RouteContext;
-import org.apache.shardingsphere.infra.route.context.RouteUnit;
-import org.apache.shardingsphere.sharding.route.engine.condition.ShardingConditions;
+import org.apache.shardingsphere.infra.route.context.RouteMapper;
 import org.apache.shardingsphere.sharding.route.engine.type.ShardingRouteEngine;
-import org.apache.shardingsphere.sharding.route.engine.type.standard.ShardingStandardRoutingEngine;
-import org.apache.shardingsphere.sharding.route.engine.type.unicast.ShardingUnicastRoutingEngine;
 import org.apache.shardingsphere.sharding.rule.ShardingRule;
+import org.apache.shardingsphere.sharding.rule.TableRule;
 
 import java.util.Collection;
 import java.util.Collections;
@@ -38,29 +36,22 @@ public final class ShardingFederatedRoutingEngine implements ShardingRouteEngine
     
     private final Collection<String> logicTables;
     
-    private final ShardingConditions shardingConditions;
-    
-    private final ConfigurationProperties properties;
-    
     @Override
     public RouteContext route(final ShardingRule shardingRule) {
         RouteContext result = new RouteContext();
         for (String each : logicTables) {
-            RouteContext newRouteContext;
-            if (shardingRule.isShardingTable(each)) {
-                newRouteContext = new ShardingStandardRoutingEngine(each, shardingConditions, properties).route(shardingRule);
-            } else {
-                newRouteContext = new ShardingUnicastRoutingEngine(Collections.singletonList(each)).route(shardingRule);
-            }
-            fillRouteContext(result, newRouteContext);
+            fillRouteContext(result, shardingRule, each);
         }
         result.setFederated(true);
         return result;
     }
     
-    private void fillRouteContext(final RouteContext routeContext, final RouteContext newRouteContext) {
-        for (RouteUnit each : newRouteContext.getRouteUnits()) {
-            routeContext.putRouteUnit(each.getDataSourceMapper(), each.getTableMappers());
+    private void fillRouteContext(final RouteContext routeContext, final ShardingRule shardingRule, final String logicTableName) {
+        TableRule tableRule = shardingRule.getTableRule(logicTableName);
+        for (DataNode each : tableRule.getActualDataNodes()) {
+            RouteMapper dataSource = new RouteMapper(each.getDataSourceName(), each.getDataSourceName());
+            RouteMapper table = new RouteMapper(logicTableName, each.getTableName());
+            routeContext.putRouteUnit(dataSource, Collections.singletonList(table));
         }
     }
 }
diff --git a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/ShardingStatementValidatorFactory.java b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/ShardingStatementValidatorFactory.java
index 3bcfb44..a46eace 100644
--- a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/ShardingStatementValidatorFactory.java
+++ b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/ShardingStatementValidatorFactory.java
@@ -127,7 +127,7 @@ public final class ShardingStatementValidatorFactory {
             return Optional.of(new ShardingDeleteStatementValidator());
         }
         if (sqlStatement instanceof SelectStatement) {
-            return Optional.of(new ShardingSelectStatementValidator(shardingConditions));
+            return Optional.of(new ShardingSelectStatementValidator());
         }
         return Optional.empty();
     }
diff --git a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/dml/impl/ShardingInsertStatementValidator.java b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/dml/impl/ShardingInsertStatementValidator.java
index b5651e8..2c004fd 100644
--- a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/dml/impl/ShardingInsertStatementValidator.java
+++ b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/dml/impl/ShardingInsertStatementValidator.java
@@ -97,7 +97,7 @@ public final class ShardingInsertStatementValidator extends ShardingDMLStatement
         Optional<SubquerySegment> insertSelect = sqlStatementContext.getSqlStatement().getInsertSelect();
         if (insertSelect.isPresent() && shardingConditions.isNeedMerge()) {
             boolean singleRoutingOrSameShardingCondition = routeContext.isSingleRouting() || shardingConditions.isSameShardingCondition();
-            Preconditions.checkState(singleRoutingOrSameShardingCondition, "Sharding conditions must be same with others.");
+            Preconditions.checkState(singleRoutingOrSameShardingCondition, "Subquery sharding conditions must be same with primary query.");
         }
         String tableName = sqlStatementContext.getSqlStatement().getTable().getTableName().getIdentifier().getValue();
         if (!routeContext.isSingleRouting() && !shardingRule.isBroadcastTable(tableName)) {
diff --git a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/dml/impl/ShardingSelectStatementValidator.java b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/dml/impl/ShardingSelectStatementValidator.java
index c3d0c3b..d912d14 100644
--- a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/dml/impl/ShardingSelectStatementValidator.java
+++ b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/main/java/org/apache/shardingsphere/sharding/route/engine/validator/dml/impl/ShardingSelectStatementValidator.java
@@ -17,13 +17,11 @@
 
 package org.apache.shardingsphere.sharding.route.engine.validator.dml.impl;
 
-import com.google.common.base.Preconditions;
 import lombok.RequiredArgsConstructor;
 import org.apache.shardingsphere.infra.binder.statement.SQLStatementContext;
 import org.apache.shardingsphere.infra.exception.ShardingSphereException;
 import org.apache.shardingsphere.infra.metadata.schema.ShardingSphereSchema;
 import org.apache.shardingsphere.infra.route.context.RouteContext;
-import org.apache.shardingsphere.sharding.route.engine.condition.ShardingConditions;
 import org.apache.shardingsphere.sharding.route.engine.validator.dml.ShardingDMLStatementValidator;
 import org.apache.shardingsphere.sharding.rule.ShardingRule;
 import org.apache.shardingsphere.sql.parser.sql.common.statement.dml.SelectStatement;
@@ -36,8 +34,6 @@ import java.util.List;
 @RequiredArgsConstructor
 public final class ShardingSelectStatementValidator extends ShardingDMLStatementValidator<SelectStatement> {
     
-    private final ShardingConditions shardingConditions;
-    
     @Override
     public void preValidate(final ShardingRule shardingRule, final SQLStatementContext<SelectStatement> sqlStatementContext, 
                             final List<Object> parameters, final ShardingSphereSchema schema) {
@@ -49,9 +45,5 @@ public final class ShardingSelectStatementValidator extends ShardingDMLStatement
     @Override
     public void postValidate(final ShardingRule shardingRule, final SQLStatementContext<SelectStatement> sqlStatementContext, 
                              final RouteContext routeContext, final ShardingSphereSchema schema) {
-        if (!routeContext.isFederated() && shardingConditions.isNeedMerge()) {
-            boolean singleRoutingOrSameShardingCondition = routeContext.isSingleRouting() || shardingConditions.isSameShardingCondition();
-            Preconditions.checkState(singleRoutingOrSameShardingCondition, "Sharding conditions must be same with others.");
-        }
     }
 }
diff --git a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/test/java/org/apache/shardingsphere/sharding/route/engine/type/ShardingRouteEngineFactoryTest.java b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/test/java/org/apache/shardingsphere/sharding/route/engine/type/ShardingRouteEngineFactoryTest.java
index a74c282..a90340d 100644
--- a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/test/java/org/apache/shardingsphere/sharding/route/engine/type/ShardingRouteEngineFactoryTest.java
+++ b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/test/java/org/apache/shardingsphere/sharding/route/engine/type/ShardingRouteEngineFactoryTest.java
@@ -20,6 +20,7 @@ package org.apache.shardingsphere.sharding.route.engine.type;
 import org.apache.shardingsphere.infra.binder.segment.table.TablesContext;
 import org.apache.shardingsphere.infra.binder.statement.SQLStatementContext;
 import org.apache.shardingsphere.infra.binder.statement.dcl.GrantStatementContext;
+import org.apache.shardingsphere.infra.binder.statement.dml.SelectStatementContext;
 import org.apache.shardingsphere.infra.config.properties.ConfigurationProperties;
 import org.apache.shardingsphere.infra.metadata.ShardingSphereMetaData;
 import org.apache.shardingsphere.infra.metadata.schema.ShardingSphereSchema;
@@ -29,6 +30,7 @@ import org.apache.shardingsphere.sharding.route.engine.type.broadcast.ShardingDa
 import org.apache.shardingsphere.sharding.route.engine.type.broadcast.ShardingInstanceBroadcastRoutingEngine;
 import org.apache.shardingsphere.sharding.route.engine.type.broadcast.ShardingTableBroadcastRoutingEngine;
 import org.apache.shardingsphere.sharding.route.engine.type.complex.ShardingComplexRoutingEngine;
+import org.apache.shardingsphere.sharding.route.engine.type.federated.ShardingFederatedRoutingEngine;
 import org.apache.shardingsphere.sharding.route.engine.type.ignore.ShardingIgnoreRoutingEngine;
 import org.apache.shardingsphere.sharding.route.engine.type.standard.ShardingStandardRoutingEngine;
 import org.apache.shardingsphere.sharding.route.engine.type.unicast.ShardingUnicastRoutingEngine;
@@ -62,11 +64,13 @@ import org.mockito.Mock;
 import org.mockito.junit.MockitoJUnitRunner;
 
 import java.util.ArrayList;
+import java.util.Arrays;
 import java.util.Collection;
 import java.util.Properties;
 
 import static org.hamcrest.CoreMatchers.instanceOf;
 import static org.junit.Assert.assertThat;
+import static org.mockito.Mockito.RETURNS_DEEP_STUBS;
 import static org.mockito.Mockito.mock;
 import static org.mockito.Mockito.when;
 
@@ -284,4 +288,20 @@ public final class ShardingRouteEngineFactoryTest {
         ShardingRouteEngine actual = ShardingRouteEngineFactory.newInstance(shardingRule, metaData, sqlStatementContext, shardingConditions, props);
         assertThat(actual, instanceOf(ShardingUnicastRoutingEngine.class));
     }
+    
+    @Test
+    public void assertNewInstanceForSubqueryWithDifferentConditions() {
+        SelectStatementContext sqlStatementContext = mock(SelectStatementContext.class, RETURNS_DEEP_STUBS);
+        tableNames.add("t_order");
+        when(sqlStatementContext.getTablesContext().getTableNames()).thenReturn(tableNames);
+        when(sqlStatementContext.isNeedExecuteByCalcite()).thenReturn(false);
+        ShardingRule shardingRule = mock(ShardingRule.class, RETURNS_DEEP_STUBS);
+        when(shardingRule.getShardingRuleTableNames(tableNames)).thenReturn(tableNames);
+        when(shardingRule.isAllShardingTables(tableNames)).thenReturn(true);
+        when(shardingRule.getTableRule("t_order").getActualDatasourceNames()).thenReturn(Arrays.asList("ds_0", "ds_1"));
+        when(shardingConditions.isNeedMerge()).thenReturn(true);
+        when(shardingConditions.isSameShardingCondition()).thenReturn(false);
+        ShardingRouteEngine actual = ShardingRouteEngineFactory.newInstance(shardingRule, metaData, sqlStatementContext, shardingConditions, props);
+        assertThat(actual, instanceOf(ShardingFederatedRoutingEngine.class));
+    }
 }
diff --git a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/test/java/org/apache/shardingsphere/sharding/route/engine/type/federated/ShardingFederatedRoutingEngineTest.java b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/test/java/org/apache/shardingsphere/sharding/route/engine/type/federated/ShardingFederatedRoutingEngineTest.java
index 0ba1145..86976b2 100644
--- a/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/test/java/org/apache/shardingsphere/sharding/route/engine/type/federated/ShardingFederatedRoutingEngineTest.java
+++ b/shardingsphere-features/shardingsphere-sharding/shardingsphere-sharding-core/src/test/java/org/apache/shardingsphere/sharding/route/engine/type/federated/ShardingFederatedRoutingEngineTest.java
@@ -17,12 +17,9 @@
 
 package org.apache.shardingsphere.sharding.route.engine.type.federated;
 
-import org.apache.shardingsphere.infra.binder.statement.SQLStatementContext;
-import org.apache.shardingsphere.infra.config.properties.ConfigurationProperties;
 import org.apache.shardingsphere.infra.route.context.RouteContext;
 import org.apache.shardingsphere.infra.route.context.RouteMapper;
 import org.apache.shardingsphere.infra.route.context.RouteUnit;
-import org.apache.shardingsphere.sharding.route.engine.condition.ShardingConditions;
 import org.apache.shardingsphere.sharding.route.engine.fixture.AbstractRoutingEngineTest;
 import org.apache.shardingsphere.sharding.rule.ShardingRule;
 import org.junit.Test;
@@ -33,7 +30,6 @@ import java.util.Collection;
 import java.util.Collections;
 import java.util.Iterator;
 import java.util.List;
-import java.util.Properties;
 
 import static org.hamcrest.CoreMatchers.is;
 import static org.junit.Assert.assertThat;
@@ -46,11 +42,10 @@ public final class ShardingFederatedRoutingEngineTest extends AbstractRoutingEng
     
     @Test
     public void assertRouteByNonConditions() {
-        ShardingFederatedRoutingEngine federatedRoutingEngine = createShardingFederatedRoutingEngine(Arrays.asList("t_order"),
-                new ShardingConditions(Collections.emptyList(), mock(SQLStatementContext.class), mock(ShardingRule.class)));
-        RouteContext routeContext = federatedRoutingEngine.route(createBasedShardingRule());
-        List<RouteUnit> routeUnits = new ArrayList<>(routeContext.getRouteUnits());
-        assertThat(routeContext.getRouteUnits().size(), is(2));
+        ShardingFederatedRoutingEngine federatedRoutingEngine = createShardingFederatedRoutingEngine(Collections.singletonList("t_order"));
+        RouteContext actual = federatedRoutingEngine.route(createBasedShardingRule());
+        List<RouteUnit> routeUnits = new ArrayList<>(actual.getRouteUnits());
+        assertThat(actual.getRouteUnits().size(), is(2));
         RouteUnit routeUnit1 = routeUnits.get(0);
         assertThat(routeUnit1.getDataSourceMapper().getActualName(), is("ds_0"));
         assertThat(routeUnit1.getTableMappers().size(), is(2));
@@ -77,58 +72,91 @@ public final class ShardingFederatedRoutingEngineTest extends AbstractRoutingEng
     
     @Test
     public void assertRouteByShardingConditions() {
-        ShardingFederatedRoutingEngine federatedRoutingEngine = createShardingFederatedRoutingEngine(Arrays.asList("t_order"), createShardingConditions("t_order"));
-        RouteContext routeContext = federatedRoutingEngine.route(createBasedShardingRule());
-        List<RouteUnit> routeUnits = new ArrayList<>(routeContext.getRouteUnits());
-        assertThat(routeContext.getRouteUnits().size(), is(1));
-        assertThat(routeUnits.get(0).getDataSourceMapper().getActualName(), is("ds_1"));
-        assertThat(routeUnits.get(0).getTableMappers().size(), is(1));
-        assertThat(routeUnits.get(0).getTableMappers().iterator().next().getActualName(), is("t_order_1"));
-        assertThat(routeUnits.get(0).getTableMappers().iterator().next().getLogicName(), is("t_order"));
+        ShardingFederatedRoutingEngine federatedRoutingEngine = createShardingFederatedRoutingEngine(Collections.singletonList("t_order"));
+        RouteContext actual = federatedRoutingEngine.route(createBasedShardingRule());
+        assertThat(actual.getRouteUnits().size(), is(2));
+        List<RouteUnit> routeUnits = new ArrayList<>(actual.getRouteUnits());
+        assertThat(routeUnits.get(0).getDataSourceMapper().getActualName(), is("ds_0"));
+        assertThat(routeUnits.get(0).getDataSourceMapper().getLogicName(), is("ds_0"));
+        assertThat(routeUnits.get(0).getTableMappers().size(), is(2));
+        List<RouteMapper> firstRouteMappers = new ArrayList<>(routeUnits.get(0).getTableMappers());
+        assertThat(firstRouteMappers.get(0).getActualName(), is("t_order_0"));
+        assertThat(firstRouteMappers.get(0).getLogicName(), is("t_order"));
+        assertThat(firstRouteMappers.get(1).getActualName(), is("t_order_1"));
+        assertThat(firstRouteMappers.get(1).getLogicName(), is("t_order"));
+        assertThat(routeUnits.get(1).getDataSourceMapper().getActualName(), is("ds_1"));
+        assertThat(routeUnits.get(1).getDataSourceMapper().getLogicName(), is("ds_1"));
+        assertThat(routeUnits.get(1).getTableMappers().size(), is(2));
+        List<RouteMapper> secondRouteMappers = new ArrayList<>(routeUnits.get(1).getTableMappers());
+        assertThat(secondRouteMappers.get(0).getActualName(), is("t_order_0"));
+        assertThat(secondRouteMappers.get(0).getLogicName(), is("t_order"));
+        assertThat(secondRouteMappers.get(1).getActualName(), is("t_order_1"));
+        assertThat(secondRouteMappers.get(1).getLogicName(), is("t_order"));
     }
     
     @Test
     public void assertRoutingForBindingTables() {
-        ShardingFederatedRoutingEngine federatedRoutingEngine = createShardingFederatedRoutingEngine(Arrays.asList("t_order", "t_order_item"), createShardingConditions("t_order"));
-        RouteContext routeContext = federatedRoutingEngine.route(createBindingShardingRule());
-        List<RouteUnit> routeUnits = new ArrayList<>(routeContext.getRouteUnits());
-        assertThat(routeContext.getRouteUnits().size(), is(1));
-        RouteUnit routeUnit = routeUnits.get(0);
-        assertThat(routeUnit.getDataSourceMapper().getActualName(), is("ds_1"));
-        assertThat(routeUnit.getDataSourceMapper().getLogicName(), is("ds_1"));
-        assertThat(routeUnit.getTableMappers().size(), is(2));
-        Collection<RouteMapper> tableMappers = routeUnit.getTableMappers();
-        Iterator<RouteMapper> iterator = tableMappers.iterator();
-        RouteMapper tableMapper1 = iterator.next();
-        assertThat(tableMapper1.getActualName(), is("t_order_1"));
-        assertThat(tableMapper1.getLogicName(), is("t_order"));
-        RouteMapper tableMapper2 = iterator.next();
-        assertThat(tableMapper2.getActualName(), is("t_order_item_1"));
-        assertThat(tableMapper2.getLogicName(), is("t_order_item"));
+        ShardingFederatedRoutingEngine federatedRoutingEngine = createShardingFederatedRoutingEngine(Arrays.asList("t_order", "t_order_item"));
+        RouteContext actual = federatedRoutingEngine.route(createBindingShardingRule());
+        assertThat(actual.getRouteUnits().size(), is(2));
+        List<RouteUnit> routeUnits = new ArrayList<>(actual.getRouteUnits());
+        assertThat(routeUnits.get(0).getDataSourceMapper().getActualName(), is("ds_0"));
+        assertThat(routeUnits.get(0).getDataSourceMapper().getLogicName(), is("ds_0"));
+        assertThat(routeUnits.get(0).getTableMappers().size(), is(4));
+        List<RouteMapper> firstRouteMappers = new ArrayList<>(routeUnits.get(0).getTableMappers());
+        assertThat(firstRouteMappers.get(0).getActualName(), is("t_order_0"));
+        assertThat(firstRouteMappers.get(0).getLogicName(), is("t_order"));
+        assertThat(firstRouteMappers.get(1).getActualName(), is("t_order_1"));
+        assertThat(firstRouteMappers.get(1).getLogicName(), is("t_order"));
+        assertThat(firstRouteMappers.get(2).getActualName(), is("t_order_item_0"));
+        assertThat(firstRouteMappers.get(2).getLogicName(), is("t_order_item"));
+        assertThat(firstRouteMappers.get(3).getActualName(), is("t_order_item_1"));
+        assertThat(firstRouteMappers.get(3).getLogicName(), is("t_order_item"));
+        assertThat(routeUnits.get(1).getDataSourceMapper().getActualName(), is("ds_1"));
+        assertThat(routeUnits.get(1).getDataSourceMapper().getLogicName(), is("ds_1"));
+        assertThat(routeUnits.get(1).getTableMappers().size(), is(4));
+        List<RouteMapper> secondRouteMappers = new ArrayList<>(routeUnits.get(1).getTableMappers());
+        assertThat(secondRouteMappers.get(0).getActualName(), is("t_order_0"));
+        assertThat(secondRouteMappers.get(0).getLogicName(), is("t_order"));
+        assertThat(secondRouteMappers.get(1).getActualName(), is("t_order_1"));
+        assertThat(secondRouteMappers.get(1).getLogicName(), is("t_order"));
+        assertThat(secondRouteMappers.get(2).getActualName(), is("t_order_item_0"));
+        assertThat(secondRouteMappers.get(2).getLogicName(), is("t_order_item"));
+        assertThat(secondRouteMappers.get(3).getActualName(), is("t_order_item_1"));
+        assertThat(secondRouteMappers.get(3).getLogicName(), is("t_order_item"));
     }
     
     @Test
     public void assertRoutingForNonLogicTable() {
-        ShardingFederatedRoutingEngine complexRoutingEngine = createShardingFederatedRoutingEngine(Collections.emptyList(), createShardingConditions("t_order"));
-        RouteContext routeContext = complexRoutingEngine.route(mock(ShardingRule.class));
-        assertThat(routeContext.getOriginalDataNodes().size(), is(0));
-        assertThat(routeContext.getRouteUnits().size(), is(0));
-        assertThat(routeContext.getRouteStageContexts().size(), is(0));
-        assertThat(routeContext.isFederated(), is(true));
+        ShardingFederatedRoutingEngine complexRoutingEngine = createShardingFederatedRoutingEngine(Collections.emptyList());
+        RouteContext actual = complexRoutingEngine.route(mock(ShardingRule.class));
+        assertThat(actual.getOriginalDataNodes().size(), is(0));
+        assertThat(actual.getRouteUnits().size(), is(0));
+        assertThat(actual.getRouteStageContexts().size(), is(0));
+        assertThat(actual.isFederated(), is(true));
     }
     
     @Test
     public void assertRoutingForShardingTableJoinBroadcastTable() {
-        ShardingFederatedRoutingEngine federatedRoutingEngine = createShardingFederatedRoutingEngine(Arrays.asList("t_config"), mock(ShardingConditions.class));
-        RouteContext routeContext = federatedRoutingEngine.route(createBroadcastShardingRule());
-        List<RouteUnit> routeUnits = new ArrayList<>(routeContext.getRouteUnits());
-        assertThat(routeContext.getRouteUnits().size(), is(1));
+        ShardingFederatedRoutingEngine federatedRoutingEngine = createShardingFederatedRoutingEngine(Collections.singletonList("t_config"));
+        RouteContext actual = federatedRoutingEngine.route(createBroadcastShardingRule());
+        List<RouteUnit> routeUnits = new ArrayList<>(actual.getRouteUnits());
+        assertThat(actual.getRouteUnits().size(), is(2));
+        assertThat(routeUnits.get(0).getDataSourceMapper().getActualName(), is("ds_0"));
+        assertThat(routeUnits.get(0).getDataSourceMapper().getLogicName(), is("ds_0"));
         assertThat(routeUnits.get(0).getTableMappers().size(), is(1));
-        assertThat(routeUnits.get(0).getTableMappers().iterator().next().getActualName(), is("t_config"));
-        assertThat(routeUnits.get(0).getTableMappers().iterator().next().getLogicName(), is("t_config"));
+        List<RouteMapper> firstRouteMappers = new ArrayList<>(routeUnits.get(0).getTableMappers());
+        assertThat(firstRouteMappers.get(0).getActualName(), is("t_config"));
+        assertThat(firstRouteMappers.get(0).getLogicName(), is("t_config"));
+        assertThat(routeUnits.get(1).getDataSourceMapper().getActualName(), is("ds_1"));
+        assertThat(routeUnits.get(1).getDataSourceMapper().getLogicName(), is("ds_1"));
+        assertThat(routeUnits.get(1).getTableMappers().size(), is(1));
+        List<RouteMapper> secondRouteMappers = new ArrayList<>(routeUnits.get(0).getTableMappers());
+        assertThat(secondRouteMappers.get(0).getActualName(), is("t_config"));
+        assertThat(secondRouteMappers.get(0).getLogicName(), is("t_config"));
     }
     
-    private ShardingFederatedRoutingEngine createShardingFederatedRoutingEngine(final Collection<String> logicTables, final ShardingConditions shardingConditions) {
-        return new ShardingFederatedRoutingEngine(logicTables, shardingConditions, new ConfigurationProperties(new Properties()));
+    private ShardingFederatedRoutingEngine createShardingFederatedRoutingEngine(final Collection<String> logicTables) {
+        return new ShardingFederatedRoutingEngine(logicTables);
     }
 }
diff --git a/shardingsphere-infra/shardingsphere-infra-binder/src/main/java/org/apache/shardingsphere/infra/binder/statement/dml/SelectStatementContext.java b/shardingsphere-infra/shardingsphere-infra-binder/src/main/java/org/apache/shardingsphere/infra/binder/statement/dml/SelectStatementContext.java
index d5bfb39..5cf18cd 100644
--- a/shardingsphere-infra/shardingsphere-infra-binder/src/main/java/org/apache/shardingsphere/infra/binder/statement/dml/SelectStatementContext.java
+++ b/shardingsphere-infra/shardingsphere-infra-binder/src/main/java/org/apache/shardingsphere/infra/binder/statement/dml/SelectStatementContext.java
@@ -77,7 +77,7 @@ public final class SelectStatementContext extends CommonSQLStatementContext<Sele
     
     private final PaginationContext paginationContext;
     
-    private final boolean containsSubquery;
+    private final Collection<SubquerySegment> subquerySegments;
     
     private final boolean needExecuteByCalcite;
     
@@ -92,8 +92,7 @@ public final class SelectStatementContext extends CommonSQLStatementContext<Sele
         projectionsContext = new ProjectionsContextEngine(schema, getDatabaseType())
                 .createProjectionsContext(getSqlStatement().getFrom(), getSqlStatement().getProjections(), groupByContext, orderByContext);
         paginationContext = new PaginationContextEngine().createPaginationContext(sqlStatement, projectionsContext, parameters);
-        Collection<SubquerySegment> subquerySegments = SubqueryExtractUtil.getSubquerySegments(getSqlStatement());
-        containsSubquery = !subquerySegments.isEmpty();
+        subquerySegments = SubqueryExtractUtil.getSubquerySegments(getSqlStatement());
         needExecuteByCalcite = checkNeedExecuteByCalcite(subquerySegments);
         this.schemaName = defaultSchemaName;
     }
@@ -124,6 +123,15 @@ public final class SelectStatementContext extends CommonSQLStatementContext<Sele
         return getSqlStatement().getFrom() instanceof JoinTableSegment;
     }
     
+    /**
+     * Judge whether contains subquery or not.
+     *
+     * @return whether contains subquery or not
+     */
+    public boolean isContainsSubquery() {
+        return !subquerySegments.isEmpty();
+    }
+    
     private boolean isContainsHaving() {
         return getSqlStatement().getHaving().isPresent();
     }
diff --git a/shardingsphere-test/shardingsphere-rewrite-test/src/test/resources/scenario/sharding/case/select.xml b/shardingsphere-test/shardingsphere-rewrite-test/src/test/resources/scenario/sharding/case/select.xml
index c4d7693..d3e4b39 100644
--- a/shardingsphere-test/shardingsphere-rewrite-test/src/test/resources/scenario/sharding/case/select.xml
+++ b/shardingsphere-test/shardingsphere-rewrite-test/src/test/resources/scenario/sharding/case/select.xml
@@ -68,6 +68,11 @@
         <output sql="SELECT * FROM (select t_account_0.account_id from t_account_0 where t_account_0.account_id=?) a WHERE account_id = 100" parameters="100" />
     </rewrite-assertion>
 
+    <rewrite-assertion id="select_with_subquery_and_different_sharding_conditions" db-type="MySQL">
+        <input sql="SELECT * FROM (select t_account.account_id from t_account where t_account.account_id=?) a WHERE account_id = 101" parameters="100" />
+        <output sql="SELECT * FROM (select t_account.account_id from t_account where t_account.account_id=?) a WHERE account_id = 101" parameters="100" />
+    </rewrite-assertion>
+
     <rewrite-assertion id="select_with_subquery_with_subquery" db-type="MySQL">
         <input sql="SELECT * FROM (select b.account_id from (select t_account.account_id from t_account) b where b.account_id=?) a WHERE account_id = 100" parameters="100" />
         <output sql="SELECT * FROM (select b.account_id from (select t_account_0.account_id from t_account_0) b where b.account_id=?) a WHERE account_id = 100" parameters="100" />
@@ -80,8 +85,7 @@
 
     <rewrite-assertion id="select_with_subquery_only_in_projection" db-type="MySQL">
         <input sql="SELECT (select id from t_account)"/>
-        <output sql="SELECT (select id from t_account_0)"/>
-        <output sql="SELECT (select id from t_account_1)"/>
+        <output sql="SELECT (select id from t_account)"/>
     </rewrite-assertion>
 
     <rewrite-assertion id="select_with_subquery_for_where_in_predicate" db-type="MySQL">