Skip to content

Commit

Permalink
Iterative optimizer
Browse files Browse the repository at this point in the history
This optimizer decouples the traversal of the plan tree (IterativeOptimizer)
from the transformation logic (Rule). The optimization loop applies rules
recursively until a fixpoint is reached.

It's implemented as PlanOptimizer so that it fits right into the existing
framework.
  • Loading branch information
martint committed Dec 22, 2016
1 parent 2831afc commit 13e6b6a
Show file tree
Hide file tree
Showing 11 changed files with 687 additions and 1 deletion.
Expand Up @@ -67,6 +67,7 @@ public final class SystemSessionProperties
public static final String OPTIMIZE_DISTINCT_AGGREGATIONS = "optimize_mixed_distinct_aggregations";
public static final String LEGACY_ORDER_BY = "legacy_order_by";
public static final String REORDER_WINDOWS = "reorder_windows";
public static final String NEW_OPTIMIZER = "new_optimizer_enabled";

private final List<PropertyMetadata<?>> sessionProperties;

Expand Down Expand Up @@ -271,6 +272,11 @@ public SystemSessionProperties(
REORDER_WINDOWS,
"Allow reordering window functions in query",
featuresConfig.isReorderWindows(),
false),
booleanSessionProperty(
NEW_OPTIMIZER,
"Experimental: enable new optimizer",
featuresConfig.isNewOptimizerEnabled(),
false));
}

Expand Down Expand Up @@ -427,4 +433,9 @@ public static boolean isLegacyOrderByEnabled(Session session)
{
return session.getSystemProperty(LEGACY_ORDER_BY, Boolean.class);
}

public static boolean isNewOptimizerEnabled(Session session)
{
return session.getSystemProperty(NEW_OPTIMIZER, Boolean.class);
}
}
Expand Up @@ -68,6 +68,7 @@ public static class ProcessingOptimization
private DataSize operatorMemoryLimitBeforeSpill = new DataSize(4, DataSize.Unit.MEGABYTE);
private Path spillerSpillPath = Paths.get(System.getProperty("java.io.tmpdir"), "presto", "spills");
private int spillerThreads = 4;
private boolean newOptimizerEnabled;

public boolean isResourceGroupsEnabled()
{
Expand Down Expand Up @@ -304,6 +305,18 @@ public FeaturesConfig setSpillEnabled(boolean spillEnabled)
return this;
}

public boolean isNewOptimizerEnabled()
{
return newOptimizerEnabled;
}

@Config("experimental.new-optimizer-enabled")
public FeaturesConfig setNewOptimizerEnabled(boolean value)
{
this.newOptimizerEnabled = value;
return this;
}

public DataSize getOperatorMemoryLimitBeforeSpill()
{
return operatorMemoryLimitBeforeSpill;
Expand Down
Expand Up @@ -16,6 +16,7 @@
import com.facebook.presto.metadata.Metadata;
import com.facebook.presto.sql.analyzer.FeaturesConfig;
import com.facebook.presto.sql.parser.SqlParser;
import com.facebook.presto.sql.planner.iterative.IterativeOptimizer;
import com.facebook.presto.sql.planner.optimizations.AddExchanges;
import com.facebook.presto.sql.planner.optimizations.AddLocalExchanges;
import com.facebook.presto.sql.planner.optimizations.BeginTableWrite;
Expand Down Expand Up @@ -56,6 +57,7 @@
import com.facebook.presto.sql.planner.optimizations.UnaliasSymbolReferences;
import com.facebook.presto.sql.planner.optimizations.WindowFilterPushDown;
import com.google.common.collect.ImmutableList;
import com.google.common.collect.ImmutableSet;
import com.google.inject.Inject;

import java.util.List;
Expand All @@ -77,6 +79,7 @@ public PlanOptimizers(Metadata metadata, SqlParser sqlParser, FeaturesConfig fea
builder.add(
new DesugaringOptimizer(metadata, sqlParser), // Clean up all the sugar in expressions, e.g. AtTimeZone, must be run before all the other optimizers
new CanonicalizeExpressions(),
new IterativeOptimizer(ImmutableSet.of()),
new ImplementFilteredAggregations(),
new ImplementSampleAsFilter(),
new SimplifyExpressions(metadata, sqlParser),
Expand Down
@@ -0,0 +1,52 @@
/*
* Licensed 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 com.facebook.presto.sql.planner.iterative;

import com.facebook.presto.sql.planner.Symbol;
import com.facebook.presto.sql.planner.plan.PlanNode;
import com.facebook.presto.sql.planner.plan.PlanNodeId;
import com.google.common.collect.ImmutableList;

import java.util.List;

public class GroupReference
extends PlanNode
{
private final int groupId;
private final List<Symbol> outputs;

public GroupReference(PlanNodeId id, int groupId, List<Symbol> outputs)
{
super(id);
this.groupId = groupId;
this.outputs = ImmutableList.copyOf(outputs);
}

public int getGroupId()
{
return groupId;
}

@Override
public List<PlanNode> getSources()
{
return ImmutableList.of();
}

@Override
public List<Symbol> getOutputSymbols()
{
return outputs;
}
}
@@ -0,0 +1,129 @@
/*
* Licensed 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 com.facebook.presto.sql.planner.iterative;

import com.facebook.presto.Session;
import com.facebook.presto.SystemSessionProperties;
import com.facebook.presto.spi.type.Type;
import com.facebook.presto.sql.planner.PlanNodeIdAllocator;
import com.facebook.presto.sql.planner.Symbol;
import com.facebook.presto.sql.planner.SymbolAllocator;
import com.facebook.presto.sql.planner.optimizations.PlanOptimizer;
import com.facebook.presto.sql.planner.plan.PlanNode;
import com.google.common.collect.ImmutableSet;

import java.util.Map;
import java.util.Optional;
import java.util.Set;

import static com.google.common.base.Preconditions.checkState;

public class IterativeOptimizer
implements PlanOptimizer
{
private final Set<Rule> rules;

public IterativeOptimizer(Set<Rule> rules)
{
this.rules = ImmutableSet.copyOf(rules);
}

@Override
public PlanNode optimize(PlanNode plan, Session session, Map<Symbol, Type> types, SymbolAllocator symbolAllocator, PlanNodeIdAllocator idAllocator)
{
if (!SystemSessionProperties.isNewOptimizerEnabled(session)) {
return plan;
}

Memo memo = new Memo(idAllocator, plan);

Lookup lookup = node -> {
if (node instanceof GroupReference) {
return memo.getNode(((GroupReference) node).getGroupId());
}

return node;
};

exploreGroup(memo, lookup, idAllocator, symbolAllocator, memo.getRootGroup());

return memo.extract();
}

private boolean exploreGroup(Memo memo, Lookup lookup, PlanNodeIdAllocator idAllocator, SymbolAllocator symbolAllocator, int group)
{
// tracks whether this group or any children groups change as
// this method executes
boolean progress = exploreNode(memo, lookup, idAllocator, symbolAllocator, group);

while (exploreChildren(memo, lookup, idAllocator, symbolAllocator, group)) {
progress = true;

// if children changed, try current group again
// in case we can match additional rules
if (!exploreNode(memo, lookup, idAllocator, symbolAllocator, group)) {
// no additional matches, so bail out
break;
}
}

return progress;
}

private boolean exploreNode(
Memo memo,
Lookup lookup,
PlanNodeIdAllocator idAllocator,
SymbolAllocator symbolAllocator,
int group)
{
PlanNode node = memo.getNode(group);

boolean done = false;
boolean progress = false;

while (!done) {
done = true;
for (Rule rule : rules) {
Optional<PlanNode> transformed = rule.apply(node, lookup, idAllocator, symbolAllocator);

if (transformed.isPresent()) {
memo.replace(group, transformed.get(), rule.getClass().getName());
node = transformed.get();

done = false;
progress = true;
}
}
}

return progress;
}

private boolean exploreChildren(Memo memo, Lookup lookup, PlanNodeIdAllocator idAllocator, SymbolAllocator symbolAllocator, int group)
{
boolean progress = false;

PlanNode expression = memo.getNode(group);
for (PlanNode child : expression.getSources()) {
checkState(child instanceof GroupReference, "Expected child to be a group reference. Found: " + child.getClass().getName());

if (exploreGroup(memo, lookup, idAllocator, symbolAllocator, ((GroupReference) child).getGroupId())) {
progress = true;
}
}

return progress;
}
}
@@ -0,0 +1,21 @@
/*
* Licensed 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 com.facebook.presto.sql.planner.iterative;

import com.facebook.presto.sql.planner.plan.PlanNode;

public interface Lookup
{
PlanNode resolve(PlanNode node);
}

0 comments on commit 13e6b6a

Please sign in to comment.