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

refactor(optimizer): split required and provided distribution property #2726

Merged
merged 20 commits into from
May 23, 2022

Conversation

st1page
Copy link
Contributor

@st1page st1page commented May 23, 2022

What's changed and what's your intention?

rethinking the distribution bugs(#2447 #1926 #1653) and strange code and think the current implementation of distribution property mix the required distribution and provided distribution up.
split required and provided distribution property.
see the comments in src/frontend/src/optimizer/property/distribution.rs for more information.

the planner test's diff is due to

  • after changing the RequiredDist::ShardByKey with the order-irrelevance bit-set, the exchange shuffle key's order changes
  • in this PR we prefer project before the exchange because it will save many IO in exchange in our current implementation

Checklist

  • I have written necessary docs and comments
  • I have added necessary unit tests and integration tests

Refer to a related PR or issue link (optional)

Copy link
Member

@fuyufjh fuyufjh left a comment

Choose a reason for hiding this comment

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

The idea LGTM

src/frontend/src/optimizer/property/distribution.rs Outdated Show resolved Hide resolved
@liurenjie1024
Copy link
Contributor

Do we also need to add RequiredOrder and ProvidedOrder? Another design is like following:

enum DistributionSpec {
Any,
...
}

enum OrderSpec {
Any,
...
}

struct RequiredPhysicalProperty {
   dist_spec: DistributionSpec,
   order_spec: OrderSpec
}

struct DerivedPhysicalProperty {
   dist_spec: DistributionSpec,
   order_spec: OrderSpec
}


@st1page
Copy link
Contributor Author

st1page commented May 23, 2022

Do we also need to add RequiredOrder and ProvidedOrder? Another design is like following:

  • I think we need to add RequiredOrder and ProvidedOrder when we actually use the order in optimizing.
  • I think the "Another design" is independent of this PR, we should split the DistributionSpec to DistributionSpec DistributionRequire to get the type safety and other benefits

@@ -1863,19 +1863,18 @@
BatchHashAgg { group_keys: [$0], aggs: [count, sum($1)] }
BatchExchange { order: [], dist: HashShard([0]) }
BatchProject { exprs: [Substr($0, 1:Int32, 2:Int32), $1] }
BatchExchange { order: [], dist: AnyShard }
Copy link
Contributor Author

@st1page st1page May 23, 2022

Choose a reason for hiding this comment

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

I think maybe there is a bug. if our batch execution can handle exchange with AnyShard? @liurenjie1024

Copy link
Contributor

Choose a reason for hiding this comment

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

Currently no.

@codecov
Copy link

codecov bot commented May 23, 2022

Codecov Report

Merging #2726 (a453f14) into main (c45c2f9) will increase coverage by 0.00%.
The diff coverage is 88.88%.

@@           Coverage Diff           @@
##             main    #2726   +/-   ##
=======================================
  Coverage   72.47%   72.48%           
=======================================
  Files         697      697           
  Lines       90329    90358   +29     
=======================================
+ Hits        65468    65498   +30     
+ Misses      24861    24860    -1     
Flag Coverage Δ
rust 72.48% <88.88%> (+<0.01%) ⬆️

Flags with carried forward coverage won't be shown. Click here to find out more.

Impacted Files Coverage Δ
...d/src/optimizer/plan_node/batch_generate_series.rs 0.00% <0.00%> (ø)
...ntend/src/optimizer/plan_node/stream_delta_join.rs 48.07% <ø> (-0.50%) ⬇️
...ontend/src/optimizer/plan_node/stream_hash_join.rs 97.05% <ø> (-0.12%) ⬇️
...ntend/src/optimizer/plan_node/stream_simple_agg.rs 95.74% <ø> (+1.99%) ⬆️
...rc/frontend/src/optimizer/plan_node/stream_topn.rs 53.84% <ø> (+1.01%) ⬆️
...frontend/src/optimizer/plan_node/batch_hash_agg.rs 73.52% <50.00%> (-0.39%) ⬇️
src/frontend/src/utils/column_index_mapping.rs 76.85% <66.66%> (-1.79%) ⬇️
...rontend/src/optimizer/plan_node/stream_hash_agg.rs 93.33% <75.00%> (+1.14%) ⬆️
.../frontend/src/optimizer/plan_node/batch_project.rs 81.81% <81.81%> (-0.44%) ⬇️
.../src/optimizer/plan_node/batch_nested_loop_join.rs 85.18% <83.33%> (-0.27%) ⬇️
... and 31 more

📣 Codecov can now indicate which changes are the most critical in Pull Requests. Learn more

Copy link
Contributor

@skyzh skyzh left a comment

Choose a reason for hiding this comment

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

Rest LGTM

@@ -40,8 +90,8 @@ impl Distribution {
Distribution::Single => DistributionMode::Single,
Distribution::Broadcast => DistributionMode::Broadcast,
Distribution::HashShard(_) => DistributionMode::Hash,
// TODO: Should panic if AnyShard or Any
_ => DistributionMode::Single,
// TODO: add round robin DistributionMode
Copy link
Contributor

Choose a reason for hiding this comment

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

When is it possible to have round-robin? 🤣

Copy link
Contributor

Choose a reason for hiding this comment

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

To increase parallelism of processing.

Copy link
Contributor

Choose a reason for hiding this comment

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

Round robin will cause a lot of problems. For example, in two-phase agg, it will cause row_count to be negative. I think long time ago we've decided to use hash distribution of the values for the local phase.

Copy link
Contributor

Choose a reason for hiding this comment

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

But it's okay to have it in batch, IMHO.

Copy link
Contributor

Choose a reason for hiding this comment

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

Yes, it should be used carefully.

src/frontend/src/optimizer/plan_node/batch_hash_join.rs Outdated Show resolved Hide resolved
@st1page st1page merged commit 3577409 into main May 23, 2022
@st1page st1page deleted the sts/optimizer_add_shard_distributed branch May 23, 2022 07:22
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment
Projects
None yet
Development

Successfully merging this pull request may close these issues.

4 participants