Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

Make PlanOptimizersStatsCollector thread-safe #16961

Merged
merged 1 commit into from
Apr 12, 2023
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -17,16 +17,19 @@
import io.trino.sql.planner.iterative.Rule;
import io.trino.sql.planner.optimizations.PlanOptimizer;

import javax.annotation.concurrent.ThreadSafe;

import java.util.Comparator;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
import java.util.concurrent.ConcurrentHashMap;

import static com.google.common.collect.ImmutableList.toImmutableList;

@ThreadSafe
public class PlanOptimizersStatsCollector
{
private final Map<Class<?>, QueryPlanOptimizerStats> stats = new HashMap<>();
private final Map<Class<?>, QueryPlanOptimizerStats> stats = new ConcurrentHashMap<>();
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

The class was extracted from IterativeOptimizer in #16365
we perhaps can still be OK with additional overhead this brings

if we want to make this class thread-safe, then

  • mark it as @Threadsafe
  • QueryPlanOptimizerStats should become thread-safe too
  • we no longer need io.trino.sql.planner.iterative.IterativeOptimizer.Context#iterativeOptimizerStatsCollector as a separate field, and we don't need io.trino.execution.querystats.PlanOptimizersStatsCollector#add method.

(I am not totally sure though whether this is the way to go.)

cc @martint

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

add is used when statistics from a single iterative optimizer are merged to collect statistics for a query.

Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

add is used when statistics from a single iterative optimizer are merged to collect statistics for a query.

this i know

however, now that PlanOptimizersStatsCollector is thread-safe, why create a new instance of that thing in IterativeOptimizer.Context?
... actually, it's not directly related to thread-safety -- it was redundant even before, so we can keep it as a follow-up

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

It was created to mirror the behavior that was implemented before. When the iterative optimizer exceeds the time limit, we log the top N rules that the current iterative optimizer has executed, as opposed to a top-level PlanOptimizersStatsCollector which collects statistics from rules across all optimizers.

Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

That's a good point.

OTOH, when iterative optimizer fails (or query is cancelled), the query completion event top N rules won't include rules from the last iterative optimizer run, which are usually the most important ones.

can you create an issue & follow-up on this?

private final int queryReportedRuleStatsLimit;

public PlanOptimizersStatsCollector(int queryReportedRuleStatsLimit)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,13 +15,18 @@

import io.trino.spi.eventlistener.QueryPlanOptimizerStatistics;

import javax.annotation.concurrent.ThreadSafe;

import java.util.concurrent.atomic.AtomicLong;

@ThreadSafe
public class QueryPlanOptimizerStats
{
private final String rule;
private long invocations;
private long applied;
private long totalTime;
private long failures;
private final AtomicLong invocations = new AtomicLong();
private final AtomicLong applied = new AtomicLong();
private final AtomicLong totalTime = new AtomicLong();
private final AtomicLong failures = new AtomicLong();

public QueryPlanOptimizerStats(String rule)
{
Expand All @@ -31,16 +36,16 @@ public QueryPlanOptimizerStats(String rule)
public void record(long nanos, boolean applied)
{
if (applied) {
this.applied += 1;
this.applied.incrementAndGet();
}

invocations += 1;
totalTime += nanos;
invocations.incrementAndGet();
totalTime.addAndGet(nanos);
}

public void recordFailure()
{
failures += 1;
failures.incrementAndGet();
}

public String getRule()
Expand All @@ -50,35 +55,35 @@ public String getRule()

public long getInvocations()
{
return invocations;
return invocations.get();
}

public long getApplied()
{
return applied;
return applied.get();
}

public long getFailures()
{
return failures;
return failures.get();
}

public long getTotalTime()
{
return totalTime;
return totalTime.get();
}

public QueryPlanOptimizerStatistics snapshot(String rule)
{
return new QueryPlanOptimizerStatistics(rule, invocations, applied, totalTime, failures);
return new QueryPlanOptimizerStatistics(rule, invocations.get(), applied.get(), totalTime.get(), failures.get());
}

public QueryPlanOptimizerStats merge(QueryPlanOptimizerStats other)
{
invocations += other.getInvocations();
applied += other.getApplied();
failures += other.getFailures();
totalTime += other.getTotalTime();
invocations.addAndGet(other.getInvocations());
applied.addAndGet(other.getApplied());
failures.addAndGet(other.getFailures());
totalTime.addAndGet(other.getTotalTime());

return this;
}
Expand Down