risingwave_frontend/optimizer/plan_node/
stream_simple_agg.rs

1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
// Copyright 2024 RisingWave Labs
//
// 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.

use fixedbitset::FixedBitSet;
use itertools::Itertools;
use pretty_xmlish::XmlNode;
use risingwave_pb::stream_plan::stream_node::PbNodeBody;

use super::generic::{self, PlanAggCall};
use super::stream::prelude::*;
use super::utils::{childless_record, plan_node_name, Distill};
use super::{ExprRewritable, PlanBase, PlanRef, PlanTreeNodeUnary, StreamNode};
use crate::expr::{ExprRewriter, ExprVisitor};
use crate::optimizer::plan_node::expr_visitable::ExprVisitable;
use crate::optimizer::property::{Distribution, MonotonicityMap};
use crate::stream_fragmenter::BuildFragmentGraphState;

#[derive(Debug, Clone, PartialEq, Eq, Hash)]
pub struct StreamSimpleAgg {
    pub base: PlanBase<Stream>,
    core: generic::Agg<PlanRef>,

    /// The index of `count(*)` in `agg_calls`.
    row_count_idx: usize,

    // Required by the downstream `RowMerge`,
    // currently only used by the `approx_percentile`'s two phase plan
    must_output_per_barrier: bool,
}

impl StreamSimpleAgg {
    pub fn new(
        core: generic::Agg<PlanRef>,
        row_count_idx: usize,
        must_output_per_barrier: bool,
    ) -> Self {
        assert_eq!(core.agg_calls[row_count_idx], PlanAggCall::count_star());

        let input = core.input.clone();
        let input_dist = input.distribution();
        let dist = match input_dist {
            Distribution::Single => Distribution::Single,
            _ => panic!(),
        };

        // Empty because watermark column(s) must be in group key and simple agg have no group key.
        let watermark_columns = FixedBitSet::with_capacity(core.output_len());

        // Simple agg executor might change the append-only behavior of the stream.
        let base = PlanBase::new_stream_with_core(
            &core,
            dist,
            false,
            false,
            watermark_columns,
            MonotonicityMap::new(),
        );
        StreamSimpleAgg {
            base,
            core,
            row_count_idx,
            must_output_per_barrier,
        }
    }

    pub fn agg_calls(&self) -> &[PlanAggCall] {
        &self.core.agg_calls
    }
}

impl Distill for StreamSimpleAgg {
    fn distill<'a>(&self) -> XmlNode<'a> {
        let name = plan_node_name!("StreamSimpleAgg",
            { "append_only", self.input().append_only() },
        );
        let mut vec = self.core.fields_pretty();
        if self.must_output_per_barrier {
            vec.push(("must_output_per_barrier", "true".into()));
        }
        childless_record(name, vec)
    }
}

impl PlanTreeNodeUnary for StreamSimpleAgg {
    fn input(&self) -> PlanRef {
        self.core.input.clone()
    }

    fn clone_with_input(&self, input: PlanRef) -> Self {
        let logical = generic::Agg {
            input,
            ..self.core.clone()
        };
        Self::new(logical, self.row_count_idx, self.must_output_per_barrier)
    }
}
impl_plan_tree_node_for_unary! { StreamSimpleAgg }

impl StreamNode for StreamSimpleAgg {
    fn to_stream_prost_body(&self, state: &mut BuildFragmentGraphState) -> PbNodeBody {
        use risingwave_pb::stream_plan::*;
        let (intermediate_state_table, agg_states, distinct_dedup_tables) =
            self.core.infer_tables(&self.base, None, None);

        PbNodeBody::SimpleAgg(SimpleAggNode {
            agg_calls: self
                .agg_calls()
                .iter()
                .map(PlanAggCall::to_protobuf)
                .collect(),
            distribution_key: self
                .base
                .distribution()
                .dist_column_indices()
                .iter()
                .map(|idx| *idx as u32)
                .collect(),
            is_append_only: self.input().append_only(),
            agg_call_states: agg_states
                .into_iter()
                .map(|s| s.into_prost(state))
                .collect(),
            intermediate_state_table: Some(
                intermediate_state_table
                    .with_id(state.gen_table_id_wrapped())
                    .to_internal_table_prost(),
            ),
            distinct_dedup_tables: distinct_dedup_tables
                .into_iter()
                .sorted_by_key(|(i, _)| *i)
                .map(|(key_idx, table)| {
                    (
                        key_idx as u32,
                        table
                            .with_id(state.gen_table_id_wrapped())
                            .to_internal_table_prost(),
                    )
                })
                .collect(),
            row_count_index: self.row_count_idx as u32,
            version: PbAggNodeVersion::Issue13465 as _,
            must_output_per_barrier: self.must_output_per_barrier,
        })
    }
}

impl ExprRewritable for StreamSimpleAgg {
    fn has_rewritable_expr(&self) -> bool {
        true
    }

    fn rewrite_exprs(&self, r: &mut dyn ExprRewriter) -> PlanRef {
        let mut core = self.core.clone();
        core.rewrite_exprs(r);
        Self::new(core, self.row_count_idx, self.must_output_per_barrier).into()
    }
}

impl ExprVisitable for StreamSimpleAgg {
    fn visit_exprs(&self, v: &mut dyn ExprVisitor) {
        self.core.visit_exprs(v);
    }
}