risingwave_meta/stream/stream_graph/
fragment.rs

1// Copyright 2025 RisingWave Labs
2//
3// Licensed under the Apache License, Version 2.0 (the "License");
4// you may not use this file except in compliance with the License.
5// You may obtain a copy of the License at
6//
7//     http://www.apache.org/licenses/LICENSE-2.0
8//
9// Unless required by applicable law or agreed to in writing, software
10// distributed under the License is distributed on an "AS IS" BASIS,
11// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
12// See the License for the specific language governing permissions and
13// limitations under the License.
14
15use std::collections::{BTreeMap, HashMap, HashSet};
16use std::num::NonZeroUsize;
17use std::ops::{Deref, DerefMut};
18use std::sync::LazyLock;
19use std::sync::atomic::AtomicU32;
20
21use anyhow::{Context, anyhow};
22use enum_as_inner::EnumAsInner;
23use itertools::Itertools;
24use risingwave_common::bail;
25use risingwave_common::catalog::{
26    CDC_SOURCE_COLUMN_NUM, ColumnCatalog, Field, FragmentTypeFlag, FragmentTypeMask, TableId,
27    generate_internal_table_name_with_type,
28};
29use risingwave_common::hash::VnodeCount;
30use risingwave_common::id::JobId;
31use risingwave_common::util::iter_util::ZipEqFast;
32use risingwave_common::util::stream_graph_visitor::{
33    self, visit_stream_node_cont, visit_stream_node_cont_mut,
34};
35use risingwave_connector::sink::catalog::SinkType;
36use risingwave_meta_model::WorkerId;
37use risingwave_pb::catalog::{PbSink, PbTable, Table};
38use risingwave_pb::ddl_service::TableJobType;
39use risingwave_pb::id::StreamNodeLocalOperatorId;
40use risingwave_pb::plan_common::{PbColumnCatalog, PbColumnDesc};
41use risingwave_pb::stream_plan::dispatch_output_mapping::TypePair;
42use risingwave_pb::stream_plan::stream_fragment_graph::{
43    Parallelism, StreamFragment, StreamFragmentEdge as StreamFragmentEdgeProto,
44};
45use risingwave_pb::stream_plan::stream_node::{NodeBody, PbNodeBody};
46use risingwave_pb::stream_plan::{
47    BackfillOrder, DispatchOutputMapping, DispatchStrategy, DispatcherType, PbStreamNode,
48    PbStreamScanType, StreamFragmentGraph as StreamFragmentGraphProto, StreamNode, StreamScanNode,
49    StreamScanType,
50};
51
52use crate::barrier::{SharedFragmentInfo, SnapshotBackfillInfo};
53use crate::controller::id::IdGeneratorManager;
54use crate::manager::{MetaSrvEnv, StreamingJob, StreamingJobType};
55use crate::model::{ActorId, Fragment, FragmentId, StreamActor};
56use crate::stream::stream_graph::id::{
57    GlobalActorIdGen, GlobalFragmentId, GlobalFragmentIdGen, GlobalTableIdGen,
58};
59use crate::stream::stream_graph::schedule::Distribution;
60use crate::{MetaError, MetaResult};
61
62/// The fragment in the building phase, including the [`StreamFragment`] from the frontend and
63/// several additional helper fields.
64#[derive(Debug, Clone)]
65pub(super) struct BuildingFragment {
66    /// The fragment structure from the frontend, with the global fragment ID.
67    inner: StreamFragment,
68
69    /// The ID of the job if it contains the streaming job node.
70    job_id: Option<JobId>,
71
72    /// The required column IDs of each upstream table.
73    /// Will be converted to indices when building the edge connected to the upstream.
74    ///
75    /// For shared CDC table on source, its `vec![]`, since the upstream source's output schema is fixed.
76    upstream_job_columns: HashMap<JobId, Vec<PbColumnDesc>>,
77}
78
79impl BuildingFragment {
80    /// Create a new [`BuildingFragment`] from a [`StreamFragment`]. The global fragment ID and
81    /// global table IDs will be correctly filled with the given `id` and `table_id_gen`.
82    fn new(
83        id: GlobalFragmentId,
84        fragment: StreamFragment,
85        job: &StreamingJob,
86        table_id_gen: GlobalTableIdGen,
87    ) -> Self {
88        let mut fragment = StreamFragment {
89            fragment_id: id.as_global_id(),
90            ..fragment
91        };
92
93        // Fill the information of the internal tables in the fragment.
94        Self::fill_internal_tables(&mut fragment, job, table_id_gen);
95
96        let job_id = Self::fill_job(&mut fragment, job).then(|| job.id());
97        let upstream_job_columns =
98            Self::extract_upstream_columns_except_cross_db_backfill(&fragment);
99
100        Self {
101            inner: fragment,
102            job_id,
103            upstream_job_columns,
104        }
105    }
106
107    /// Extract the internal tables from the fragment.
108    fn extract_internal_tables(&self) -> Vec<Table> {
109        let mut fragment = self.inner.clone();
110        let mut tables = Vec::new();
111        stream_graph_visitor::visit_internal_tables(&mut fragment, |table, _| {
112            tables.push(table.clone());
113        });
114        tables
115    }
116
117    /// Fill the information with the internal tables in the fragment.
118    fn fill_internal_tables(
119        fragment: &mut StreamFragment,
120        job: &StreamingJob,
121        table_id_gen: GlobalTableIdGen,
122    ) {
123        let fragment_id = fragment.fragment_id;
124        stream_graph_visitor::visit_internal_tables(fragment, |table, table_type_name| {
125            table.id = table_id_gen
126                .to_global_id(table.id.as_raw_id())
127                .as_global_id();
128            table.schema_id = job.schema_id();
129            table.database_id = job.database_id();
130            table.name = generate_internal_table_name_with_type(
131                &job.name(),
132                fragment_id,
133                table.id,
134                table_type_name,
135            );
136            table.fragment_id = fragment_id;
137            table.owner = job.owner();
138            table.job_id = Some(job.id());
139        });
140    }
141
142    /// Fill the information with the job in the fragment.
143    fn fill_job(fragment: &mut StreamFragment, job: &StreamingJob) -> bool {
144        let job_id = job.id();
145        let fragment_id = fragment.fragment_id;
146        let mut has_job = false;
147
148        stream_graph_visitor::visit_fragment_mut(fragment, |node_body| match node_body {
149            NodeBody::Materialize(materialize_node) => {
150                materialize_node.table_id = job_id.as_mv_table_id();
151
152                // Fill the table field of `MaterializeNode` from the job.
153                let table = materialize_node.table.insert(job.table().unwrap().clone());
154                table.fragment_id = fragment_id; // this will later be synced back to `job.table` with `set_info_from_graph`
155                // In production, do not include full definition in the table in plan node.
156                if cfg!(not(debug_assertions)) {
157                    table.definition = job.name();
158                }
159
160                has_job = true;
161            }
162            NodeBody::Sink(sink_node) => {
163                sink_node.sink_desc.as_mut().unwrap().id = job_id.as_sink_id();
164
165                has_job = true;
166            }
167            NodeBody::Dml(dml_node) => {
168                dml_node.table_id = job_id.as_mv_table_id();
169                dml_node.table_version_id = job.table_version_id().unwrap();
170            }
171            NodeBody::StreamFsFetch(fs_fetch_node) => {
172                if let StreamingJob::Table(table_source, _, _) = job
173                    && let Some(node_inner) = fs_fetch_node.node_inner.as_mut()
174                    && let Some(source) = table_source
175                {
176                    node_inner.source_id = source.id;
177                    if let Some(id) = source.optional_associated_table_id {
178                        node_inner.associated_table_id = Some(id.into());
179                    }
180                }
181            }
182            NodeBody::Source(source_node) => {
183                match job {
184                    // Note: For table without connector, it has a dummy Source node.
185                    // Note: For table with connector, it's source node has a source id different with the table id (job id), assigned in create_job_catalog.
186                    StreamingJob::Table(source, _table, _table_job_type) => {
187                        if let Some(source_inner) = source_node.source_inner.as_mut()
188                            && let Some(source) = source
189                        {
190                            debug_assert_ne!(source.id, job_id.as_raw_id());
191                            source_inner.source_id = source.id;
192                            if let Some(id) = source.optional_associated_table_id {
193                                source_inner.associated_table_id = Some(id.into());
194                            }
195                        }
196                    }
197                    StreamingJob::Source(source) => {
198                        has_job = true;
199                        if let Some(source_inner) = source_node.source_inner.as_mut() {
200                            debug_assert_eq!(source.id, job_id.as_raw_id());
201                            source_inner.source_id = source.id;
202                            if let Some(id) = source.optional_associated_table_id {
203                                source_inner.associated_table_id = Some(id.into());
204                            }
205                        }
206                    }
207                    // For other job types, no need to fill the source id, since it refers to an existing source.
208                    _ => {}
209                }
210            }
211            NodeBody::StreamCdcScan(node) => {
212                if let Some(table_desc) = node.cdc_table_desc.as_mut() {
213                    table_desc.table_id = job_id.as_mv_table_id();
214                }
215            }
216            NodeBody::VectorIndexWrite(node) => {
217                let table = node.table.as_mut().unwrap();
218                table.id = job_id.as_mv_table_id();
219                table.database_id = job.database_id();
220                table.schema_id = job.schema_id();
221                table.fragment_id = fragment_id;
222                #[cfg(not(debug_assertions))]
223                {
224                    table.definition = job.name();
225                }
226
227                has_job = true;
228            }
229            _ => {}
230        });
231
232        has_job
233    }
234
235    /// Extract the required columns of each upstream table except for cross-db backfill.
236    fn extract_upstream_columns_except_cross_db_backfill(
237        fragment: &StreamFragment,
238    ) -> HashMap<JobId, Vec<PbColumnDesc>> {
239        let mut table_columns = HashMap::new();
240
241        stream_graph_visitor::visit_fragment(fragment, |node_body| {
242            let (table_id, column_ids) = match node_body {
243                NodeBody::StreamScan(stream_scan) => {
244                    if stream_scan.get_stream_scan_type().unwrap()
245                        == StreamScanType::CrossDbSnapshotBackfill
246                    {
247                        return;
248                    }
249                    (
250                        stream_scan.table_id.as_job_id(),
251                        stream_scan.upstream_columns(),
252                    )
253                }
254                NodeBody::CdcFilter(cdc_filter) => (
255                    cdc_filter.upstream_source_id.as_share_source_job_id(),
256                    vec![],
257                ),
258                NodeBody::SourceBackfill(backfill) => (
259                    backfill.upstream_source_id.as_share_source_job_id(),
260                    // FIXME: only pass required columns instead of all columns here
261                    backfill.column_descs(),
262                ),
263                _ => return,
264            };
265            table_columns
266                .try_insert(table_id, column_ids)
267                .expect("currently there should be no two same upstream tables in a fragment");
268        });
269
270        table_columns
271    }
272
273    pub fn has_shuffled_backfill(&self) -> bool {
274        let stream_node = match self.inner.node.as_ref() {
275            Some(node) => node,
276            _ => return false,
277        };
278        let mut has_shuffled_backfill = false;
279        let has_shuffled_backfill_mut_ref = &mut has_shuffled_backfill;
280        visit_stream_node_cont(stream_node, |node| {
281            let is_shuffled_backfill = if let Some(node) = &node.node_body
282                && let Some(node) = node.as_stream_scan()
283            {
284                node.stream_scan_type == StreamScanType::ArrangementBackfill as i32
285                    || node.stream_scan_type == StreamScanType::SnapshotBackfill as i32
286            } else {
287                false
288            };
289            if is_shuffled_backfill {
290                *has_shuffled_backfill_mut_ref = true;
291                false
292            } else {
293                true
294            }
295        });
296        has_shuffled_backfill
297    }
298}
299
300impl Deref for BuildingFragment {
301    type Target = StreamFragment;
302
303    fn deref(&self) -> &Self::Target {
304        &self.inner
305    }
306}
307
308impl DerefMut for BuildingFragment {
309    fn deref_mut(&mut self) -> &mut Self::Target {
310        &mut self.inner
311    }
312}
313
314/// The ID of an edge in the fragment graph. For different types of edges, the ID will be in
315/// different variants.
316#[derive(Debug, Clone, Copy, PartialEq, Eq, Hash, EnumAsInner)]
317pub(super) enum EdgeId {
318    /// The edge between two building (internal) fragments.
319    Internal {
320        /// The ID generated by the frontend, generally the operator ID of `Exchange`.
321        /// See [`StreamFragmentEdgeProto`].
322        link_id: u64,
323    },
324
325    /// The edge between an upstream external fragment and downstream building fragment. Used for
326    /// MV on MV.
327    UpstreamExternal {
328        /// The ID of the upstream table or materialized view.
329        upstream_job_id: JobId,
330        /// The ID of the downstream fragment.
331        downstream_fragment_id: GlobalFragmentId,
332    },
333
334    /// The edge between an upstream building fragment and downstream external fragment. Used for
335    /// schema change (replace table plan).
336    DownstreamExternal(DownstreamExternalEdgeId),
337}
338
339#[derive(Debug, Clone, Copy, PartialEq, Eq, Hash)]
340pub(super) struct DownstreamExternalEdgeId {
341    /// The ID of the original upstream fragment (`Materialize`).
342    pub(super) original_upstream_fragment_id: GlobalFragmentId,
343    /// The ID of the downstream fragment.
344    pub(super) downstream_fragment_id: GlobalFragmentId,
345}
346
347/// The edge in the fragment graph.
348///
349/// The edge can be either internal or external. This is distinguished by the [`EdgeId`].
350#[derive(Debug, Clone)]
351pub(super) struct StreamFragmentEdge {
352    /// The ID of the edge.
353    pub id: EdgeId,
354
355    /// The strategy used for dispatching the data.
356    pub dispatch_strategy: DispatchStrategy,
357}
358
359impl StreamFragmentEdge {
360    fn from_protobuf(edge: &StreamFragmentEdgeProto) -> Self {
361        Self {
362            // By creating an edge from the protobuf, we know that the edge is from the frontend and
363            // is internal.
364            id: EdgeId::Internal {
365                link_id: edge.link_id,
366            },
367            dispatch_strategy: edge.get_dispatch_strategy().unwrap().clone(),
368        }
369    }
370}
371
372fn clone_fragment(
373    fragment: &Fragment,
374    id_generator_manager: &IdGeneratorManager,
375    actor_id_counter: &AtomicU32,
376) -> Fragment {
377    let fragment_id = GlobalFragmentIdGen::new(id_generator_manager, 1)
378        .to_global_id(0)
379        .as_global_id();
380    let actor_id_gen = GlobalActorIdGen::new(actor_id_counter, fragment.actors.len() as _);
381    Fragment {
382        fragment_id,
383        fragment_type_mask: fragment.fragment_type_mask,
384        distribution_type: fragment.distribution_type,
385        actors: fragment
386            .actors
387            .iter()
388            .enumerate()
389            .map(|(i, actor)| StreamActor {
390                actor_id: actor_id_gen.to_global_id(i as _).as_global_id(),
391                fragment_id,
392                vnode_bitmap: actor.vnode_bitmap.clone(),
393                mview_definition: actor.mview_definition.clone(),
394                expr_context: actor.expr_context.clone(),
395                config_override: actor.config_override.clone(),
396            })
397            .collect(),
398        state_table_ids: fragment.state_table_ids.clone(),
399        maybe_vnode_count: fragment.maybe_vnode_count,
400        nodes: fragment.nodes.clone(),
401    }
402}
403
404pub fn check_sink_fragments_support_refresh_schema(
405    fragments: &BTreeMap<FragmentId, Fragment>,
406) -> MetaResult<()> {
407    if fragments.len() != 1 {
408        return Err(anyhow!(
409            "sink with auto schema change should have only 1 fragment, but got {:?}",
410            fragments.len()
411        )
412        .into());
413    }
414    let (_, fragment) = fragments.first_key_value().expect("non-empty");
415    let sink_node = &fragment.nodes;
416    let PbNodeBody::Sink(_) = sink_node.node_body.as_ref().unwrap() else {
417        return Err(anyhow!("expect PbNodeBody::Sink but got: {:?}", sink_node.node_body).into());
418    };
419    let [stream_scan_node] = sink_node.input.as_slice() else {
420        panic!("Sink has more than 1 input: {:?}", sink_node.input);
421    };
422    let PbNodeBody::StreamScan(scan) = stream_scan_node.node_body.as_ref().unwrap() else {
423        return Err(anyhow!(
424            "expect PbNodeBody::StreamScan but got: {:?}",
425            stream_scan_node.node_body
426        )
427        .into());
428    };
429    let stream_scan_type = PbStreamScanType::try_from(scan.stream_scan_type).unwrap();
430    if stream_scan_type != PbStreamScanType::ArrangementBackfill {
431        return Err(anyhow!(
432            "unsupported stream_scan_type for auto refresh schema: {:?}",
433            stream_scan_type
434        )
435        .into());
436    }
437    let [merge_node, _batch_plan_node] = stream_scan_node.input.as_slice() else {
438        panic!(
439            "the number of StreamScan inputs is not 2: {:?}",
440            stream_scan_node.input
441        );
442    };
443    let NodeBody::Merge(_) = merge_node.node_body.as_ref().unwrap() else {
444        return Err(anyhow!(
445            "expect PbNodeBody::Merge but got: {:?}",
446            merge_node.node_body
447        )
448        .into());
449    };
450    Ok(())
451}
452
453pub fn rewrite_refresh_schema_sink_fragment(
454    original_sink_fragment: &Fragment,
455    sink: &PbSink,
456    newly_added_columns: &[ColumnCatalog],
457    upstream_table: &PbTable,
458    upstream_table_fragment_id: FragmentId,
459    id_generator_manager: &IdGeneratorManager,
460    actor_id_counter: &AtomicU32,
461) -> MetaResult<(Fragment, Vec<PbColumnCatalog>, Option<PbTable>)> {
462    let mut new_sink_columns = sink.columns.clone();
463    fn extend_sink_columns(
464        sink_columns: &mut Vec<PbColumnCatalog>,
465        new_columns: &[ColumnCatalog],
466        get_column_name: impl Fn(&String) -> String,
467    ) {
468        let next_column_id = sink_columns
469            .iter()
470            .map(|col| col.column_desc.as_ref().unwrap().column_id + 1)
471            .max()
472            .unwrap_or(1);
473        sink_columns.extend(new_columns.iter().enumerate().map(|(i, col)| {
474            let mut col = col.to_protobuf();
475            let column_desc = col.column_desc.as_mut().unwrap();
476            column_desc.column_id = next_column_id + (i as i32);
477            column_desc.name = get_column_name(&column_desc.name);
478            col
479        }));
480    }
481    extend_sink_columns(&mut new_sink_columns, newly_added_columns, |name| {
482        name.clone()
483    });
484
485    let mut new_sink_fragment = clone_fragment(
486        original_sink_fragment,
487        id_generator_manager,
488        actor_id_counter,
489    );
490    let sink_node = &mut new_sink_fragment.nodes;
491    let PbNodeBody::Sink(sink_node_body) = sink_node.node_body.as_mut().unwrap() else {
492        return Err(anyhow!("expect PbNodeBody::Sink but got: {:?}", sink_node.node_body).into());
493    };
494    let [stream_scan_node] = sink_node.input.as_mut_slice() else {
495        panic!("Sink has more than 1 input: {:?}", sink_node.input);
496    };
497    let PbNodeBody::StreamScan(scan) = stream_scan_node.node_body.as_mut().unwrap() else {
498        return Err(anyhow!(
499            "expect PbNodeBody::StreamScan but got: {:?}",
500            stream_scan_node.node_body
501        )
502        .into());
503    };
504    let [merge_node, _batch_plan_node] = stream_scan_node.input.as_mut_slice() else {
505        panic!(
506            "the number of StreamScan inputs is not 2: {:?}",
507            stream_scan_node.input
508        );
509    };
510    let NodeBody::Merge(merge) = merge_node.node_body.as_mut().unwrap() else {
511        return Err(anyhow!(
512            "expect PbNodeBody::Merge but got: {:?}",
513            merge_node.node_body
514        )
515        .into());
516    };
517    // update sink_node
518    // following logic in <StreamSink as Explain>::distill
519    sink_node.identity = {
520        let sink_type = SinkType::from_proto(sink.sink_type());
521        let sink_type_str = sink_type.type_str();
522        let column_names = new_sink_columns
523            .iter()
524            .map(|col| {
525                ColumnCatalog::from(col.clone())
526                    .name_with_hidden()
527                    .to_string()
528            })
529            .join(", ");
530        let downstream_pk = if !sink_type.is_append_only() {
531            let downstream_pk = sink
532                .downstream_pk
533                .iter()
534                .map(|i| &sink.columns[*i as usize].column_desc.as_ref().unwrap().name)
535                .collect_vec();
536            format!(", downstream_pk: {downstream_pk:?}")
537        } else {
538            "".to_owned()
539        };
540        format!("StreamSink {{ type: {sink_type_str}, columns: [{column_names}]{downstream_pk} }}")
541    };
542    sink_node
543        .fields
544        .extend(newly_added_columns.iter().map(|col| {
545            Field::new(
546                format!("{}.{}", upstream_table.name, col.column_desc.name),
547                col.data_type().clone(),
548            )
549            .to_prost()
550        }));
551
552    let new_log_store_table = if let Some(log_store_table) = &mut sink_node_body.table {
553        extend_sink_columns(&mut log_store_table.columns, newly_added_columns, |name| {
554            format!("{}_{}", upstream_table.name, name)
555        });
556        Some(log_store_table.clone())
557    } else {
558        None
559    };
560    sink_node_body.sink_desc.as_mut().unwrap().column_catalogs = new_sink_columns.clone();
561
562    // update stream scan node
563    stream_scan_node
564        .fields
565        .extend(newly_added_columns.iter().map(|col| {
566            Field::new(
567                format!("{}.{}", upstream_table.name, col.column_desc.name),
568                col.data_type().clone(),
569            )
570            .to_prost()
571        }));
572    // following logic in <StreamTableScan as Explain>::distill
573    stream_scan_node.identity = {
574        let columns = stream_scan_node
575            .fields
576            .iter()
577            .map(|col| &col.name)
578            .join(", ");
579        format!("StreamTableScan {{ table: t, columns: [{columns}] }}")
580    };
581
582    let stream_scan_type = PbStreamScanType::try_from(scan.stream_scan_type).unwrap();
583    if stream_scan_type != PbStreamScanType::ArrangementBackfill {
584        return Err(anyhow!(
585            "unsupported stream_scan_type for auto refresh schema: {:?}",
586            stream_scan_type
587        )
588        .into());
589    }
590    scan.arrangement_table = Some(upstream_table.clone());
591    scan.output_indices.extend(
592        (0..newly_added_columns.len()).map(|i| (i + scan.upstream_column_ids.len()) as u32),
593    );
594    scan.upstream_column_ids.extend(
595        newly_added_columns
596            .iter()
597            .map(|col| col.column_id().get_id()),
598    );
599    let table_desc = scan.table_desc.as_mut().unwrap();
600    table_desc
601        .value_indices
602        .extend((0..newly_added_columns.len()).map(|i| (i + table_desc.columns.len()) as u32));
603    table_desc.columns.extend(
604        newly_added_columns
605            .iter()
606            .map(|col| col.column_desc.to_protobuf()),
607    );
608
609    // update merge node
610    merge_node.fields = scan
611        .upstream_column_ids
612        .iter()
613        .map(|&column_id| {
614            let col = upstream_table
615                .columns
616                .iter()
617                .find(|c| c.column_desc.as_ref().unwrap().column_id == column_id)
618                .unwrap();
619            let col_desc = col.column_desc.as_ref().unwrap();
620            Field::new(
621                col_desc.name.clone(),
622                col_desc.column_type.as_ref().unwrap().into(),
623            )
624            .to_prost()
625        })
626        .collect();
627    merge.upstream_fragment_id = upstream_table_fragment_id;
628    Ok((new_sink_fragment, new_sink_columns, new_log_store_table))
629}
630
631/// Adjacency list (G) of backfill orders.
632/// `G[10] -> [1, 2, 11]`
633/// means for the backfill node in `fragment 10`
634/// should be backfilled before the backfill nodes in `fragment 1, 2 and 11`.
635pub type FragmentBackfillOrder = HashMap<FragmentId, Vec<FragmentId>>;
636
637/// In-memory representation of a **Fragment** Graph, built from the [`StreamFragmentGraphProto`]
638/// from the frontend.
639///
640/// This only includes nodes and edges of the current job itself. It will be converted to [`CompleteStreamFragmentGraph`] later,
641/// that contains the additional information of pre-existing
642/// fragments, which are connected to the graph's top-most or bottom-most fragments.
643#[derive(Default, Debug)]
644pub struct StreamFragmentGraph {
645    /// stores all the fragments in the graph.
646    pub(super) fragments: HashMap<GlobalFragmentId, BuildingFragment>,
647
648    /// stores edges between fragments: upstream => downstream.
649    pub(super) downstreams:
650        HashMap<GlobalFragmentId, HashMap<GlobalFragmentId, StreamFragmentEdge>>,
651
652    /// stores edges between fragments: downstream -> upstream.
653    pub(super) upstreams: HashMap<GlobalFragmentId, HashMap<GlobalFragmentId, StreamFragmentEdge>>,
654
655    /// Dependent relations of this job.
656    dependent_table_ids: HashSet<TableId>,
657
658    /// The default parallelism of the job, specified by the `STREAMING_PARALLELISM` session
659    /// variable. If not specified, all active worker slots will be used.
660    specified_parallelism: Option<NonZeroUsize>,
661
662    /// Specified max parallelism, i.e., expected vnode count for the graph.
663    ///
664    /// The scheduler on the meta service will use this as a hint to decide the vnode count
665    /// for each fragment.
666    ///
667    /// Note that the actual vnode count may be different from this value.
668    /// For example, a no-shuffle exchange between current fragment graph and an existing
669    /// upstream fragment graph requires two fragments to be in the same distribution,
670    /// thus the same vnode count.
671    max_parallelism: usize,
672
673    /// The backfill ordering strategy of the graph.
674    backfill_order: BackfillOrder,
675}
676
677impl StreamFragmentGraph {
678    /// Create a new [`StreamFragmentGraph`] from the given [`StreamFragmentGraphProto`], with all
679    /// global IDs correctly filled.
680    pub fn new(
681        env: &MetaSrvEnv,
682        proto: StreamFragmentGraphProto,
683        job: &StreamingJob,
684    ) -> MetaResult<Self> {
685        let fragment_id_gen =
686            GlobalFragmentIdGen::new(env.id_gen_manager(), proto.fragments.len() as u64);
687        // Note: in SQL backend, the ids generated here are fake and will be overwritten again
688        // with `refill_internal_table_ids` later.
689        // TODO: refactor the code to remove this step.
690        let table_id_gen = GlobalTableIdGen::new(env.id_gen_manager(), proto.table_ids_cnt as u64);
691
692        // Create nodes.
693        let fragments: HashMap<_, _> = proto
694            .fragments
695            .into_iter()
696            .map(|(id, fragment)| {
697                let id = fragment_id_gen.to_global_id(id.as_raw_id());
698                let fragment = BuildingFragment::new(id, fragment, job, table_id_gen);
699                (id, fragment)
700            })
701            .collect();
702
703        assert_eq!(
704            fragments
705                .values()
706                .map(|f| f.extract_internal_tables().len() as u32)
707                .sum::<u32>(),
708            proto.table_ids_cnt
709        );
710
711        // Create edges.
712        let mut downstreams = HashMap::new();
713        let mut upstreams = HashMap::new();
714
715        for edge in proto.edges {
716            let upstream_id = fragment_id_gen.to_global_id(edge.upstream_id.as_raw_id());
717            let downstream_id = fragment_id_gen.to_global_id(edge.downstream_id.as_raw_id());
718            let edge = StreamFragmentEdge::from_protobuf(&edge);
719
720            upstreams
721                .entry(downstream_id)
722                .or_insert_with(HashMap::new)
723                .try_insert(upstream_id, edge.clone())
724                .unwrap();
725            downstreams
726                .entry(upstream_id)
727                .or_insert_with(HashMap::new)
728                .try_insert(downstream_id, edge)
729                .unwrap();
730        }
731
732        // Note: Here we directly use the field `dependent_table_ids` in the proto (resolved in
733        // frontend), instead of visiting the graph ourselves.
734        let dependent_table_ids = proto.dependent_table_ids.iter().copied().collect();
735
736        let specified_parallelism = if let Some(Parallelism { parallelism }) = proto.parallelism {
737            Some(NonZeroUsize::new(parallelism as usize).context("parallelism should not be 0")?)
738        } else {
739            None
740        };
741
742        let max_parallelism = proto.max_parallelism as usize;
743        let backfill_order = proto.backfill_order.unwrap_or(BackfillOrder {
744            order: Default::default(),
745        });
746
747        Ok(Self {
748            fragments,
749            downstreams,
750            upstreams,
751            dependent_table_ids,
752            specified_parallelism,
753            max_parallelism,
754            backfill_order,
755        })
756    }
757
758    /// Retrieve the **incomplete** internal tables map of the whole graph.
759    ///
760    /// Note that some fields in the table catalogs are not filled during the current phase, e.g.,
761    /// `fragment_id`, `vnode_count`. They will be all filled after a `TableFragments` is built.
762    /// Be careful when using the returned values.
763    pub fn incomplete_internal_tables(&self) -> BTreeMap<TableId, Table> {
764        let mut tables = BTreeMap::new();
765        for fragment in self.fragments.values() {
766            for table in fragment.extract_internal_tables() {
767                let table_id = table.id;
768                tables
769                    .try_insert(table_id, table)
770                    .unwrap_or_else(|_| panic!("duplicated table id `{}`", table_id));
771            }
772        }
773        tables
774    }
775
776    /// Refill the internal tables' `table_id`s according to the given map, typically obtained from
777    /// `create_internal_table_catalog`.
778    pub fn refill_internal_table_ids(&mut self, table_id_map: HashMap<TableId, TableId>) {
779        for fragment in self.fragments.values_mut() {
780            stream_graph_visitor::visit_internal_tables(
781                &mut fragment.inner,
782                |table, _table_type_name| {
783                    let target = table_id_map.get(&table.id).cloned().unwrap();
784                    table.id = target;
785                },
786            );
787        }
788    }
789
790    /// Use a trivial algorithm to match the internal tables of the new graph for
791    /// `ALTER TABLE` or `ALTER SOURCE`.
792    pub fn fit_internal_tables_trivial(
793        &mut self,
794        mut old_internal_tables: Vec<Table>,
795    ) -> MetaResult<()> {
796        let mut new_internal_table_ids = Vec::new();
797        for fragment in self.fragments.values() {
798            for table in &fragment.extract_internal_tables() {
799                new_internal_table_ids.push(table.id);
800            }
801        }
802
803        if new_internal_table_ids.len() != old_internal_tables.len() {
804            bail!(
805                "Different number of internal tables. New: {}, Old: {}",
806                new_internal_table_ids.len(),
807                old_internal_tables.len()
808            );
809        }
810        old_internal_tables.sort_by(|a, b| a.id.cmp(&b.id));
811        new_internal_table_ids.sort();
812
813        let internal_table_id_map = new_internal_table_ids
814            .into_iter()
815            .zip_eq_fast(old_internal_tables.into_iter())
816            .collect::<HashMap<_, _>>();
817
818        // TODO(alter-mv): unify this with `fit_internal_table_ids_with_mapping` after we
819        // confirm the behavior is the same.
820        for fragment in self.fragments.values_mut() {
821            stream_graph_visitor::visit_internal_tables(
822                &mut fragment.inner,
823                |table, _table_type_name| {
824                    // XXX: this replaces the entire table, instead of just the id!
825                    let target = internal_table_id_map.get(&table.id).cloned().unwrap();
826                    *table = target;
827                },
828            );
829        }
830
831        Ok(())
832    }
833
834    /// Fit the internal tables' `table_id`s according to the given mapping.
835    pub fn fit_internal_table_ids_with_mapping(&mut self, mut matches: HashMap<TableId, Table>) {
836        for fragment in self.fragments.values_mut() {
837            stream_graph_visitor::visit_internal_tables(
838                &mut fragment.inner,
839                |table, _table_type_name| {
840                    let target = matches.remove(&table.id).unwrap_or_else(|| {
841                        panic!("no matching table for table {}({})", table.id, table.name)
842                    });
843                    table.id = target.id;
844                    table.maybe_vnode_count = target.maybe_vnode_count;
845                },
846            );
847        }
848    }
849
850    pub fn fit_snapshot_backfill_epochs(
851        &mut self,
852        mut snapshot_backfill_epochs: HashMap<StreamNodeLocalOperatorId, u64>,
853    ) {
854        for fragment in self.fragments.values_mut() {
855            visit_stream_node_cont_mut(fragment.node.as_mut().unwrap(), |node| {
856                if let PbNodeBody::StreamScan(scan) = node.node_body.as_mut().unwrap()
857                    && let StreamScanType::SnapshotBackfill
858                    | StreamScanType::CrossDbSnapshotBackfill = scan.stream_scan_type()
859                {
860                    let Some(epoch) = snapshot_backfill_epochs.remove(&node.operator_id) else {
861                        panic!("no snapshot epoch found for node {:?}", node)
862                    };
863                    scan.snapshot_backfill_epoch = Some(epoch);
864                }
865                true
866            })
867        }
868    }
869
870    /// Returns the fragment id where the streaming job node located.
871    pub fn table_fragment_id(&self) -> FragmentId {
872        self.fragments
873            .values()
874            .filter(|b| b.job_id.is_some())
875            .map(|b| b.fragment_id)
876            .exactly_one()
877            .expect("require exactly 1 materialize/sink/cdc source node when creating the streaming job")
878    }
879
880    /// Returns the fragment id where the table dml is received.
881    pub fn dml_fragment_id(&self) -> Option<FragmentId> {
882        self.fragments
883            .values()
884            .filter(|b| {
885                FragmentTypeMask::from(b.fragment_type_mask).contains(FragmentTypeFlag::Dml)
886            })
887            .map(|b| b.fragment_id)
888            .at_most_one()
889            .expect("require at most 1 dml node when creating the streaming job")
890    }
891
892    /// Get the dependent streaming job ids of this job.
893    pub fn dependent_table_ids(&self) -> &HashSet<TableId> {
894        &self.dependent_table_ids
895    }
896
897    /// Get the parallelism of the job, if specified by the user.
898    pub fn specified_parallelism(&self) -> Option<NonZeroUsize> {
899        self.specified_parallelism
900    }
901
902    /// Get the expected vnode count of the graph. See documentation of the field for more details.
903    pub fn max_parallelism(&self) -> usize {
904        self.max_parallelism
905    }
906
907    /// Get downstreams of a fragment.
908    fn get_downstreams(
909        &self,
910        fragment_id: GlobalFragmentId,
911    ) -> &HashMap<GlobalFragmentId, StreamFragmentEdge> {
912        self.downstreams.get(&fragment_id).unwrap_or(&EMPTY_HASHMAP)
913    }
914
915    /// Get upstreams of a fragment.
916    fn get_upstreams(
917        &self,
918        fragment_id: GlobalFragmentId,
919    ) -> &HashMap<GlobalFragmentId, StreamFragmentEdge> {
920        self.upstreams.get(&fragment_id).unwrap_or(&EMPTY_HASHMAP)
921    }
922
923    pub fn collect_snapshot_backfill_info(
924        &self,
925    ) -> MetaResult<(Option<SnapshotBackfillInfo>, SnapshotBackfillInfo)> {
926        Self::collect_snapshot_backfill_info_impl(self.fragments.values().map(|fragment| {
927            (
928                fragment.node.as_ref().unwrap(),
929                fragment.fragment_type_mask.into(),
930            )
931        }))
932    }
933
934    /// Returns `Ok((Some(``snapshot_backfill_info``), ``cross_db_snapshot_backfill_info``))`
935    pub fn collect_snapshot_backfill_info_impl(
936        fragments: impl IntoIterator<Item = (&PbStreamNode, FragmentTypeMask)>,
937    ) -> MetaResult<(Option<SnapshotBackfillInfo>, SnapshotBackfillInfo)> {
938        let mut prev_stream_scan: Option<(Option<SnapshotBackfillInfo>, StreamScanNode)> = None;
939        let mut cross_db_info = SnapshotBackfillInfo {
940            upstream_mv_table_id_to_backfill_epoch: Default::default(),
941        };
942        let mut result = Ok(());
943        for (node, fragment_type_mask) in fragments {
944            visit_stream_node_cont(node, |node| {
945                if let Some(NodeBody::StreamScan(stream_scan)) = node.node_body.as_ref() {
946                    let stream_scan_type = StreamScanType::try_from(stream_scan.stream_scan_type)
947                        .expect("invalid stream_scan_type");
948                    let is_snapshot_backfill = match stream_scan_type {
949                        StreamScanType::SnapshotBackfill => {
950                            assert!(
951                                fragment_type_mask
952                                    .contains(FragmentTypeFlag::SnapshotBackfillStreamScan)
953                            );
954                            true
955                        }
956                        StreamScanType::CrossDbSnapshotBackfill => {
957                            assert!(
958                                fragment_type_mask
959                                    .contains(FragmentTypeFlag::CrossDbSnapshotBackfillStreamScan)
960                            );
961                            cross_db_info
962                                .upstream_mv_table_id_to_backfill_epoch
963                                .insert(stream_scan.table_id, stream_scan.snapshot_backfill_epoch);
964
965                            return true;
966                        }
967                        _ => false,
968                    };
969
970                    match &mut prev_stream_scan {
971                        Some((prev_snapshot_backfill_info, prev_stream_scan)) => {
972                            match (prev_snapshot_backfill_info, is_snapshot_backfill) {
973                                (Some(prev_snapshot_backfill_info), true) => {
974                                    prev_snapshot_backfill_info
975                                        .upstream_mv_table_id_to_backfill_epoch
976                                        .insert(
977                                            stream_scan.table_id,
978                                            stream_scan.snapshot_backfill_epoch,
979                                        );
980                                    true
981                                }
982                                (None, false) => true,
983                                (_, _) => {
984                                    result = Err(anyhow!("must be either all snapshot_backfill or no snapshot_backfill. Curr: {stream_scan:?} Prev: {prev_stream_scan:?}").into());
985                                    false
986                                }
987                            }
988                        }
989                        None => {
990                            prev_stream_scan = Some((
991                                if is_snapshot_backfill {
992                                    Some(SnapshotBackfillInfo {
993                                        upstream_mv_table_id_to_backfill_epoch: HashMap::from_iter(
994                                            [(
995                                                stream_scan.table_id,
996                                                stream_scan.snapshot_backfill_epoch,
997                                            )],
998                                        ),
999                                    })
1000                                } else {
1001                                    None
1002                                },
1003                                *stream_scan.clone(),
1004                            ));
1005                            true
1006                        }
1007                    }
1008                } else {
1009                    true
1010                }
1011            })
1012        }
1013        result.map(|_| {
1014            (
1015                prev_stream_scan
1016                    .map(|(snapshot_backfill_info, _)| snapshot_backfill_info)
1017                    .unwrap_or(None),
1018                cross_db_info,
1019            )
1020        })
1021    }
1022
1023    /// Collect the mapping from table / `source_id` -> `fragment_id`
1024    pub fn collect_backfill_mapping(&self) -> HashMap<u32, Vec<FragmentId>> {
1025        let mut mapping = HashMap::new();
1026        for (fragment_id, fragment) in &self.fragments {
1027            let fragment_id = fragment_id.as_global_id();
1028            let fragment_mask = fragment.fragment_type_mask;
1029            let candidates = [FragmentTypeFlag::StreamScan, FragmentTypeFlag::SourceScan];
1030            let has_some_scan = candidates
1031                .into_iter()
1032                .any(|flag| (fragment_mask & flag as u32) > 0);
1033            if has_some_scan {
1034                visit_stream_node_cont(fragment.node.as_ref().unwrap(), |node| {
1035                    match node.node_body.as_ref() {
1036                        Some(NodeBody::StreamScan(stream_scan)) => {
1037                            let table_id = stream_scan.table_id;
1038                            let fragments: &mut Vec<_> =
1039                                mapping.entry(table_id.as_raw_id()).or_default();
1040                            fragments.push(fragment_id);
1041                            // each fragment should have only 1 scan node.
1042                            false
1043                        }
1044                        Some(NodeBody::SourceBackfill(source_backfill)) => {
1045                            let source_id = source_backfill.upstream_source_id;
1046                            let fragments: &mut Vec<_> =
1047                                mapping.entry(source_id.as_raw_id()).or_default();
1048                            fragments.push(fragment_id);
1049                            // each fragment should have only 1 scan node.
1050                            false
1051                        }
1052                        _ => true,
1053                    }
1054                })
1055            }
1056        }
1057        mapping
1058    }
1059
1060    /// Initially the mapping that comes from frontend is between `table_ids`.
1061    /// We should remap it to fragment level, since we track progress by actor, and we can get
1062    /// a fragment <-> actor mapping
1063    pub fn create_fragment_backfill_ordering(&self) -> FragmentBackfillOrder {
1064        let mapping = self.collect_backfill_mapping();
1065        let mut fragment_ordering: HashMap<FragmentId, Vec<FragmentId>> = HashMap::new();
1066
1067        // 1. Add backfill dependencies
1068        for (rel_id, downstream_rel_ids) in &self.backfill_order.order {
1069            let fragment_ids = mapping.get(rel_id).unwrap();
1070            for fragment_id in fragment_ids {
1071                let downstream_fragment_ids = downstream_rel_ids
1072                    .data
1073                    .iter()
1074                    .flat_map(|downstream_rel_id| mapping.get(downstream_rel_id).unwrap().iter())
1075                    .copied()
1076                    .collect();
1077                fragment_ordering.insert(*fragment_id, downstream_fragment_ids);
1078            }
1079        }
1080
1081        // If no backfill order is specified, we still need to ensure that all backfill fragments
1082        // run before LocalityProvider fragments.
1083        if fragment_ordering.is_empty() {
1084            for value in mapping.values() {
1085                for &fragment_id in value {
1086                    fragment_ordering.entry(fragment_id).or_default();
1087                }
1088            }
1089        }
1090
1091        // 2. Add dependencies: all backfill fragments should run before LocalityProvider fragments
1092        let locality_provider_dependencies = self.find_locality_provider_dependencies();
1093
1094        let backfill_fragments: HashSet<FragmentId> = mapping.values().flatten().copied().collect();
1095
1096        // Calculate LocalityProvider root fragments (zero indegree)
1097        // Root fragments are those that appear as keys but never appear as downstream dependencies
1098        let all_locality_provider_fragments: HashSet<FragmentId> =
1099            locality_provider_dependencies.keys().copied().collect();
1100        let downstream_locality_provider_fragments: HashSet<FragmentId> =
1101            locality_provider_dependencies
1102                .values()
1103                .flatten()
1104                .copied()
1105                .collect();
1106        let locality_provider_root_fragments: Vec<FragmentId> = all_locality_provider_fragments
1107            .difference(&downstream_locality_provider_fragments)
1108            .copied()
1109            .collect();
1110
1111        // For each backfill fragment, add only the root LocalityProvider fragments as dependents
1112        // This ensures backfill completes before any LocalityProvider starts, while minimizing dependencies
1113        for &backfill_fragment_id in &backfill_fragments {
1114            fragment_ordering
1115                .entry(backfill_fragment_id)
1116                .or_default()
1117                .extend(locality_provider_root_fragments.iter().copied());
1118        }
1119
1120        // 3. Add LocalityProvider internal dependencies
1121        for (fragment_id, downstream_fragments) in locality_provider_dependencies {
1122            fragment_ordering
1123                .entry(fragment_id)
1124                .or_default()
1125                .extend(downstream_fragments);
1126        }
1127
1128        // Deduplicate downstream entries per fragment; overlaps are common when the same fragment
1129        // is reached via multiple paths (e.g., with StreamShare) and would otherwise appear
1130        // multiple times.
1131        for downstream in fragment_ordering.values_mut() {
1132            let mut seen = HashSet::new();
1133            downstream.retain(|id| seen.insert(*id));
1134        }
1135
1136        fragment_ordering
1137    }
1138
1139    pub fn find_locality_provider_fragment_state_table_mapping(
1140        &self,
1141    ) -> HashMap<FragmentId, Vec<TableId>> {
1142        let mut mapping: HashMap<FragmentId, Vec<TableId>> = HashMap::new();
1143
1144        for (fragment_id, fragment) in &self.fragments {
1145            let fragment_id = fragment_id.as_global_id();
1146
1147            // Check if this fragment contains a LocalityProvider node
1148            if let Some(node) = fragment.node.as_ref() {
1149                let mut state_table_ids = Vec::new();
1150
1151                visit_stream_node_cont(node, |stream_node| {
1152                    if let Some(NodeBody::LocalityProvider(locality_provider)) =
1153                        stream_node.node_body.as_ref()
1154                    {
1155                        // Collect state table ID (except the progress table)
1156                        let state_table_id = locality_provider
1157                            .state_table
1158                            .as_ref()
1159                            .expect("must have state table")
1160                            .id;
1161                        state_table_ids.push(state_table_id);
1162                        false // Stop visiting once we find a LocalityProvider
1163                    } else {
1164                        true // Continue visiting
1165                    }
1166                });
1167
1168                if !state_table_ids.is_empty() {
1169                    mapping.insert(fragment_id, state_table_ids);
1170                }
1171            }
1172        }
1173
1174        mapping
1175    }
1176
1177    /// Find dependency relationships among fragments containing `LocalityProvider` nodes.
1178    /// Returns a mapping where each fragment ID maps to a list of fragment IDs that should be processed after it.
1179    /// Following the same semantics as `FragmentBackfillOrder`:
1180    /// `G[10] -> [1, 2, 11]` means `LocalityProvider` in fragment 10 should be processed
1181    /// before `LocalityProviders` in fragments 1, 2, and 11.
1182    ///
1183    /// This method assumes each fragment contains at most one `LocalityProvider` node.
1184    pub fn find_locality_provider_dependencies(&self) -> HashMap<FragmentId, Vec<FragmentId>> {
1185        let mut locality_provider_fragments = HashSet::new();
1186        let mut dependencies: HashMap<FragmentId, Vec<FragmentId>> = HashMap::new();
1187
1188        // First, identify all fragments that contain LocalityProvider nodes
1189        for (fragment_id, fragment) in &self.fragments {
1190            let fragment_id = fragment_id.as_global_id();
1191            let has_locality_provider = self.fragment_has_locality_provider(fragment);
1192
1193            if has_locality_provider {
1194                locality_provider_fragments.insert(fragment_id);
1195                dependencies.entry(fragment_id).or_default();
1196            }
1197        }
1198
1199        // Build dependency relationships between LocalityProvider fragments
1200        // For each LocalityProvider fragment, find all downstream LocalityProvider fragments
1201        // The upstream fragment should be processed before the downstream fragments
1202        for &provider_fragment_id in &locality_provider_fragments {
1203            let provider_fragment_global_id = GlobalFragmentId::new(provider_fragment_id);
1204
1205            // Find all fragments downstream from this LocalityProvider fragment
1206            let mut visited = HashSet::new();
1207            let mut downstream_locality_providers = Vec::new();
1208
1209            self.collect_downstream_locality_providers(
1210                provider_fragment_global_id,
1211                &locality_provider_fragments,
1212                &mut visited,
1213                &mut downstream_locality_providers,
1214            );
1215
1216            // This fragment should be processed before all its downstream LocalityProvider fragments
1217            dependencies
1218                .entry(provider_fragment_id)
1219                .or_default()
1220                .extend(downstream_locality_providers);
1221        }
1222
1223        dependencies
1224    }
1225
1226    fn fragment_has_locality_provider(&self, fragment: &BuildingFragment) -> bool {
1227        let mut has_locality_provider = false;
1228
1229        if let Some(node) = fragment.node.as_ref() {
1230            visit_stream_node_cont(node, |stream_node| {
1231                if let Some(NodeBody::LocalityProvider(_)) = stream_node.node_body.as_ref() {
1232                    has_locality_provider = true;
1233                    false // Stop visiting once we find a LocalityProvider
1234                } else {
1235                    true // Continue visiting
1236                }
1237            });
1238        }
1239
1240        has_locality_provider
1241    }
1242
1243    /// Recursively collect downstream `LocalityProvider` fragments
1244    fn collect_downstream_locality_providers(
1245        &self,
1246        current_fragment_id: GlobalFragmentId,
1247        locality_provider_fragments: &HashSet<FragmentId>,
1248        visited: &mut HashSet<GlobalFragmentId>,
1249        downstream_providers: &mut Vec<FragmentId>,
1250    ) {
1251        if visited.contains(&current_fragment_id) {
1252            return;
1253        }
1254        visited.insert(current_fragment_id);
1255
1256        // Check all downstream fragments
1257        for &downstream_id in self.get_downstreams(current_fragment_id).keys() {
1258            let downstream_fragment_id = downstream_id.as_global_id();
1259
1260            // If the downstream fragment is a LocalityProvider, add it to results
1261            if locality_provider_fragments.contains(&downstream_fragment_id) {
1262                downstream_providers.push(downstream_fragment_id);
1263            }
1264
1265            // Recursively check further downstream
1266            self.collect_downstream_locality_providers(
1267                downstream_id,
1268                locality_provider_fragments,
1269                visited,
1270                downstream_providers,
1271            );
1272        }
1273    }
1274}
1275
1276/// Fill snapshot epoch for `StreamScanNode` of `SnapshotBackfill`.
1277/// Return `true` when has change applied.
1278pub fn fill_snapshot_backfill_epoch(
1279    node: &mut StreamNode,
1280    snapshot_backfill_info: Option<&SnapshotBackfillInfo>,
1281    cross_db_snapshot_backfill_info: &SnapshotBackfillInfo,
1282) -> MetaResult<bool> {
1283    let mut result = Ok(());
1284    let mut applied = false;
1285    visit_stream_node_cont_mut(node, |node| {
1286        if let Some(NodeBody::StreamScan(stream_scan)) = node.node_body.as_mut()
1287            && (stream_scan.stream_scan_type == StreamScanType::SnapshotBackfill as i32
1288                || stream_scan.stream_scan_type == StreamScanType::CrossDbSnapshotBackfill as i32)
1289        {
1290            result = try {
1291                let table_id = stream_scan.table_id;
1292                let snapshot_epoch = cross_db_snapshot_backfill_info
1293                    .upstream_mv_table_id_to_backfill_epoch
1294                    .get(&table_id)
1295                    .or_else(|| {
1296                        snapshot_backfill_info.and_then(|snapshot_backfill_info| {
1297                            snapshot_backfill_info
1298                                .upstream_mv_table_id_to_backfill_epoch
1299                                .get(&table_id)
1300                        })
1301                    })
1302                    .ok_or_else(|| anyhow!("upstream table id not covered: {}", table_id))?
1303                    .ok_or_else(|| anyhow!("upstream table id not set: {}", table_id))?;
1304                if let Some(prev_snapshot_epoch) =
1305                    stream_scan.snapshot_backfill_epoch.replace(snapshot_epoch)
1306                {
1307                    Err(anyhow!(
1308                        "snapshot backfill epoch set again: {} {} {}",
1309                        table_id,
1310                        prev_snapshot_epoch,
1311                        snapshot_epoch
1312                    ))?;
1313                }
1314                applied = true;
1315            };
1316            result.is_ok()
1317        } else {
1318            true
1319        }
1320    });
1321    result.map(|_| applied)
1322}
1323
1324static EMPTY_HASHMAP: LazyLock<HashMap<GlobalFragmentId, StreamFragmentEdge>> =
1325    LazyLock::new(HashMap::new);
1326
1327/// A fragment that is either being built or already exists. Used for generalize the logic of
1328/// [`crate::stream::ActorGraphBuilder`].
1329#[derive(Debug, Clone, EnumAsInner)]
1330pub(super) enum EitherFragment {
1331    /// An internal fragment that is being built for the current streaming job.
1332    Building(BuildingFragment),
1333
1334    /// An existing fragment that is external but connected to the fragments being built.
1335    Existing(SharedFragmentInfo),
1336}
1337
1338/// A wrapper of [`StreamFragmentGraph`] that contains the additional information of pre-existing
1339/// fragments, which are connected to the graph's top-most or bottom-most fragments.
1340///
1341/// For example,
1342/// - if we're going to build a mview on an existing mview, the upstream fragment containing the
1343///   `Materialize` node will be included in this structure.
1344/// - if we're going to replace the plan of a table with downstream mviews, the downstream fragments
1345///   containing the `StreamScan` nodes will be included in this structure.
1346#[derive(Debug)]
1347pub struct CompleteStreamFragmentGraph {
1348    /// The fragment graph of the streaming job being built.
1349    building_graph: StreamFragmentGraph,
1350
1351    /// The required information of existing fragments.
1352    existing_fragments: HashMap<GlobalFragmentId, SharedFragmentInfo>,
1353
1354    /// The location of the actors in the existing fragments.
1355    existing_actor_location: HashMap<ActorId, WorkerId>,
1356
1357    /// Extra edges between existing fragments and the building fragments.
1358    extra_downstreams: HashMap<GlobalFragmentId, HashMap<GlobalFragmentId, StreamFragmentEdge>>,
1359
1360    /// Extra edges between existing fragments and the building fragments.
1361    extra_upstreams: HashMap<GlobalFragmentId, HashMap<GlobalFragmentId, StreamFragmentEdge>>,
1362}
1363
1364pub struct FragmentGraphUpstreamContext {
1365    /// Root fragment is the root of upstream stream graph, which can be a
1366    /// mview fragment or source fragment for cdc source job
1367    pub upstream_root_fragments: HashMap<JobId, (SharedFragmentInfo, PbStreamNode)>,
1368    pub upstream_actor_location: HashMap<ActorId, WorkerId>,
1369}
1370
1371pub struct FragmentGraphDownstreamContext {
1372    pub original_root_fragment_id: FragmentId,
1373    pub downstream_fragments: Vec<(DispatcherType, SharedFragmentInfo, PbStreamNode)>,
1374    pub downstream_actor_location: HashMap<ActorId, WorkerId>,
1375}
1376
1377impl CompleteStreamFragmentGraph {
1378    /// Create a new [`CompleteStreamFragmentGraph`] with empty existing fragments, i.e., there's no
1379    /// upstream mviews.
1380    #[cfg(test)]
1381    pub fn for_test(graph: StreamFragmentGraph) -> Self {
1382        Self {
1383            building_graph: graph,
1384            existing_fragments: Default::default(),
1385            existing_actor_location: Default::default(),
1386            extra_downstreams: Default::default(),
1387            extra_upstreams: Default::default(),
1388        }
1389    }
1390
1391    /// Create a new [`CompleteStreamFragmentGraph`] for newly created job (which has no downstreams).
1392    /// e.g., MV on MV and CDC/Source Table with the upstream existing
1393    /// `Materialize` or `Source` fragments.
1394    pub fn with_upstreams(
1395        graph: StreamFragmentGraph,
1396        upstream_context: FragmentGraphUpstreamContext,
1397        job_type: StreamingJobType,
1398    ) -> MetaResult<Self> {
1399        Self::build_helper(graph, Some(upstream_context), None, job_type)
1400    }
1401
1402    /// Create a new [`CompleteStreamFragmentGraph`] for replacing an existing table/source,
1403    /// with the downstream existing `StreamScan`/`StreamSourceScan` fragments.
1404    pub fn with_downstreams(
1405        graph: StreamFragmentGraph,
1406        downstream_context: FragmentGraphDownstreamContext,
1407        job_type: StreamingJobType,
1408    ) -> MetaResult<Self> {
1409        Self::build_helper(graph, None, Some(downstream_context), job_type)
1410    }
1411
1412    /// For replacing an existing table based on shared cdc source, which has both upstreams and downstreams.
1413    pub fn with_upstreams_and_downstreams(
1414        graph: StreamFragmentGraph,
1415        upstream_context: FragmentGraphUpstreamContext,
1416        downstream_context: FragmentGraphDownstreamContext,
1417        job_type: StreamingJobType,
1418    ) -> MetaResult<Self> {
1419        Self::build_helper(
1420            graph,
1421            Some(upstream_context),
1422            Some(downstream_context),
1423            job_type,
1424        )
1425    }
1426
1427    /// The core logic of building a [`CompleteStreamFragmentGraph`], i.e., adding extra upstream/downstream fragments.
1428    fn build_helper(
1429        mut graph: StreamFragmentGraph,
1430        upstream_ctx: Option<FragmentGraphUpstreamContext>,
1431        downstream_ctx: Option<FragmentGraphDownstreamContext>,
1432        job_type: StreamingJobType,
1433    ) -> MetaResult<Self> {
1434        let mut extra_downstreams = HashMap::new();
1435        let mut extra_upstreams = HashMap::new();
1436        let mut existing_fragments = HashMap::new();
1437
1438        let mut existing_actor_location = HashMap::new();
1439
1440        if let Some(FragmentGraphUpstreamContext {
1441            upstream_root_fragments,
1442            upstream_actor_location,
1443        }) = upstream_ctx
1444        {
1445            for (&id, fragment) in &mut graph.fragments {
1446                let uses_shuffled_backfill = fragment.has_shuffled_backfill();
1447
1448                for (&upstream_job_id, required_columns) in &fragment.upstream_job_columns {
1449                    let (upstream_fragment, nodes) = upstream_root_fragments
1450                        .get(&upstream_job_id)
1451                        .context("upstream fragment not found")?;
1452                    let upstream_root_fragment_id =
1453                        GlobalFragmentId::new(upstream_fragment.fragment_id);
1454
1455                    let edge = match job_type {
1456                        StreamingJobType::Table(TableJobType::SharedCdcSource) => {
1457                            // we traverse all fragments in the graph, and we should find out the
1458                            // CdcFilter fragment and add an edge between upstream source fragment and it.
1459                            assert_ne!(
1460                                (fragment.fragment_type_mask & FragmentTypeFlag::CdcFilter as u32),
1461                                0
1462                            );
1463
1464                            tracing::debug!(
1465                                ?upstream_root_fragment_id,
1466                                ?required_columns,
1467                                identity = ?fragment.inner.get_node().unwrap().get_identity(),
1468                                current_frag_id=?id,
1469                                "CdcFilter with upstream source fragment"
1470                            );
1471
1472                            StreamFragmentEdge {
1473                                id: EdgeId::UpstreamExternal {
1474                                    upstream_job_id,
1475                                    downstream_fragment_id: id,
1476                                },
1477                                // We always use `NoShuffle` for the exchange between the upstream
1478                                // `Source` and the downstream `StreamScan` of the new cdc table.
1479                                dispatch_strategy: DispatchStrategy {
1480                                    r#type: DispatcherType::NoShuffle as _,
1481                                    dist_key_indices: vec![], // not used for `NoShuffle`
1482                                    output_mapping: DispatchOutputMapping::identical(
1483                                        CDC_SOURCE_COLUMN_NUM as _,
1484                                    )
1485                                    .into(),
1486                                },
1487                            }
1488                        }
1489
1490                        // handle MV on MV/Source
1491                        StreamingJobType::MaterializedView
1492                        | StreamingJobType::Sink
1493                        | StreamingJobType::Index => {
1494                            // Build the extra edges between the upstream `Materialize` and
1495                            // the downstream `StreamScan` of the new job.
1496                            if upstream_fragment
1497                                .fragment_type_mask
1498                                .contains(FragmentTypeFlag::Mview)
1499                            {
1500                                // Resolve the required output columns from the upstream materialized view.
1501                                let (dist_key_indices, output_mapping) = {
1502                                    let mview_node =
1503                                        nodes.get_node_body().unwrap().as_materialize().unwrap();
1504                                    let all_columns = mview_node.column_descs();
1505                                    let dist_key_indices = mview_node.dist_key_indices();
1506                                    let output_mapping = gen_output_mapping(
1507                                        required_columns,
1508                                        &all_columns,
1509                                    )
1510                                    .context(
1511                                        "BUG: column not found in the upstream materialized view",
1512                                    )?;
1513                                    (dist_key_indices, output_mapping)
1514                                };
1515                                let dispatch_strategy = mv_on_mv_dispatch_strategy(
1516                                    uses_shuffled_backfill,
1517                                    dist_key_indices,
1518                                    output_mapping,
1519                                );
1520
1521                                StreamFragmentEdge {
1522                                    id: EdgeId::UpstreamExternal {
1523                                        upstream_job_id,
1524                                        downstream_fragment_id: id,
1525                                    },
1526                                    dispatch_strategy,
1527                                }
1528                            }
1529                            // Build the extra edges between the upstream `Source` and
1530                            // the downstream `SourceBackfill` of the new job.
1531                            else if upstream_fragment
1532                                .fragment_type_mask
1533                                .contains(FragmentTypeFlag::Source)
1534                            {
1535                                let output_mapping = {
1536                                    let source_node =
1537                                        nodes.get_node_body().unwrap().as_source().unwrap();
1538
1539                                    let all_columns = source_node.column_descs().unwrap();
1540                                    gen_output_mapping(required_columns, &all_columns).context(
1541                                        "BUG: column not found in the upstream source node",
1542                                    )?
1543                                };
1544
1545                                StreamFragmentEdge {
1546                                    id: EdgeId::UpstreamExternal {
1547                                        upstream_job_id,
1548                                        downstream_fragment_id: id,
1549                                    },
1550                                    // We always use `NoShuffle` for the exchange between the upstream
1551                                    // `Source` and the downstream `StreamScan` of the new MV.
1552                                    dispatch_strategy: DispatchStrategy {
1553                                        r#type: DispatcherType::NoShuffle as _,
1554                                        dist_key_indices: vec![], // not used for `NoShuffle`
1555                                        output_mapping: Some(output_mapping),
1556                                    },
1557                                }
1558                            } else {
1559                                bail!(
1560                                    "the upstream fragment should be a MView or Source, got fragment type: {:b}",
1561                                    upstream_fragment.fragment_type_mask
1562                                )
1563                            }
1564                        }
1565                        StreamingJobType::Source | StreamingJobType::Table(_) => {
1566                            bail!(
1567                                "the streaming job shouldn't have an upstream fragment, job_type: {:?}",
1568                                job_type
1569                            )
1570                        }
1571                    };
1572
1573                    // put the edge into the extra edges
1574                    extra_downstreams
1575                        .entry(upstream_root_fragment_id)
1576                        .or_insert_with(HashMap::new)
1577                        .try_insert(id, edge.clone())
1578                        .unwrap();
1579                    extra_upstreams
1580                        .entry(id)
1581                        .or_insert_with(HashMap::new)
1582                        .try_insert(upstream_root_fragment_id, edge)
1583                        .unwrap();
1584                }
1585            }
1586
1587            existing_fragments.extend(
1588                upstream_root_fragments
1589                    .into_values()
1590                    .map(|(f, _)| (GlobalFragmentId::new(f.fragment_id), f)),
1591            );
1592
1593            existing_actor_location.extend(upstream_actor_location);
1594        }
1595
1596        if let Some(FragmentGraphDownstreamContext {
1597            original_root_fragment_id,
1598            downstream_fragments,
1599            downstream_actor_location,
1600        }) = downstream_ctx
1601        {
1602            let original_table_fragment_id = GlobalFragmentId::new(original_root_fragment_id);
1603            let table_fragment_id = GlobalFragmentId::new(graph.table_fragment_id());
1604
1605            // Build the extra edges between the `Materialize` and the downstream `StreamScan` of the
1606            // existing materialized views.
1607            for (dispatcher_type, fragment, nodes) in &downstream_fragments {
1608                let id = GlobalFragmentId::new(fragment.fragment_id);
1609
1610                // Similar to `extract_upstream_columns_except_cross_db_backfill`.
1611                let output_columns = {
1612                    let mut res = None;
1613
1614                    stream_graph_visitor::visit_stream_node_body(nodes, |node_body| {
1615                        let columns = match node_body {
1616                            NodeBody::StreamScan(stream_scan) => stream_scan.upstream_columns(),
1617                            NodeBody::SourceBackfill(source_backfill) => {
1618                                // FIXME: only pass required columns instead of all columns here
1619                                source_backfill.column_descs()
1620                            }
1621                            _ => return,
1622                        };
1623                        res = Some(columns);
1624                    });
1625
1626                    res.context("failed to locate downstream scan")?
1627                };
1628
1629                let table_fragment = graph.fragments.get(&table_fragment_id).unwrap();
1630                let nodes = table_fragment.node.as_ref().unwrap();
1631
1632                let (dist_key_indices, output_mapping) = match job_type {
1633                    StreamingJobType::Table(_) | StreamingJobType::MaterializedView => {
1634                        let mview_node = nodes.get_node_body().unwrap().as_materialize().unwrap();
1635                        let all_columns = mview_node.column_descs();
1636                        let dist_key_indices = mview_node.dist_key_indices();
1637                        let output_mapping = gen_output_mapping(&output_columns, &all_columns)
1638                            .ok_or_else(|| {
1639                                MetaError::invalid_parameter(
1640                                    "unable to drop the column due to \
1641                                     being referenced by downstream materialized views or sinks",
1642                                )
1643                            })?;
1644                        (dist_key_indices, output_mapping)
1645                    }
1646
1647                    StreamingJobType::Source => {
1648                        let source_node = nodes.get_node_body().unwrap().as_source().unwrap();
1649                        let all_columns = source_node.column_descs().unwrap();
1650                        let output_mapping = gen_output_mapping(&output_columns, &all_columns)
1651                            .ok_or_else(|| {
1652                                MetaError::invalid_parameter(
1653                                    "unable to drop the column due to \
1654                                     being referenced by downstream materialized views or sinks",
1655                                )
1656                            })?;
1657                        assert_eq!(*dispatcher_type, DispatcherType::NoShuffle);
1658                        (
1659                            vec![], // not used for `NoShuffle`
1660                            output_mapping,
1661                        )
1662                    }
1663
1664                    _ => bail!("unsupported job type for replacement: {job_type:?}"),
1665                };
1666
1667                let edge = StreamFragmentEdge {
1668                    id: EdgeId::DownstreamExternal(DownstreamExternalEdgeId {
1669                        original_upstream_fragment_id: original_table_fragment_id,
1670                        downstream_fragment_id: id,
1671                    }),
1672                    dispatch_strategy: DispatchStrategy {
1673                        r#type: *dispatcher_type as i32,
1674                        output_mapping: Some(output_mapping),
1675                        dist_key_indices,
1676                    },
1677                };
1678
1679                extra_downstreams
1680                    .entry(table_fragment_id)
1681                    .or_insert_with(HashMap::new)
1682                    .try_insert(id, edge.clone())
1683                    .unwrap();
1684                extra_upstreams
1685                    .entry(id)
1686                    .or_insert_with(HashMap::new)
1687                    .try_insert(table_fragment_id, edge)
1688                    .unwrap();
1689            }
1690
1691            existing_fragments.extend(
1692                downstream_fragments
1693                    .into_iter()
1694                    .map(|(_, f, _)| (GlobalFragmentId::new(f.fragment_id), f)),
1695            );
1696
1697            existing_actor_location.extend(downstream_actor_location);
1698        }
1699
1700        Ok(Self {
1701            building_graph: graph,
1702            existing_fragments,
1703            existing_actor_location,
1704            extra_downstreams,
1705            extra_upstreams,
1706        })
1707    }
1708}
1709
1710/// Generate the `output_mapping` for [`DispatchStrategy`] from given columns.
1711fn gen_output_mapping(
1712    required_columns: &[PbColumnDesc],
1713    upstream_columns: &[PbColumnDesc],
1714) -> Option<DispatchOutputMapping> {
1715    let len = required_columns.len();
1716    let mut indices = vec![0; len];
1717    let mut types = None;
1718
1719    for (i, r) in required_columns.iter().enumerate() {
1720        let (ui, u) = upstream_columns
1721            .iter()
1722            .find_position(|&u| u.column_id == r.column_id)?;
1723        indices[i] = ui as u32;
1724
1725        // Only if we encounter type change (`ALTER TABLE ALTER COLUMN TYPE`) will we generate a
1726        // non-empty `types`.
1727        if u.column_type != r.column_type {
1728            types.get_or_insert_with(|| vec![TypePair::default(); len])[i] = TypePair {
1729                upstream: u.column_type.clone(),
1730                downstream: r.column_type.clone(),
1731            };
1732        }
1733    }
1734
1735    // If there's no type change, indicate it by empty `types`.
1736    let types = types.unwrap_or(Vec::new());
1737
1738    Some(DispatchOutputMapping { indices, types })
1739}
1740
1741fn mv_on_mv_dispatch_strategy(
1742    uses_shuffled_backfill: bool,
1743    dist_key_indices: Vec<u32>,
1744    output_mapping: DispatchOutputMapping,
1745) -> DispatchStrategy {
1746    if uses_shuffled_backfill {
1747        if !dist_key_indices.is_empty() {
1748            DispatchStrategy {
1749                r#type: DispatcherType::Hash as _,
1750                dist_key_indices,
1751                output_mapping: Some(output_mapping),
1752            }
1753        } else {
1754            DispatchStrategy {
1755                r#type: DispatcherType::Simple as _,
1756                dist_key_indices: vec![], // empty for Simple
1757                output_mapping: Some(output_mapping),
1758            }
1759        }
1760    } else {
1761        DispatchStrategy {
1762            r#type: DispatcherType::NoShuffle as _,
1763            dist_key_indices: vec![], // not used for `NoShuffle`
1764            output_mapping: Some(output_mapping),
1765        }
1766    }
1767}
1768
1769impl CompleteStreamFragmentGraph {
1770    /// Returns **all** fragment IDs in the complete graph, including the ones that are not in the
1771    /// building graph.
1772    pub(super) fn all_fragment_ids(&self) -> impl Iterator<Item = GlobalFragmentId> + '_ {
1773        self.building_graph
1774            .fragments
1775            .keys()
1776            .chain(self.existing_fragments.keys())
1777            .copied()
1778    }
1779
1780    /// Returns an iterator of **all** edges in the complete graph, including the external edges.
1781    pub(super) fn all_edges(
1782        &self,
1783    ) -> impl Iterator<Item = (GlobalFragmentId, GlobalFragmentId, &StreamFragmentEdge)> + '_ {
1784        self.building_graph
1785            .downstreams
1786            .iter()
1787            .chain(self.extra_downstreams.iter())
1788            .flat_map(|(&from, tos)| tos.iter().map(move |(&to, edge)| (from, to, edge)))
1789    }
1790
1791    /// Returns the distribution of the existing fragments.
1792    pub(super) fn existing_distribution(&self) -> HashMap<GlobalFragmentId, Distribution> {
1793        self.existing_fragments
1794            .iter()
1795            .map(|(&id, f)| {
1796                (
1797                    id,
1798                    Distribution::from_fragment(f, &self.existing_actor_location),
1799                )
1800            })
1801            .collect()
1802    }
1803
1804    /// Generate topological order of **all** fragments in this graph, including the ones that are
1805    /// not in the building graph. Returns error if the graph is not a DAG and topological sort can
1806    /// not be done.
1807    ///
1808    /// For MV on MV, the first fragment popped out from the heap will be the top-most node, or the
1809    /// `Sink` / `Materialize` in stream graph.
1810    pub(super) fn topo_order(&self) -> MetaResult<Vec<GlobalFragmentId>> {
1811        let mut topo = Vec::new();
1812        let mut downstream_cnts = HashMap::new();
1813
1814        // Iterate all fragments.
1815        for fragment_id in self.all_fragment_ids() {
1816            // Count how many downstreams we have for a given fragment.
1817            let downstream_cnt = self.get_downstreams(fragment_id).count();
1818            if downstream_cnt == 0 {
1819                topo.push(fragment_id);
1820            } else {
1821                downstream_cnts.insert(fragment_id, downstream_cnt);
1822            }
1823        }
1824
1825        let mut i = 0;
1826        while let Some(&fragment_id) = topo.get(i) {
1827            i += 1;
1828            // Find if we can process more fragments.
1829            for (upstream_job_id, _) in self.get_upstreams(fragment_id) {
1830                let downstream_cnt = downstream_cnts.get_mut(&upstream_job_id).unwrap();
1831                *downstream_cnt -= 1;
1832                if *downstream_cnt == 0 {
1833                    downstream_cnts.remove(&upstream_job_id);
1834                    topo.push(upstream_job_id);
1835                }
1836            }
1837        }
1838
1839        if !downstream_cnts.is_empty() {
1840            // There are fragments that are not processed yet.
1841            bail!("graph is not a DAG");
1842        }
1843
1844        Ok(topo)
1845    }
1846
1847    /// Seal a [`BuildingFragment`] from the graph into a [`Fragment`], which will be further used
1848    /// to build actors on the compute nodes and persist into meta store.
1849    pub(super) fn seal_fragment(
1850        &self,
1851        id: GlobalFragmentId,
1852        actors: Vec<StreamActor>,
1853        distribution: Distribution,
1854        stream_node: StreamNode,
1855    ) -> Fragment {
1856        let building_fragment = self.get_fragment(id).into_building().unwrap();
1857        let internal_tables = building_fragment.extract_internal_tables();
1858        let BuildingFragment {
1859            inner,
1860            job_id,
1861            upstream_job_columns: _,
1862        } = building_fragment;
1863
1864        let distribution_type = distribution.to_distribution_type();
1865        let vnode_count = distribution.vnode_count();
1866
1867        let materialized_fragment_id =
1868            if FragmentTypeMask::from(inner.fragment_type_mask).contains(FragmentTypeFlag::Mview) {
1869                job_id.map(JobId::as_mv_table_id)
1870            } else {
1871                None
1872            };
1873
1874        let vector_index_fragment_id =
1875            if inner.fragment_type_mask & FragmentTypeFlag::VectorIndexWrite as u32 != 0 {
1876                job_id.map(JobId::as_mv_table_id)
1877            } else {
1878                None
1879            };
1880
1881        let state_table_ids = internal_tables
1882            .iter()
1883            .map(|t| t.id)
1884            .chain(materialized_fragment_id)
1885            .chain(vector_index_fragment_id)
1886            .collect();
1887
1888        Fragment {
1889            fragment_id: inner.fragment_id,
1890            fragment_type_mask: inner.fragment_type_mask.into(),
1891            distribution_type,
1892            actors,
1893            state_table_ids,
1894            maybe_vnode_count: VnodeCount::set(vnode_count).to_protobuf(),
1895            nodes: stream_node,
1896        }
1897    }
1898
1899    /// Get a fragment from the complete graph, which can be either a building fragment or an
1900    /// existing fragment.
1901    pub(super) fn get_fragment(&self, fragment_id: GlobalFragmentId) -> EitherFragment {
1902        if let Some(fragment) = self.existing_fragments.get(&fragment_id) {
1903            EitherFragment::Existing(fragment.clone())
1904        } else {
1905            EitherFragment::Building(
1906                self.building_graph
1907                    .fragments
1908                    .get(&fragment_id)
1909                    .unwrap()
1910                    .clone(),
1911            )
1912        }
1913    }
1914
1915    /// Get **all** downstreams of a fragment, including the ones that are not in the building
1916    /// graph.
1917    pub(super) fn get_downstreams(
1918        &self,
1919        fragment_id: GlobalFragmentId,
1920    ) -> impl Iterator<Item = (GlobalFragmentId, &StreamFragmentEdge)> {
1921        self.building_graph
1922            .get_downstreams(fragment_id)
1923            .iter()
1924            .chain(
1925                self.extra_downstreams
1926                    .get(&fragment_id)
1927                    .into_iter()
1928                    .flatten(),
1929            )
1930            .map(|(&id, edge)| (id, edge))
1931    }
1932
1933    /// Get **all** upstreams of a fragment, including the ones that are not in the building
1934    /// graph.
1935    pub(super) fn get_upstreams(
1936        &self,
1937        fragment_id: GlobalFragmentId,
1938    ) -> impl Iterator<Item = (GlobalFragmentId, &StreamFragmentEdge)> {
1939        self.building_graph
1940            .get_upstreams(fragment_id)
1941            .iter()
1942            .chain(self.extra_upstreams.get(&fragment_id).into_iter().flatten())
1943            .map(|(&id, edge)| (id, edge))
1944    }
1945
1946    /// Returns all building fragments in the graph.
1947    pub(super) fn building_fragments(&self) -> &HashMap<GlobalFragmentId, BuildingFragment> {
1948        &self.building_graph.fragments
1949    }
1950
1951    /// Returns all building fragments in the graph, mutable.
1952    pub(super) fn building_fragments_mut(
1953        &mut self,
1954    ) -> &mut HashMap<GlobalFragmentId, BuildingFragment> {
1955        &mut self.building_graph.fragments
1956    }
1957
1958    /// Get the expected vnode count of the building graph. See documentation of the field for more details.
1959    pub(super) fn max_parallelism(&self) -> usize {
1960        self.building_graph.max_parallelism()
1961    }
1962}