[mv](nereids) mv cost related PRs (#35652 #35701 #35864 #36368 #36789 #34970) (#37097)

## Proposed changes
pick from #35652 #35701 #35864 #36368 #36789 #34970

Issue Number: close #xxx

<!--Describe your changes.-->
This commit is contained in:
minghong
2024-07-04 09:42:11 +08:00
committed by GitHub
parent 077fda4259
commit 26be313d40
25 changed files with 494 additions and 299 deletions

View File

@ -164,6 +164,18 @@ public class MaterializedIndexMeta implements Writable, GsonPostProcessable {
initColumnNameMap();
}
public List<Column> getPrefixKeyColumns() {
List<Column> keys = Lists.newArrayList();
for (Column col : schema) {
if (col.isKey()) {
keys.add(col);
} else {
break;
}
}
return keys;
}
public void setSchemaHash(int newSchemaHash) {
this.schemaHash = newSchemaHash;
}

View File

@ -17,6 +17,7 @@
package org.apache.doris.nereids.cost;
import org.apache.doris.catalog.Column;
import org.apache.doris.catalog.KeysType;
import org.apache.doris.catalog.OlapTable;
import org.apache.doris.nereids.PlanContext;
@ -24,14 +25,19 @@ import org.apache.doris.nereids.properties.DistributionSpec;
import org.apache.doris.nereids.properties.DistributionSpecGather;
import org.apache.doris.nereids.properties.DistributionSpecHash;
import org.apache.doris.nereids.properties.DistributionSpecReplicated;
import org.apache.doris.nereids.trees.expressions.ComparisonPredicate;
import org.apache.doris.nereids.trees.expressions.Expression;
import org.apache.doris.nereids.trees.expressions.SlotReference;
import org.apache.doris.nereids.trees.expressions.literal.Literal;
import org.apache.doris.nereids.trees.plans.Plan;
import org.apache.doris.nereids.trees.plans.algebra.OlapScan;
import org.apache.doris.nereids.trees.plans.physical.PhysicalAssertNumRows;
import org.apache.doris.nereids.trees.plans.physical.PhysicalDeferMaterializeOlapScan;
import org.apache.doris.nereids.trees.plans.physical.PhysicalDeferMaterializeTopN;
import org.apache.doris.nereids.trees.plans.physical.PhysicalDistribute;
import org.apache.doris.nereids.trees.plans.physical.PhysicalEsScan;
import org.apache.doris.nereids.trees.plans.physical.PhysicalFileScan;
import org.apache.doris.nereids.trees.plans.physical.PhysicalFilter;
import org.apache.doris.nereids.trees.plans.physical.PhysicalGenerate;
import org.apache.doris.nereids.trees.plans.physical.PhysicalHashAggregate;
import org.apache.doris.nereids.trees.plans.physical.PhysicalHashJoin;
@ -52,8 +58,11 @@ import org.apache.doris.qe.SessionVariable;
import org.apache.doris.statistics.Statistics;
import com.google.common.base.Preconditions;
import com.google.common.collect.Sets;
import java.util.Collections;
import java.util.List;
import java.util.Set;
class CostModelV1 extends PlanVisitor<Cost, PlanContext> {
@ -113,6 +122,57 @@ class CostModelV1 extends PlanVisitor<Cost, PlanContext> {
return CostV1.ofCpu(context.getSessionVariable(), rows - aggMvBonus);
}
private Set<Column> getColumnForRangePredicate(Set<Expression> expressions) {
Set<Column> columns = Sets.newHashSet();
for (Expression expr : expressions) {
if (expr instanceof ComparisonPredicate) {
ComparisonPredicate compare = (ComparisonPredicate) expr;
boolean hasLiteral = compare.left() instanceof Literal || compare.right() instanceof Literal;
boolean hasSlot = compare.left() instanceof SlotReference || compare.right() instanceof SlotReference;
if (hasSlot && hasLiteral) {
if (compare.left() instanceof SlotReference) {
if (((SlotReference) compare.left()).getColumn().isPresent()) {
columns.add(((SlotReference) compare.left()).getColumn().get());
}
} else {
if (((SlotReference) compare.right()).getColumn().isPresent()) {
columns.add(((SlotReference) compare.right()).getColumn().get());
}
}
}
}
}
return columns;
}
@Override
public Cost visitPhysicalFilter(PhysicalFilter<? extends Plan> filter, PlanContext context) {
double exprCost = expressionTreeCost(filter.getExpressions());
double filterCostFactor = 0.0001;
if (ConnectContext.get() != null) {
filterCostFactor = ConnectContext.get().getSessionVariable().filterCostFactor;
}
int prefixIndexMatched = 0;
if (filter.getGroupExpression().isPresent()) {
OlapScan olapScan = (OlapScan) filter.getGroupExpression().get().getFirstChildPlan(OlapScan.class);
if (olapScan != null) {
// check prefix index
long idxId = olapScan.getSelectedIndexId();
List<Column> keyColumns = olapScan.getTable().getIndexMetaByIndexId(idxId).getPrefixKeyColumns();
Set<Column> predicateColumns = getColumnForRangePredicate(filter.getConjuncts());
for (Column col : keyColumns) {
if (predicateColumns.contains(col)) {
prefixIndexMatched++;
} else {
break;
}
}
}
}
return CostV1.ofCpu(context.getSessionVariable(),
(filter.getConjuncts().size() - prefixIndexMatched + exprCost) * filterCostFactor);
}
@Override
public Cost visitPhysicalDeferMaterializeOlapScan(PhysicalDeferMaterializeOlapScan deferMaterializeOlapScan,
PlanContext context) {
@ -141,7 +201,8 @@ class CostModelV1 extends PlanVisitor<Cost, PlanContext> {
@Override
public Cost visitPhysicalProject(PhysicalProject<? extends Plan> physicalProject, PlanContext context) {
return CostV1.ofCpu(context.getSessionVariable(), 1);
double exprCost = expressionTreeCost(physicalProject.getProjects());
return CostV1.ofCpu(context.getSessionVariable(), exprCost + 1);
}
@Override
@ -252,16 +313,29 @@ class CostModelV1 extends PlanVisitor<Cost, PlanContext> {
intputRowCount * childStatistics.dataSizeFactor() * RANDOM_SHUFFLE_TO_HASH_SHUFFLE_FACTOR / beNumber);
}
private double expressionTreeCost(List<? extends Expression> expressions) {
double exprCost = 0.0;
ExpressionCostEvaluator expressionCostEvaluator = new ExpressionCostEvaluator();
for (Expression expr : expressions) {
if (!(expr instanceof SlotReference)) {
exprCost += expr.accept(expressionCostEvaluator, null);
}
}
return exprCost;
}
@Override
public Cost visitPhysicalHashAggregate(
PhysicalHashAggregate<? extends Plan> aggregate, PlanContext context) {
Statistics inputStatistics = context.getChildStatistics(0);
double exprCost = expressionTreeCost(aggregate.getExpressions());
if (aggregate.getAggPhase().isLocal()) {
return CostV1.of(context.getSessionVariable(), inputStatistics.getRowCount() / beNumber,
return CostV1.of(context.getSessionVariable(),
exprCost / 100 + inputStatistics.getRowCount() / beNumber,
inputStatistics.getRowCount() / beNumber, 0);
} else {
// global
return CostV1.of(context.getSessionVariable(), inputStatistics.getRowCount(),
return CostV1.of(context.getSessionVariable(), exprCost / 100 + inputStatistics.getRowCount(),
inputStatistics.getRowCount(), 0);
}
}
@ -289,7 +363,7 @@ class CostModelV1 extends PlanVisitor<Cost, PlanContext> {
double leftRowCount = probeStats.getRowCount();
double rightRowCount = buildStats.getRowCount();
if (leftRowCount == rightRowCount
if ((long) leftRowCount == (long) rightRowCount
&& physicalHashJoin.getGroupExpression().isPresent()
&& physicalHashJoin.getGroupExpression().get().getOwnerGroup() != null
&& !physicalHashJoin.getGroupExpression().get().getOwnerGroup().isStatsReliable()) {

View File

@ -0,0 +1,86 @@
// 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.doris.nereids.cost;
import org.apache.doris.nereids.trees.expressions.Alias;
import org.apache.doris.nereids.trees.expressions.Expression;
import org.apache.doris.nereids.trees.expressions.SlotReference;
import org.apache.doris.nereids.trees.expressions.literal.Literal;
import org.apache.doris.nereids.trees.expressions.visitor.ExpressionVisitor;
import org.apache.doris.nereids.types.ArrayType;
import org.apache.doris.nereids.types.CharType;
import org.apache.doris.nereids.types.DecimalV2Type;
import org.apache.doris.nereids.types.DecimalV3Type;
import org.apache.doris.nereids.types.MapType;
import org.apache.doris.nereids.types.StringType;
import org.apache.doris.nereids.types.StructType;
import org.apache.doris.nereids.types.VarcharType;
import com.google.common.collect.Maps;
import java.util.Map;
/**
* expression cost is calculated by
* 1. non-leaf tree node count: N
* 2. expression which contains input of stringType or complexType(array/json/struct...), add cost
*/
public class ExpressionCostEvaluator extends ExpressionVisitor<Double, Void> {
private static Map<Class, Double> dataTypeCost = Maps.newHashMap();
static {
dataTypeCost.put(DecimalV2Type.class, 1.5);
dataTypeCost.put(DecimalV3Type.class, 1.5);
dataTypeCost.put(StringType.class, 2.0);
dataTypeCost.put(CharType.class, 2.0);
dataTypeCost.put(VarcharType.class, 2.0);
dataTypeCost.put(ArrayType.class, 3.0);
dataTypeCost.put(MapType.class, 3.0);
dataTypeCost.put(StructType.class, 3.0);
}
@Override
public Double visit(Expression expr, Void context) {
double cost = 0.0;
for (Expression child : expr.children()) {
cost += child.accept(this, context);
// the more children, the more computing cost
cost += dataTypeCost.getOrDefault(child.getDataType().getClass(), 0.1);
}
return cost;
}
@Override
public Double visitSlotReference(SlotReference slot, Void context) {
return 0.0;
}
@Override
public Double visitLiteral(Literal literal, Void context) {
return 0.0;
}
@Override
public Double visitAlias(Alias alias, Void context) {
Expression child = alias.child();
if (child instanceof SlotReference) {
return 0.0;
}
return alias.child().accept(this, context);
}
}

View File

@ -349,4 +349,28 @@ public class GroupExpression {
public ObjectId getId() {
return id;
}
/**
* the first child plan of clazz
* @param clazz the operator type, like join/aggregate
* @return child operator of type clazz, if not found, return null
*/
public Plan getFirstChildPlan(Class clazz) {
for (Group childGroup : children) {
for (GroupExpression logical : childGroup.getLogicalExpressions()) {
if (clazz.isInstance(logical.getPlan())) {
return logical.getPlan();
}
}
}
// for dphyp
for (Group childGroup : children) {
for (GroupExpression physical : childGroup.getPhysicalExpressions()) {
if (clazz.isInstance(physical.getPlan())) {
return physical.getPlan();
}
}
}
return null;
}
}

View File

@ -603,24 +603,24 @@ public class StatsCalculator extends DefaultPlanVisitor<Statistics, Void> {
private Statistics computeFilter(Filter filter) {
Statistics stats = groupExpression.childStatistics(0);
Plan plan = tryToFindChild(groupExpression);
boolean isOnBaseTable = false;
if (plan != null) {
if (plan instanceof OlapScan) {
isOnBaseTable = true;
} else if (plan instanceof Aggregate) {
Aggregate agg = ((Aggregate<?>) plan);
List<NamedExpression> expressions = agg.getOutputExpressions();
Set<Slot> slots = expressions
.stream()
.filter(Alias.class::isInstance)
.filter(s -> ((Alias) s).child().anyMatch(AggregateFunction.class::isInstance))
.map(NamedExpression::toSlot).collect(Collectors.toSet());
Expression predicate = filter.getPredicate();
if (predicate.anyMatch(s -> slots.contains(s))) {
return new FilterEstimation(slots).estimate(filter.getPredicate(), stats);
}
} else if (plan instanceof LogicalJoin && filter instanceof LogicalFilter
if (groupExpression.getFirstChildPlan(OlapScan.class) != null) {
return new FilterEstimation(true).estimate(filter.getPredicate(), stats);
}
if (groupExpression.getFirstChildPlan(Aggregate.class) != null) {
Aggregate agg = (Aggregate<?>) groupExpression.getFirstChildPlan(Aggregate.class);
List<NamedExpression> expressions = agg.getOutputExpressions();
Set<Slot> slots = expressions
.stream()
.filter(Alias.class::isInstance)
.filter(s -> ((Alias) s).child().anyMatch(AggregateFunction.class::isInstance))
.map(NamedExpression::toSlot).collect(Collectors.toSet());
Expression predicate = filter.getPredicate();
if (predicate.anyMatch(s -> slots.contains(s))) {
return new FilterEstimation(slots).estimate(filter.getPredicate(), stats);
}
} else if (groupExpression.getFirstChildPlan(LogicalJoin.class) != null) {
LogicalJoin plan = (LogicalJoin) groupExpression.getFirstChildPlan(LogicalJoin.class);
if (filter instanceof LogicalFilter
&& filter.getConjuncts().stream().anyMatch(e -> e instanceof IsNull)) {
Statistics isNullStats = computeGeneratedIsNullStats((LogicalJoin) plan, filter);
if (isNullStats != null) {
@ -640,8 +640,7 @@ public class StatsCalculator extends DefaultPlanVisitor<Statistics, Void> {
}
}
}
return new FilterEstimation(isOnBaseTable).estimate(filter.getPredicate(), stats);
return new FilterEstimation(false).estimate(filter.getPredicate(), stats);
}
private Statistics computeGeneratedIsNullStats(LogicalJoin join, Filter filter) {

View File

@ -1205,6 +1205,8 @@ public class SessionVariable implements Serializable, Writable {
@VariableMgr.VarAttr(name = ENABLE_NEW_COST_MODEL, needForward = true)
private boolean enableNewCostModel = false;
@VariableMgr.VarAttr(name = "filter_cost_factor", needForward = true)
public double filterCostFactor = 0.0001;
@VariableMgr.VarAttr(name = NEREIDS_STAR_SCHEMA_SUPPORT)
private boolean nereidsStarSchemaSupport = true;

View File

@ -26,22 +26,22 @@ PhysicalResultSink
----------------------------PhysicalOlapScan[customer_demographics] apply RFs: RF4
----------------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------PhysicalProject
--------------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = store_sales.ss_customer_sk)) otherCondition=() build RFs:RF3 c_customer_sk->[ss_customer_sk]
--------------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = web_sales.ws_bill_customer_sk)) otherCondition=()
----------------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------PhysicalProject
--------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ss_sold_date_sk]
--------------------------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF3 d_date_sk->[ws_sold_date_sk]
----------------------------------------PhysicalProject
------------------------------------------PhysicalOlapScan[store_sales] apply RFs: RF2 RF3
------------------------------------------PhysicalOlapScan[web_sales] apply RFs: RF3
----------------------------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------------------PhysicalProject
--------------------------------------------filter((date_dim.d_moy <= 6) and (date_dim.d_moy >= 3) and (date_dim.d_year = 2001))
----------------------------------------------PhysicalOlapScan[date_dim]
----------------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = web_sales.ws_bill_customer_sk)) otherCondition=()
----------------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = store_sales.ss_customer_sk)) otherCondition=() build RFs:RF2 c_customer_sk->[ss_customer_sk]
------------------------------------PhysicalDistribute[DistributionSpecHash]
--------------------------------------PhysicalProject
----------------------------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[ws_sold_date_sk]
----------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[ss_sold_date_sk]
------------------------------------------PhysicalProject
--------------------------------------------PhysicalOlapScan[web_sales] apply RFs: RF1
--------------------------------------------PhysicalOlapScan[store_sales] apply RFs: RF1 RF2
------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------------PhysicalProject
----------------------------------------------filter((date_dim.d_moy <= 6) and (date_dim.d_moy >= 3) and (date_dim.d_year = 2001))

View File

@ -12,23 +12,23 @@ PhysicalResultSink
------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((cs1.cs_order_number = cr1.cr_order_number)) otherCondition=()
--------------------PhysicalProject
----------------------hashJoin[INNER_JOIN] hashCondition=((cs1.cs_call_center_sk = call_center.cc_call_center_sk)) otherCondition=() build RFs:RF3 cc_call_center_sk->[cs_call_center_sk]
------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((cs1.cs_order_number = cs2.cs_order_number)) otherCondition=(( not (cs_warehouse_sk = cs_warehouse_sk))) build RFs:RF2 cs_order_number->[cs_order_number]
--------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF2
--------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------hashJoin[INNER_JOIN] hashCondition=((cs1.cs_ship_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF1 ca_address_sk->[cs_ship_addr_sk]
------------------------hashJoin[INNER_JOIN] hashCondition=((cs1.cs_ship_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF2 ca_address_sk->[cs_ship_addr_sk]
--------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((cs1.cs_order_number = cs2.cs_order_number)) otherCondition=(( not (cs_warehouse_sk = cs_warehouse_sk))) build RFs:RF1 cs_order_number->[cs_order_number]
----------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------PhysicalProject
--------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF1
----------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------hashJoin[INNER_JOIN] hashCondition=((cs1.cs_ship_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[cs_ship_date_sk]
--------------------------------PhysicalProject
----------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF0 RF1 RF3
----------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF0 RF2 RF3
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((date_dim.d_date <= '2002-05-31') and (date_dim.d_date >= '2002-04-01'))
--------------------------------------PhysicalOlapScan[date_dim]
------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------PhysicalProject
----------------------------------filter((customer_address.ca_state = 'WV'))
------------------------------------PhysicalOlapScan[customer_address]
--------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------PhysicalProject
------------------------------filter((customer_address.ca_state = 'WV'))
--------------------------------PhysicalOlapScan[customer_address]
------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------PhysicalProject
----------------------------filter(cc_county IN ('Barrow County', 'Daviess County', 'Luce County', 'Richland County', 'Ziebach County'))

View File

@ -27,6 +27,9 @@ PhysicalCteAnchor ( cteId=CTEId#1 )
----------------hashJoin[INNER_JOIN] hashCondition=((expr_cast(d_week_seq1 as BIGINT) = expr_(d_week_seq2 - 53))) otherCondition=()
------------------PhysicalDistribute[DistributionSpecHash]
--------------------PhysicalProject
----------------------PhysicalCteConsumer ( cteId=CTEId#1 ) apply RFs: RF2
------------------PhysicalDistribute[DistributionSpecHash]
--------------------PhysicalProject
----------------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_week_seq = d_week_seq1)) otherCondition=() build RFs:RF1 d_week_seq->[d_week_seq]
------------------------PhysicalDistribute[DistributionSpecExecutionAny]
--------------------------PhysicalProject
@ -35,9 +38,6 @@ PhysicalCteAnchor ( cteId=CTEId#1 )
--------------------------PhysicalProject
----------------------------filter((date_dim.d_year = 1998))
------------------------------PhysicalOlapScan[date_dim]
------------------PhysicalDistribute[DistributionSpecHash]
--------------------PhysicalProject
----------------------PhysicalCteConsumer ( cteId=CTEId#1 ) apply RFs: RF2
--------------PhysicalDistribute[DistributionSpecReplicated]
----------------PhysicalProject
------------------filter((date_dim.d_year = 1999))

View File

@ -20,51 +20,50 @@ PhysicalCteAnchor ( cteId=CTEId#1 )
----------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_promo_sk = promotion.p_promo_sk)) otherCondition=()
------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_hdemo_sk = hd1.hd_demo_sk)) otherCondition=()
--------------------------------------PhysicalProject
----------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_first_shipto_date_sk = d3.d_date_sk)) otherCondition=()
----------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = d1.d_date_sk)) otherCondition=() build RFs:RF11 d_date_sk->[ss_sold_date_sk]
------------------------------------------PhysicalProject
--------------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_first_sales_date_sk = d2.d_date_sk)) otherCondition=()
--------------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_first_shipto_date_sk = d3.d_date_sk)) otherCondition=()
----------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_addr_sk = ad1.ca_address_sk)) otherCondition=()
------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_current_cdemo_sk = cd2.cd_demo_sk)) otherCondition=(( not (cd_marital_status = cd_marital_status)))
--------------------------------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------------------PhysicalProject
--------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_first_sales_date_sk = d2.d_date_sk)) otherCondition=()
----------------------------------------------------PhysicalProject
------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = d1.d_date_sk)) otherCondition=() build RFs:RF7 d_date_sk->[ss_sold_date_sk]
--------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_cdemo_sk = cd1.cd_demo_sk)) otherCondition=()
------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_item_sk = cs_ui.cs_item_sk)) otherCondition=() build RFs:RF7 cs_item_sk->[sr_item_sk,ss_item_sk]
--------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_current_cdemo_sk = cd2.cd_demo_sk)) otherCondition=(( not (cd_marital_status = cd_marital_status)))
----------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_customer_sk = customer.c_customer_sk)) otherCondition=()
------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_cdemo_sk = cd1.cd_demo_sk)) otherCondition=()
--------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------------------------------------------PhysicalProject
------------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_item_sk = cs_ui.cs_item_sk)) otherCondition=() build RFs:RF4 cs_item_sk->[sr_item_sk,ss_item_sk]
----------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_customer_sk = customer.c_customer_sk)) otherCondition=()
------------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
--------------------------------------------------------------------PhysicalProject
----------------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_item_sk = store_returns.sr_item_sk) and (store_sales.ss_ticket_number = store_returns.sr_ticket_number)) otherCondition=()
------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------PhysicalOlapScan[store_sales] apply RFs: RF4 RF7 RF19
--------------------------------------------------------------------------PhysicalOlapScan[store_sales] apply RFs: RF7 RF11 RF19
------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------PhysicalOlapScan[store_returns] apply RFs: RF4 RF19
--------------------------------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------------------------------------PhysicalProject
------------------------------------------------------------------------filter((sale > (2 * refund)))
--------------------------------------------------------------------------hashAgg[GLOBAL]
----------------------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------------------------------------------------hashAgg[LOCAL]
--------------------------------------------------------------------------------PhysicalProject
----------------------------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_item_sk = catalog_returns.cr_item_sk) and (catalog_sales.cs_order_number = catalog_returns.cr_order_number)) otherCondition=()
------------------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF19
------------------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------------------PhysicalOlapScan[catalog_returns] apply RFs: RF19
--------------------------------------------------------------------------PhysicalOlapScan[store_returns] apply RFs: RF7 RF19
------------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
--------------------------------------------------------------------PhysicalProject
----------------------------------------------------------------------PhysicalOlapScan[customer]
--------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------------------------------------------PhysicalProject
------------------------------------------------------------------PhysicalOlapScan[customer]
------------------------------------------------------------------PhysicalOlapScan[customer_demographics]
----------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------------------------------PhysicalProject
--------------------------------------------------------------PhysicalOlapScan[customer_demographics]
--------------------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------------------------PhysicalProject
------------------------------------------------------------filter(d_year IN (2001, 2002))
--------------------------------------------------------------PhysicalOlapScan[date_dim]
--------------------------------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------------------------------PhysicalProject
------------------------------------------------------PhysicalOlapScan[customer_demographics]
------------------------------------------------------------filter((sale > (2 * refund)))
--------------------------------------------------------------hashAgg[GLOBAL]
----------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------------------------------------hashAgg[LOCAL]
--------------------------------------------------------------------PhysicalProject
----------------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_item_sk = catalog_returns.cr_item_sk) and (catalog_sales.cs_order_number = catalog_returns.cr_order_number)) otherCondition=()
------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF19
------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------PhysicalOlapScan[catalog_returns] apply RFs: RF19
----------------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------------------------------PhysicalProject
--------------------------------------------------------PhysicalOlapScan[date_dim]
------------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------------------PhysicalProject
----------------------------------------------------PhysicalOlapScan[customer_address]
@ -73,7 +72,8 @@ PhysicalCteAnchor ( cteId=CTEId#1 )
--------------------------------------------------PhysicalOlapScan[date_dim]
------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------------PhysicalProject
----------------------------------------------PhysicalOlapScan[date_dim]
----------------------------------------------filter(d_year IN (2001, 2002))
------------------------------------------------PhysicalOlapScan[date_dim]
--------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------PhysicalProject
------------------------------------------PhysicalOlapScan[household_demographics]

View File

@ -6,52 +6,51 @@ PhysicalResultSink
------PhysicalTopN[LOCAL_SORT]
--------PhysicalProject
----------filter(((coalesce(ws_qty, 0) > 0) OR (coalesce(cs_qty, 0) > 0)))
------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((cs.cs_customer_sk = ss.ss_customer_sk) and (cs.cs_item_sk = ss.ss_item_sk) and (cs.cs_sold_year = ss.ss_sold_year)) otherCondition=()
--------------PhysicalProject
----------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((ws.ws_customer_sk = ss.ss_customer_sk) and (ws.ws_item_sk = ss.ss_item_sk) and (ws.ws_sold_year = ss.ss_sold_year)) otherCondition=()
------------------PhysicalProject
--------------------hashAgg[GLOBAL]
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------hashAgg[LOCAL]
--------------------------PhysicalProject
----------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((store_returns.sr_ticket_number = store_sales.ss_ticket_number) and (store_sales.ss_item_sk = store_returns.sr_item_sk)) otherCondition=()
------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ss_sold_date_sk]
--------------------------------PhysicalProject
----------------------------------PhysicalOlapScan[store_sales] apply RFs: RF2
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((date_dim.d_year = 2000))
--------------------------------------PhysicalOlapScan[date_dim]
------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((ws.ws_customer_sk = ss.ss_customer_sk) and (ws.ws_item_sk = ss.ss_item_sk) and (ws.ws_sold_year = ss.ss_sold_year)) otherCondition=()
--------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((cs.cs_customer_sk = ss.ss_customer_sk) and (cs.cs_item_sk = ss.ss_item_sk) and (cs.cs_sold_year = ss.ss_sold_year)) otherCondition=()
----------------PhysicalProject
------------------hashAgg[GLOBAL]
--------------------PhysicalDistribute[DistributionSpecHash]
----------------------hashAgg[LOCAL]
------------------------PhysicalProject
--------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((store_returns.sr_ticket_number = store_sales.ss_ticket_number) and (store_sales.ss_item_sk = store_returns.sr_item_sk)) otherCondition=()
----------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ss_sold_date_sk]
------------------------------PhysicalProject
--------------------------------PhysicalOlapScan[store_returns]
------------------PhysicalProject
--------------------hashAgg[GLOBAL]
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------hashAgg[LOCAL]
--------------------------PhysicalProject
----------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((web_returns.wr_order_number = web_sales.ws_order_number) and (web_sales.ws_item_sk = web_returns.wr_item_sk)) otherCondition=()
------------------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[ws_sold_date_sk]
--------------------------------PhysicalOlapScan[store_sales] apply RFs: RF2
------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------PhysicalProject
----------------------------------PhysicalOlapScan[web_sales] apply RFs: RF1
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((date_dim.d_year = 2000))
--------------------------------------PhysicalOlapScan[date_dim]
----------------------------------filter((date_dim.d_year = 2000))
------------------------------------PhysicalOlapScan[date_dim]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[store_returns]
----------------PhysicalProject
------------------hashAgg[GLOBAL]
--------------------PhysicalDistribute[DistributionSpecHash]
----------------------hashAgg[LOCAL]
------------------------PhysicalProject
--------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((catalog_returns.cr_order_number = catalog_sales.cs_order_number) and (catalog_sales.cs_item_sk = catalog_returns.cr_item_sk)) otherCondition=()
----------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[cs_sold_date_sk]
------------------------------PhysicalProject
--------------------------------PhysicalOlapScan[web_returns]
--------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF1
------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------PhysicalProject
----------------------------------filter((date_dim.d_year = 2000))
------------------------------------PhysicalOlapScan[date_dim]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[catalog_returns]
--------------PhysicalProject
----------------hashAgg[GLOBAL]
------------------PhysicalDistribute[DistributionSpecHash]
--------------------hashAgg[LOCAL]
----------------------PhysicalProject
------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((catalog_returns.cr_order_number = catalog_sales.cs_order_number) and (catalog_sales.cs_item_sk = catalog_returns.cr_item_sk)) otherCondition=()
--------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[cs_sold_date_sk]
------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((web_returns.wr_order_number = web_sales.ws_order_number) and (web_sales.ws_item_sk = web_returns.wr_item_sk)) otherCondition=()
--------------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ws_sold_date_sk]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF0
------------------------------PhysicalOlapScan[web_sales] apply RFs: RF0
----------------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------PhysicalProject
--------------------------------filter((date_dim.d_year = 2000))
----------------------------------PhysicalOlapScan[date_dim]
--------------------------PhysicalProject
----------------------------PhysicalOlapScan[catalog_returns]
----------------------------PhysicalOlapScan[web_returns]

View File

@ -15,32 +15,32 @@ PhysicalResultSink
------------------------PhysicalDistribute[DistributionSpecHash]
--------------------------hashAgg[LOCAL]
----------------------------PhysicalProject
------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_promo_sk = promotion.p_promo_sk)) otherCondition=() build RFs:RF3 p_promo_sk->[ss_promo_sk]
------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_store_sk = store.s_store_sk)) otherCondition=()
--------------------------------PhysicalProject
----------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_store_sk = store.s_store_sk)) otherCondition=()
------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_item_sk = item.i_item_sk)) otherCondition=() build RFs:RF1 i_item_sk->[ss_item_sk]
----------------------------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((store_sales.ss_item_sk = store_returns.sr_item_sk) and (store_sales.ss_ticket_number = store_returns.sr_ticket_number)) otherCondition=()
------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_promo_sk = promotion.p_promo_sk)) otherCondition=() build RFs:RF2 p_promo_sk->[ss_promo_sk]
--------------------------------------PhysicalProject
----------------------------------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((store_sales.ss_item_sk = store_returns.sr_item_sk) and (store_sales.ss_ticket_number = store_returns.sr_ticket_number)) otherCondition=()
----------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_item_sk = item.i_item_sk)) otherCondition=() build RFs:RF1 i_item_sk->[ss_item_sk]
------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk]
--------------------------------------------PhysicalProject
----------------------------------------------PhysicalOlapScan[store_sales] apply RFs: RF0 RF1 RF3
----------------------------------------------PhysicalOlapScan[store_sales] apply RFs: RF0 RF1 RF2
--------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------------PhysicalProject
------------------------------------------------filter((date_dim.d_date <= '1998-09-27') and (date_dim.d_date >= '1998-08-28'))
--------------------------------------------------PhysicalOlapScan[date_dim]
------------------------------------------PhysicalProject
--------------------------------------------PhysicalOlapScan[store_returns]
------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------------PhysicalProject
----------------------------------------------filter((item.i_current_price > 50.00))
------------------------------------------------PhysicalOlapScan[item]
--------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------PhysicalProject
------------------------------------------filter((item.i_current_price > 50.00))
--------------------------------------------PhysicalOlapScan[item]
------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------PhysicalProject
----------------------------------------PhysicalOlapScan[store]
------------------------------------------filter((promotion.p_channel_tv = 'N'))
--------------------------------------------PhysicalOlapScan[promotion]
------------------------------------PhysicalProject
--------------------------------------PhysicalOlapScan[store_returns]
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((promotion.p_channel_tv = 'N'))
--------------------------------------PhysicalOlapScan[promotion]
------------------------------------PhysicalOlapScan[store]
--------------------PhysicalProject
----------------------hashAgg[GLOBAL]
------------------------PhysicalDistribute[DistributionSpecHash]
@ -48,26 +48,26 @@ PhysicalResultSink
----------------------------PhysicalProject
------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_promo_sk = promotion.p_promo_sk)) otherCondition=() build RFs:RF7 p_promo_sk->[cs_promo_sk]
--------------------------------PhysicalProject
----------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF6 d_date_sk->[cs_sold_date_sk]
------------------------------------PhysicalProject
--------------------------------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((catalog_sales.cs_item_sk = catalog_returns.cr_item_sk) and (catalog_sales.cs_order_number = catalog_returns.cr_order_number)) otherCondition=()
----------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_item_sk = item.i_item_sk)) otherCondition=() build RFs:RF5 i_item_sk->[cs_item_sk]
------------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_catalog_page_sk = catalog_page.cp_catalog_page_sk)) otherCondition=()
----------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_catalog_page_sk = catalog_page.cp_catalog_page_sk)) otherCondition=()
------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_item_sk = item.i_item_sk)) otherCondition=() build RFs:RF5 i_item_sk->[cs_item_sk]
--------------------------------------PhysicalProject
----------------------------------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((catalog_sales.cs_item_sk = catalog_returns.cr_item_sk) and (catalog_sales.cs_order_number = catalog_returns.cr_order_number)) otherCondition=()
------------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF4 d_date_sk->[cs_sold_date_sk]
--------------------------------------------PhysicalProject
----------------------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF5 RF6 RF7
----------------------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF4 RF5 RF7
--------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------------PhysicalProject
------------------------------------------------PhysicalOlapScan[catalog_page]
------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------------PhysicalProject
----------------------------------------------filter((item.i_current_price > 50.00))
------------------------------------------------PhysicalOlapScan[item]
------------------------------------------------filter((date_dim.d_date <= '1998-09-27') and (date_dim.d_date >= '1998-08-28'))
--------------------------------------------------PhysicalOlapScan[date_dim]
------------------------------------------PhysicalProject
--------------------------------------------PhysicalOlapScan[catalog_returns]
--------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------PhysicalProject
------------------------------------------PhysicalOlapScan[catalog_returns]
------------------------------------------filter((item.i_current_price > 50.00))
--------------------------------------------PhysicalOlapScan[item]
------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------PhysicalProject
----------------------------------------filter((date_dim.d_date <= '1998-09-27') and (date_dim.d_date >= '1998-08-28'))
------------------------------------------PhysicalOlapScan[date_dim]
----------------------------------------PhysicalOlapScan[catalog_page]
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((promotion.p_channel_tv = 'N'))

View File

@ -17,18 +17,18 @@ PhysicalResultSink
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[web_sales] apply RFs: RF2
--------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF1 ca_address_sk->[ws_ship_addr_sk]
------------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ws_ship_date_sk]
----------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[ws_ship_date_sk]
------------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF0 ca_address_sk->[ws_ship_addr_sk]
--------------------------------PhysicalProject
----------------------------------PhysicalOlapScan[web_sales] apply RFs: RF0 RF1 RF3
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((date_dim.d_date <= '2000-04-01') and (date_dim.d_date >= '2000-02-01'))
--------------------------------------PhysicalOlapScan[date_dim]
------------------------------------filter((customer_address.ca_state = 'OK'))
--------------------------------------PhysicalOlapScan[customer_address]
------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------PhysicalProject
----------------------------------filter((customer_address.ca_state = 'OK'))
------------------------------------PhysicalOlapScan[customer_address]
----------------------------------filter((date_dim.d_date <= '2000-04-01') and (date_dim.d_date >= '2000-02-01'))
------------------------------------PhysicalOlapScan[date_dim]
------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------PhysicalProject
----------------------------filter((web_site.web_company_name = 'pri'))

View File

@ -12,23 +12,23 @@ PhysicalResultSink
------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((cs1.cs_order_number = cr1.cr_order_number)) otherCondition=()
--------------------PhysicalProject
----------------------hashJoin[INNER_JOIN] hashCondition=((cs1.cs_call_center_sk = call_center.cc_call_center_sk)) otherCondition=() build RFs:RF3 cc_call_center_sk->[cs_call_center_sk]
------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((cs1.cs_order_number = cs2.cs_order_number)) otherCondition=(( not (cs_warehouse_sk = cs_warehouse_sk))) build RFs:RF2 cs_order_number->[cs_order_number]
--------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF2
--------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------hashJoin[INNER_JOIN] hashCondition=((cs1.cs_ship_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF1 ca_address_sk->[cs_ship_addr_sk]
------------------------hashJoin[INNER_JOIN] hashCondition=((cs1.cs_ship_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF2 ca_address_sk->[cs_ship_addr_sk]
--------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((cs1.cs_order_number = cs2.cs_order_number)) otherCondition=(( not (cs_warehouse_sk = cs_warehouse_sk))) build RFs:RF1 cs_order_number->[cs_order_number]
----------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------PhysicalProject
--------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF1
----------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------hashJoin[INNER_JOIN] hashCondition=((cs1.cs_ship_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[cs_ship_date_sk]
--------------------------------PhysicalProject
----------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF0 RF1 RF3
----------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF0 RF2 RF3
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((date_dim.d_date <= '2002-05-31') and (date_dim.d_date >= '2002-04-01'))
--------------------------------------PhysicalOlapScan[date_dim]
------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------PhysicalProject
----------------------------------filter((customer_address.ca_state = 'WV'))
------------------------------------PhysicalOlapScan[customer_address]
--------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------PhysicalProject
------------------------------filter((customer_address.ca_state = 'WV'))
--------------------------------PhysicalOlapScan[customer_address]
------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------PhysicalProject
----------------------------filter(cc_county IN ('Barrow County', 'Daviess County', 'Luce County', 'Richland County', 'Ziebach County'))

View File

@ -27,6 +27,9 @@ PhysicalCteAnchor ( cteId=CTEId#1 )
----------------hashJoin[INNER_JOIN] hashCondition=((expr_cast(d_week_seq1 as BIGINT) = expr_(d_week_seq2 - 53))) otherCondition=()
------------------PhysicalDistribute[DistributionSpecHash]
--------------------PhysicalProject
----------------------PhysicalCteConsumer ( cteId=CTEId#1 ) apply RFs: RF2
------------------PhysicalDistribute[DistributionSpecHash]
--------------------PhysicalProject
----------------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_week_seq = d_week_seq1)) otherCondition=() build RFs:RF1 d_week_seq->[d_week_seq]
------------------------PhysicalDistribute[DistributionSpecExecutionAny]
--------------------------PhysicalProject
@ -35,9 +38,6 @@ PhysicalCteAnchor ( cteId=CTEId#1 )
--------------------------PhysicalProject
----------------------------filter((date_dim.d_year = 1998))
------------------------------PhysicalOlapScan[date_dim]
------------------PhysicalDistribute[DistributionSpecHash]
--------------------PhysicalProject
----------------------PhysicalCteConsumer ( cteId=CTEId#1 ) apply RFs: RF2
--------------PhysicalDistribute[DistributionSpecReplicated]
----------------PhysicalProject
------------------filter((date_dim.d_year = 1999))

View File

@ -20,51 +20,50 @@ PhysicalCteAnchor ( cteId=CTEId#1 )
----------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_promo_sk = promotion.p_promo_sk)) otherCondition=() build RFs:RF13 p_promo_sk->[ss_promo_sk]
------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_hdemo_sk = hd1.hd_demo_sk)) otherCondition=() build RFs:RF12 hd_demo_sk->[ss_hdemo_sk]
--------------------------------------PhysicalProject
----------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_first_shipto_date_sk = d3.d_date_sk)) otherCondition=() build RFs:RF11 d_date_sk->[c_first_shipto_date_sk]
----------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = d1.d_date_sk)) otherCondition=() build RFs:RF11 d_date_sk->[ss_sold_date_sk]
------------------------------------------PhysicalProject
--------------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_first_sales_date_sk = d2.d_date_sk)) otherCondition=() build RFs:RF10 d_date_sk->[c_first_sales_date_sk]
--------------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_first_shipto_date_sk = d3.d_date_sk)) otherCondition=() build RFs:RF10 d_date_sk->[c_first_shipto_date_sk]
----------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_addr_sk = ad1.ca_address_sk)) otherCondition=() build RFs:RF9 ca_address_sk->[ss_addr_sk]
------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_current_cdemo_sk = cd2.cd_demo_sk)) otherCondition=(( not (cd_marital_status = cd_marital_status))) build RFs:RF8 cd_demo_sk->[c_current_cdemo_sk]
--------------------------------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------------------PhysicalProject
--------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_first_sales_date_sk = d2.d_date_sk)) otherCondition=() build RFs:RF8 d_date_sk->[c_first_sales_date_sk]
----------------------------------------------------PhysicalProject
------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = d1.d_date_sk)) otherCondition=() build RFs:RF7 d_date_sk->[ss_sold_date_sk]
--------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_cdemo_sk = cd1.cd_demo_sk)) otherCondition=() build RFs:RF6 cd_demo_sk->[ss_cdemo_sk]
------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_item_sk = cs_ui.cs_item_sk)) otherCondition=() build RFs:RF7 cs_item_sk->[sr_item_sk,ss_item_sk]
--------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((customer.c_current_cdemo_sk = cd2.cd_demo_sk)) otherCondition=(( not (cd_marital_status = cd_marital_status))) build RFs:RF6 cd_demo_sk->[c_current_cdemo_sk]
----------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_customer_sk = customer.c_customer_sk)) otherCondition=() build RFs:RF5 c_customer_sk->[ss_customer_sk]
------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_cdemo_sk = cd1.cd_demo_sk)) otherCondition=() build RFs:RF5 cd_demo_sk->[ss_cdemo_sk]
--------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------------------------------------------PhysicalProject
------------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_item_sk = cs_ui.cs_item_sk)) otherCondition=() build RFs:RF4 cs_item_sk->[sr_item_sk,ss_item_sk]
----------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_customer_sk = customer.c_customer_sk)) otherCondition=() build RFs:RF4 c_customer_sk->[ss_customer_sk]
------------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
--------------------------------------------------------------------PhysicalProject
----------------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_item_sk = store_returns.sr_item_sk) and (store_sales.ss_ticket_number = store_returns.sr_ticket_number)) otherCondition=() build RFs:RF2 sr_item_sk->[ss_item_sk];RF3 sr_ticket_number->[ss_ticket_number]
------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------PhysicalOlapScan[store_sales] apply RFs: RF2 RF3 RF4 RF5 RF6 RF7 RF9 RF12 RF13 RF16 RF19
--------------------------------------------------------------------------PhysicalOlapScan[store_sales] apply RFs: RF2 RF3 RF4 RF5 RF7 RF9 RF11 RF12 RF13 RF16 RF19
------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------PhysicalOlapScan[store_returns] apply RFs: RF4 RF19
--------------------------------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------------------------------------PhysicalProject
------------------------------------------------------------------------filter((sale > (2 * refund)))
--------------------------------------------------------------------------hashAgg[GLOBAL]
----------------------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------------------------------------------------hashAgg[LOCAL]
--------------------------------------------------------------------------------PhysicalProject
----------------------------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_item_sk = catalog_returns.cr_item_sk) and (catalog_sales.cs_order_number = catalog_returns.cr_order_number)) otherCondition=() build RFs:RF0 cr_item_sk->[cs_item_sk];RF1 cr_order_number->[cs_order_number]
------------------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF0 RF1 RF19
------------------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------------------PhysicalOlapScan[catalog_returns] apply RFs: RF19
--------------------------------------------------------------------------PhysicalOlapScan[store_returns] apply RFs: RF7 RF19
------------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
--------------------------------------------------------------------PhysicalProject
----------------------------------------------------------------------PhysicalOlapScan[customer] apply RFs: RF6 RF8 RF10 RF14 RF15
--------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------------------------------------------PhysicalProject
------------------------------------------------------------------PhysicalOlapScan[customer] apply RFs: RF8 RF10 RF11 RF14 RF15
------------------------------------------------------------------PhysicalOlapScan[customer_demographics]
----------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------------------------------PhysicalProject
--------------------------------------------------------------PhysicalOlapScan[customer_demographics]
--------------------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------------------------PhysicalProject
------------------------------------------------------------filter(d_year IN (2001, 2002))
--------------------------------------------------------------PhysicalOlapScan[date_dim]
--------------------------------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------------------------------PhysicalProject
------------------------------------------------------PhysicalOlapScan[customer_demographics]
------------------------------------------------------------filter((sale > (2 * refund)))
--------------------------------------------------------------hashAgg[GLOBAL]
----------------------------------------------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------------------------------------hashAgg[LOCAL]
--------------------------------------------------------------------PhysicalProject
----------------------------------------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_item_sk = catalog_returns.cr_item_sk) and (catalog_sales.cs_order_number = catalog_returns.cr_order_number)) otherCondition=() build RFs:RF0 cr_item_sk->[cs_item_sk];RF1 cr_order_number->[cs_order_number]
------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF0 RF1 RF19
------------------------------------------------------------------------PhysicalProject
--------------------------------------------------------------------------PhysicalOlapScan[catalog_returns] apply RFs: RF19
----------------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------------------------------PhysicalProject
--------------------------------------------------------PhysicalOlapScan[date_dim]
------------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------------------PhysicalProject
----------------------------------------------------PhysicalOlapScan[customer_address]
@ -73,7 +72,8 @@ PhysicalCteAnchor ( cteId=CTEId#1 )
--------------------------------------------------PhysicalOlapScan[date_dim]
------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------------PhysicalProject
----------------------------------------------PhysicalOlapScan[date_dim]
----------------------------------------------filter(d_year IN (2001, 2002))
------------------------------------------------PhysicalOlapScan[date_dim]
--------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------PhysicalProject
------------------------------------------PhysicalOlapScan[household_demographics] apply RFs: RF17

View File

@ -6,52 +6,51 @@ PhysicalResultSink
------PhysicalTopN[LOCAL_SORT]
--------PhysicalProject
----------filter(((coalesce(ws_qty, 0) > 0) OR (coalesce(cs_qty, 0) > 0)))
------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((cs.cs_customer_sk = ss.ss_customer_sk) and (cs.cs_item_sk = ss.ss_item_sk) and (cs.cs_sold_year = ss.ss_sold_year)) otherCondition=()
--------------PhysicalProject
----------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((ws.ws_customer_sk = ss.ss_customer_sk) and (ws.ws_item_sk = ss.ss_item_sk) and (ws.ws_sold_year = ss.ss_sold_year)) otherCondition=()
------------------PhysicalProject
--------------------hashAgg[GLOBAL]
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------hashAgg[LOCAL]
--------------------------PhysicalProject
----------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((store_returns.sr_ticket_number = store_sales.ss_ticket_number) and (store_sales.ss_item_sk = store_returns.sr_item_sk)) otherCondition=()
------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ss_sold_date_sk]
--------------------------------PhysicalProject
----------------------------------PhysicalOlapScan[store_sales] apply RFs: RF2
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((date_dim.d_year = 2000))
--------------------------------------PhysicalOlapScan[date_dim]
------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((ws.ws_customer_sk = ss.ss_customer_sk) and (ws.ws_item_sk = ss.ss_item_sk) and (ws.ws_sold_year = ss.ss_sold_year)) otherCondition=()
--------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((cs.cs_customer_sk = ss.ss_customer_sk) and (cs.cs_item_sk = ss.ss_item_sk) and (cs.cs_sold_year = ss.ss_sold_year)) otherCondition=()
----------------PhysicalProject
------------------hashAgg[GLOBAL]
--------------------PhysicalDistribute[DistributionSpecHash]
----------------------hashAgg[LOCAL]
------------------------PhysicalProject
--------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((store_returns.sr_ticket_number = store_sales.ss_ticket_number) and (store_sales.ss_item_sk = store_returns.sr_item_sk)) otherCondition=()
----------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ss_sold_date_sk]
------------------------------PhysicalProject
--------------------------------PhysicalOlapScan[store_returns]
------------------PhysicalProject
--------------------hashAgg[GLOBAL]
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------hashAgg[LOCAL]
--------------------------PhysicalProject
----------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((web_returns.wr_order_number = web_sales.ws_order_number) and (web_sales.ws_item_sk = web_returns.wr_item_sk)) otherCondition=()
------------------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[ws_sold_date_sk]
--------------------------------PhysicalOlapScan[store_sales] apply RFs: RF2
------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------PhysicalProject
----------------------------------PhysicalOlapScan[web_sales] apply RFs: RF1
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((date_dim.d_year = 2000))
--------------------------------------PhysicalOlapScan[date_dim]
----------------------------------filter((date_dim.d_year = 2000))
------------------------------------PhysicalOlapScan[date_dim]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[store_returns]
----------------PhysicalProject
------------------hashAgg[GLOBAL]
--------------------PhysicalDistribute[DistributionSpecHash]
----------------------hashAgg[LOCAL]
------------------------PhysicalProject
--------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((catalog_returns.cr_order_number = catalog_sales.cs_order_number) and (catalog_sales.cs_item_sk = catalog_returns.cr_item_sk)) otherCondition=()
----------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[cs_sold_date_sk]
------------------------------PhysicalProject
--------------------------------PhysicalOlapScan[web_returns]
--------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF1
------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------PhysicalProject
----------------------------------filter((date_dim.d_year = 2000))
------------------------------------PhysicalOlapScan[date_dim]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[catalog_returns]
--------------PhysicalProject
----------------hashAgg[GLOBAL]
------------------PhysicalDistribute[DistributionSpecHash]
--------------------hashAgg[LOCAL]
----------------------PhysicalProject
------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((catalog_returns.cr_order_number = catalog_sales.cs_order_number) and (catalog_sales.cs_item_sk = catalog_returns.cr_item_sk)) otherCondition=()
--------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[cs_sold_date_sk]
------------------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((web_returns.wr_order_number = web_sales.ws_order_number) and (web_sales.ws_item_sk = web_returns.wr_item_sk)) otherCondition=()
--------------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ws_sold_date_sk]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF0
------------------------------PhysicalOlapScan[web_sales] apply RFs: RF0
----------------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------PhysicalProject
--------------------------------filter((date_dim.d_year = 2000))
----------------------------------PhysicalOlapScan[date_dim]
--------------------------PhysicalProject
----------------------------PhysicalOlapScan[catalog_returns]
----------------------------PhysicalOlapScan[web_returns]

View File

@ -15,12 +15,12 @@ PhysicalResultSink
------------------------PhysicalDistribute[DistributionSpecHash]
--------------------------hashAgg[LOCAL]
----------------------------PhysicalProject
------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_promo_sk = promotion.p_promo_sk)) otherCondition=() build RFs:RF3 p_promo_sk->[ss_promo_sk]
------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_store_sk = store.s_store_sk)) otherCondition=() build RFs:RF3 s_store_sk->[ss_store_sk]
--------------------------------PhysicalProject
----------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_store_sk = store.s_store_sk)) otherCondition=() build RFs:RF2 s_store_sk->[ss_store_sk]
------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_item_sk = item.i_item_sk)) otherCondition=() build RFs:RF1 i_item_sk->[ss_item_sk]
----------------------------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((store_sales.ss_item_sk = store_returns.sr_item_sk) and (store_sales.ss_ticket_number = store_returns.sr_ticket_number)) otherCondition=()
------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_promo_sk = promotion.p_promo_sk)) otherCondition=() build RFs:RF2 p_promo_sk->[ss_promo_sk]
--------------------------------------PhysicalProject
----------------------------------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((store_sales.ss_item_sk = store_returns.sr_item_sk) and (store_sales.ss_ticket_number = store_returns.sr_ticket_number)) otherCondition=()
----------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_item_sk = item.i_item_sk)) otherCondition=() build RFs:RF1 i_item_sk->[ss_item_sk]
------------------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk]
--------------------------------------------PhysicalProject
----------------------------------------------PhysicalOlapScan[store_sales] apply RFs: RF0 RF1 RF2 RF3
@ -28,19 +28,19 @@ PhysicalResultSink
----------------------------------------------PhysicalProject
------------------------------------------------filter((date_dim.d_date <= '1998-09-27') and (date_dim.d_date >= '1998-08-28'))
--------------------------------------------------PhysicalOlapScan[date_dim]
------------------------------------------PhysicalProject
--------------------------------------------PhysicalOlapScan[store_returns]
------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------------PhysicalProject
----------------------------------------------filter((item.i_current_price > 50.00))
------------------------------------------------PhysicalOlapScan[item]
--------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------PhysicalProject
------------------------------------------filter((item.i_current_price > 50.00))
--------------------------------------------PhysicalOlapScan[item]
------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------PhysicalProject
----------------------------------------PhysicalOlapScan[store]
------------------------------------------filter((promotion.p_channel_tv = 'N'))
--------------------------------------------PhysicalOlapScan[promotion]
------------------------------------PhysicalProject
--------------------------------------PhysicalOlapScan[store_returns]
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((promotion.p_channel_tv = 'N'))
--------------------------------------PhysicalOlapScan[promotion]
------------------------------------PhysicalOlapScan[store]
--------------------PhysicalProject
----------------------hashAgg[GLOBAL]
------------------------PhysicalDistribute[DistributionSpecHash]
@ -48,26 +48,26 @@ PhysicalResultSink
----------------------------PhysicalProject
------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_promo_sk = promotion.p_promo_sk)) otherCondition=() build RFs:RF7 p_promo_sk->[cs_promo_sk]
--------------------------------PhysicalProject
----------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF6 d_date_sk->[cs_sold_date_sk]
------------------------------------PhysicalProject
--------------------------------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((catalog_sales.cs_item_sk = catalog_returns.cr_item_sk) and (catalog_sales.cs_order_number = catalog_returns.cr_order_number)) otherCondition=()
----------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_item_sk = item.i_item_sk)) otherCondition=() build RFs:RF5 i_item_sk->[cs_item_sk]
------------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_catalog_page_sk = catalog_page.cp_catalog_page_sk)) otherCondition=() build RFs:RF4 cp_catalog_page_sk->[cs_catalog_page_sk]
----------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_catalog_page_sk = catalog_page.cp_catalog_page_sk)) otherCondition=() build RFs:RF6 cp_catalog_page_sk->[cs_catalog_page_sk]
------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_item_sk = item.i_item_sk)) otherCondition=() build RFs:RF5 i_item_sk->[cs_item_sk]
--------------------------------------PhysicalProject
----------------------------------------hashJoin[LEFT_OUTER_JOIN] hashCondition=((catalog_sales.cs_item_sk = catalog_returns.cr_item_sk) and (catalog_sales.cs_order_number = catalog_returns.cr_order_number)) otherCondition=()
------------------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF4 d_date_sk->[cs_sold_date_sk]
--------------------------------------------PhysicalProject
----------------------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF4 RF5 RF6 RF7
--------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------------PhysicalProject
------------------------------------------------PhysicalOlapScan[catalog_page]
------------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------------PhysicalProject
----------------------------------------------filter((item.i_current_price > 50.00))
------------------------------------------------PhysicalOlapScan[item]
------------------------------------------------filter((date_dim.d_date <= '1998-09-27') and (date_dim.d_date >= '1998-08-28'))
--------------------------------------------------PhysicalOlapScan[date_dim]
------------------------------------------PhysicalProject
--------------------------------------------PhysicalOlapScan[catalog_returns]
--------------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------------PhysicalProject
------------------------------------------PhysicalOlapScan[catalog_returns]
------------------------------------------filter((item.i_current_price > 50.00))
--------------------------------------------PhysicalOlapScan[item]
------------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------------PhysicalProject
----------------------------------------filter((date_dim.d_date <= '1998-09-27') and (date_dim.d_date >= '1998-08-28'))
------------------------------------------PhysicalOlapScan[date_dim]
----------------------------------------PhysicalOlapScan[catalog_page]
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((promotion.p_channel_tv = 'N'))

View File

@ -17,18 +17,18 @@ PhysicalResultSink
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[web_sales] apply RFs: RF2
--------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF1 ca_address_sk->[ws_ship_addr_sk]
------------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ws_ship_date_sk]
----------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[ws_ship_date_sk]
------------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF0 ca_address_sk->[ws_ship_addr_sk]
--------------------------------PhysicalProject
----------------------------------PhysicalOlapScan[web_sales] apply RFs: RF0 RF1 RF3
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((date_dim.d_date <= '2000-04-01') and (date_dim.d_date >= '2000-02-01'))
--------------------------------------PhysicalOlapScan[date_dim]
------------------------------------filter((customer_address.ca_state = 'OK'))
--------------------------------------PhysicalOlapScan[customer_address]
------------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------------PhysicalProject
----------------------------------filter((customer_address.ca_state = 'OK'))
------------------------------------PhysicalOlapScan[customer_address]
----------------------------------filter((date_dim.d_date <= '2000-04-01') and (date_dim.d_date >= '2000-02-01'))
------------------------------------PhysicalOlapScan[date_dim]
------------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------------PhysicalProject
----------------------------filter((web_site.web_company_name = 'pri'))

View File

@ -10,33 +10,33 @@ PhysicalResultSink
--------------hashAgg[LOCAL]
----------------PhysicalProject
------------------filter((ifnull($c$1, FALSE) OR ifnull($c$2, FALSE)))
--------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = store_sales.ss_customer_sk)) otherCondition=() build RFs:RF5 c_customer_sk->[ss_customer_sk]
--------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = catalog_sales.cs_ship_customer_sk)) otherCondition=()
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------PhysicalProject
--------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF4 d_date_sk->[ss_sold_date_sk]
--------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF5 d_date_sk->[cs_sold_date_sk]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[store_sales] apply RFs: RF4 RF5
------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF5
----------------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------PhysicalProject
--------------------------------filter((date_dim.d_moy <= 4) and (date_dim.d_moy >= 1) and (date_dim.d_year = 2001))
----------------------------------PhysicalOlapScan[date_dim]
----------------------PhysicalProject
------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = web_sales.ws_bill_customer_sk)) otherCondition=()
------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = store_sales.ss_customer_sk)) otherCondition=() build RFs:RF4 c_customer_sk->[ss_customer_sk]
--------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------PhysicalProject
------------------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF3 d_date_sk->[ws_sold_date_sk]
------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF3 d_date_sk->[ss_sold_date_sk]
--------------------------------PhysicalProject
----------------------------------PhysicalOlapScan[web_sales] apply RFs: RF3
----------------------------------PhysicalOlapScan[store_sales] apply RFs: RF3 RF4
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((date_dim.d_moy <= 4) and (date_dim.d_moy >= 1) and (date_dim.d_year = 2001))
--------------------------------------PhysicalOlapScan[date_dim]
--------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = catalog_sales.cs_ship_customer_sk)) otherCondition=()
--------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = web_sales.ws_bill_customer_sk)) otherCondition=()
----------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------PhysicalProject
--------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[cs_sold_date_sk]
--------------------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ws_sold_date_sk]
----------------------------------PhysicalProject
------------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF2
------------------------------------PhysicalOlapScan[web_sales] apply RFs: RF2
----------------------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------------PhysicalProject
--------------------------------------filter((date_dim.d_moy <= 4) and (date_dim.d_moy >= 1) and (date_dim.d_year = 2001))

View File

@ -10,33 +10,33 @@ PhysicalResultSink
--------------hashAgg[LOCAL]
----------------PhysicalProject
------------------filter((ifnull($c$1, FALSE) OR ifnull($c$2, FALSE)))
--------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = store_sales.ss_customer_sk)) otherCondition=() build RFs:RF5 c_customer_sk->[ss_customer_sk]
--------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = catalog_sales.cs_ship_customer_sk)) otherCondition=()
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------PhysicalProject
--------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF4 d_date_sk->[ss_sold_date_sk]
--------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF5 d_date_sk->[cs_sold_date_sk]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[store_sales] apply RFs: RF4 RF5
------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF5
----------------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------PhysicalProject
--------------------------------filter((date_dim.d_moy <= 4) and (date_dim.d_moy >= 1) and (date_dim.d_year = 2001))
----------------------------------PhysicalOlapScan[date_dim]
----------------------PhysicalProject
------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = web_sales.ws_bill_customer_sk)) otherCondition=()
------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = store_sales.ss_customer_sk)) otherCondition=() build RFs:RF4 c_customer_sk->[ss_customer_sk]
--------------------------PhysicalDistribute[DistributionSpecHash]
----------------------------PhysicalProject
------------------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF3 d_date_sk->[ws_sold_date_sk]
------------------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF3 d_date_sk->[ss_sold_date_sk]
--------------------------------PhysicalProject
----------------------------------PhysicalOlapScan[web_sales] apply RFs: RF3
----------------------------------PhysicalOlapScan[store_sales] apply RFs: RF3 RF4
--------------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------------PhysicalProject
------------------------------------filter((date_dim.d_moy <= 4) and (date_dim.d_moy >= 1) and (date_dim.d_year = 2001))
--------------------------------------PhysicalOlapScan[date_dim]
--------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = catalog_sales.cs_ship_customer_sk)) otherCondition=()
--------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((c.c_customer_sk = web_sales.ws_bill_customer_sk)) otherCondition=()
----------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------PhysicalProject
--------------------------------hashJoin[INNER_JOIN] hashCondition=((catalog_sales.cs_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[cs_sold_date_sk]
--------------------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ws_sold_date_sk]
----------------------------------PhysicalProject
------------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF2
------------------------------------PhysicalOlapScan[web_sales] apply RFs: RF2
----------------------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------------PhysicalProject
--------------------------------------filter((date_dim.d_moy <= 4) and (date_dim.d_moy >= 1) and (date_dim.d_year = 2001))

View File

@ -12,20 +12,20 @@ PhysicalResultSink
------------------PhysicalProject
--------------------hashJoin[INNER_JOIN] hashCondition=((supplier.s_suppkey = l1.l_suppkey)) otherCondition=() build RFs:RF3 s_suppkey->[l_suppkey]
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------hashJoin[INNER_JOIN] hashCondition=((orders.o_orderkey = l1.l_orderkey)) otherCondition=() build RFs:RF2 o_orderkey->[l_orderkey,l_orderkey]
--------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((l2.l_orderkey = l1.l_orderkey)) otherCondition=(( not (l_suppkey = l_suppkey))) build RFs:RF1 l_orderkey->[l_orderkey]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[lineitem] apply RFs: RF1 RF2
------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((l2.l_orderkey = l1.l_orderkey)) otherCondition=(( not (l_suppkey = l_suppkey))) build RFs:RF2 l_orderkey->[l_orderkey]
--------------------------PhysicalProject
----------------------------PhysicalOlapScan[lineitem] apply RFs: RF2
--------------------------hashJoin[INNER_JOIN] hashCondition=((orders.o_orderkey = l1.l_orderkey)) otherCondition=() build RFs:RF1 o_orderkey->[l_orderkey]
----------------------------hashJoin[RIGHT_ANTI_JOIN] hashCondition=((l3.l_orderkey = l1.l_orderkey)) otherCondition=(( not (l_suppkey = l_suppkey))) build RFs:RF0 l_orderkey->[l_orderkey]
------------------------------PhysicalProject
--------------------------------filter((l3.l_receiptdate > l3.l_commitdate))
----------------------------------PhysicalOlapScan[lineitem] apply RFs: RF0
------------------------------PhysicalProject
--------------------------------filter((l1.l_receiptdate > l1.l_commitdate))
----------------------------------PhysicalOlapScan[lineitem] apply RFs: RF2 RF3
--------------------------PhysicalProject
----------------------------filter((orders.o_orderstatus = 'F'))
------------------------------PhysicalOlapScan[orders]
----------------------------------PhysicalOlapScan[lineitem] apply RFs: RF1 RF3
----------------------------PhysicalProject
------------------------------filter((orders.o_orderstatus = 'F'))
--------------------------------PhysicalOlapScan[orders]
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------PhysicalProject
--------------------------PhysicalOlapScan[supplier] apply RFs: RF4

View File

@ -10,9 +10,9 @@ PhysicalResultSink
--------------PhysicalProject
----------------hashJoin[INNER_JOIN] hashCondition=((supplier.s_nationkey = nation.n_nationkey)) otherCondition=()
------------------PhysicalProject
--------------------hashJoin[INNER_JOIN] hashCondition=((supplier.s_suppkey = lineitem.l_suppkey)) otherCondition=()
--------------------hashJoin[INNER_JOIN] hashCondition=((partsupp.ps_partkey = lineitem.l_partkey) and (partsupp.ps_suppkey = lineitem.l_suppkey)) otherCondition=()
----------------------PhysicalProject
------------------------hashJoin[INNER_JOIN] hashCondition=((partsupp.ps_partkey = lineitem.l_partkey) and (partsupp.ps_suppkey = lineitem.l_suppkey)) otherCondition=()
------------------------hashJoin[INNER_JOIN] hashCondition=((supplier.s_suppkey = lineitem.l_suppkey)) otherCondition=()
--------------------------PhysicalProject
----------------------------hashJoin[INNER_JOIN] hashCondition=((part.p_partkey = lineitem.l_partkey)) otherCondition=() build RFs:RF1 p_partkey->[l_partkey]
------------------------------PhysicalDistribute[DistributionSpecHash]
@ -25,12 +25,12 @@ PhysicalResultSink
--------------------------------PhysicalProject
----------------------------------filter((p_name like '%green%'))
------------------------------------PhysicalOlapScan[part]
--------------------------PhysicalDistribute[DistributionSpecHash]
--------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[partsupp]
----------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------PhysicalOlapScan[supplier]
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------PhysicalProject
--------------------------PhysicalOlapScan[supplier]
--------------------------PhysicalOlapScan[partsupp]
------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------PhysicalProject
----------------------PhysicalOlapScan[nation]

View File

@ -12,20 +12,20 @@ PhysicalResultSink
------------------PhysicalProject
--------------------hashJoin[INNER_JOIN] hashCondition=((supplier.s_suppkey = l1.l_suppkey)) otherCondition=() build RFs:RF3 s_suppkey->[l_suppkey]
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------hashJoin[INNER_JOIN] hashCondition=((orders.o_orderkey = l1.l_orderkey)) otherCondition=() build RFs:RF2 o_orderkey->[l_orderkey,l_orderkey]
--------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((l2.l_orderkey = l1.l_orderkey)) otherCondition=(( not (l_suppkey = l_suppkey))) build RFs:RF1 l_orderkey->[l_orderkey]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[lineitem] apply RFs: RF1 RF2
------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((l2.l_orderkey = l1.l_orderkey)) otherCondition=(( not (l_suppkey = l_suppkey))) build RFs:RF2 l_orderkey->[l_orderkey]
--------------------------PhysicalProject
----------------------------PhysicalOlapScan[lineitem] apply RFs: RF2
--------------------------hashJoin[INNER_JOIN] hashCondition=((orders.o_orderkey = l1.l_orderkey)) otherCondition=() build RFs:RF1 o_orderkey->[l_orderkey]
----------------------------hashJoin[RIGHT_ANTI_JOIN] hashCondition=((l3.l_orderkey = l1.l_orderkey)) otherCondition=(( not (l_suppkey = l_suppkey))) build RFs:RF0 l_orderkey->[l_orderkey]
------------------------------PhysicalProject
--------------------------------filter((l3.l_receiptdate > l3.l_commitdate))
----------------------------------PhysicalOlapScan[lineitem] apply RFs: RF0
------------------------------PhysicalProject
--------------------------------filter((l1.l_receiptdate > l1.l_commitdate))
----------------------------------PhysicalOlapScan[lineitem] apply RFs: RF2 RF3
--------------------------PhysicalProject
----------------------------filter((orders.o_orderstatus = 'F'))
------------------------------PhysicalOlapScan[orders]
----------------------------------PhysicalOlapScan[lineitem] apply RFs: RF1 RF3
----------------------------PhysicalProject
------------------------------filter((orders.o_orderstatus = 'F'))
--------------------------------PhysicalOlapScan[orders]
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------PhysicalProject
--------------------------PhysicalOlapScan[supplier] apply RFs: RF4

View File

@ -10,9 +10,9 @@ PhysicalResultSink
--------------PhysicalProject
----------------hashJoin[INNER_JOIN] hashCondition=((supplier.s_nationkey = nation.n_nationkey)) otherCondition=() build RFs:RF5 n_nationkey->[s_nationkey]
------------------PhysicalProject
--------------------hashJoin[INNER_JOIN] hashCondition=((supplier.s_suppkey = lineitem.l_suppkey)) otherCondition=() build RFs:RF4 s_suppkey->[l_suppkey,ps_suppkey]
--------------------hashJoin[INNER_JOIN] hashCondition=((partsupp.ps_partkey = lineitem.l_partkey) and (partsupp.ps_suppkey = lineitem.l_suppkey)) otherCondition=() build RFs:RF3 ps_suppkey->[l_suppkey,s_suppkey];RF4 ps_partkey->[l_partkey,p_partkey]
----------------------PhysicalProject
------------------------hashJoin[INNER_JOIN] hashCondition=((partsupp.ps_partkey = lineitem.l_partkey) and (partsupp.ps_suppkey = lineitem.l_suppkey)) otherCondition=() build RFs:RF2 ps_suppkey->[l_suppkey];RF3 ps_partkey->[l_partkey,p_partkey]
------------------------hashJoin[INNER_JOIN] hashCondition=((supplier.s_suppkey = lineitem.l_suppkey)) otherCondition=() build RFs:RF2 s_suppkey->[l_suppkey]
--------------------------PhysicalProject
----------------------------hashJoin[INNER_JOIN] hashCondition=((part.p_partkey = lineitem.l_partkey)) otherCondition=() build RFs:RF1 p_partkey->[l_partkey]
------------------------------PhysicalDistribute[DistributionSpecHash]
@ -24,13 +24,13 @@ PhysicalResultSink
------------------------------PhysicalDistribute[DistributionSpecHash]
--------------------------------PhysicalProject
----------------------------------filter((p_name like '%green%'))
------------------------------------PhysicalOlapScan[part] apply RFs: RF3
--------------------------PhysicalDistribute[DistributionSpecHash]
------------------------------------PhysicalOlapScan[part] apply RFs: RF4
--------------------------PhysicalDistribute[DistributionSpecReplicated]
----------------------------PhysicalProject
------------------------------PhysicalOlapScan[partsupp] apply RFs: RF4
----------------------PhysicalDistribute[DistributionSpecReplicated]
------------------------------PhysicalOlapScan[supplier] apply RFs: RF3 RF5
----------------------PhysicalDistribute[DistributionSpecHash]
------------------------PhysicalProject
--------------------------PhysicalOlapScan[supplier] apply RFs: RF5
--------------------------PhysicalOlapScan[partsupp]
------------------PhysicalDistribute[DistributionSpecReplicated]
--------------------PhysicalProject
----------------------PhysicalOlapScan[nation]