From 7708a13cfc8e301f8e813dc531f235532d8c50c2 Mon Sep 17 00:00:00 2001 From: xxchan Date: Tue, 2 Jan 2024 15:27:17 +0800 Subject: [PATCH] feat: add kafka backfill frontend --- .git-blame-ignore-revs | 3 + Cargo.lock | 69 +++--- proto/catalog.proto | 13 +- src/common/src/monitor/connection.rs | 3 +- src/common/src/util/iter_util.rs | 26 +++ src/common/src/util/stream_graph_visitor.rs | 3 + .../src/parser/additional_columns.rs | 61 +++++ src/connector/src/source/base.rs | 4 + src/connector/src/source/cdc/mod.rs | 2 +- src/connector/src/source/reader/desc.rs | 74 +----- .../src/binder/relation/table_or_source.rs | 6 + src/frontend/src/catalog/source_catalog.rs | 8 + src/frontend/src/handler/create_source.rs | 15 +- .../src/optimizer/plan_node/logical_source.rs | 23 +- .../plan_node/logical_source_backfill.rs | 207 +++++++++++++++++ src/frontend/src/optimizer/plan_node/mod.rs | 15 +- .../plan_node/stream_cdc_table_scan.rs | 9 +- .../plan_node/stream_source_backfill.rs | 188 +++++++++++++++ src/frontend/src/planner/relation.rs | 19 +- src/frontend/src/stream_fragmenter/mod.rs | 12 +- src/meta/src/barrier/command.rs | 37 ++- src/meta/src/barrier/mod.rs | 1 + src/meta/src/barrier/progress.rs | 8 + src/meta/src/barrier/schedule.rs | 2 + src/meta/src/controller/catalog.rs | 4 +- src/meta/src/controller/fragment.rs | 29 +-- src/meta/src/controller/streaming_job.rs | 3 +- src/meta/src/manager/catalog/database.rs | 6 +- src/meta/src/manager/catalog/fragment.rs | 23 ++ src/meta/src/manager/catalog/mod.rs | 9 +- src/meta/src/manager/metadata.rs | 16 ++ src/meta/src/model/stream.rs | 57 ++--- src/meta/src/rpc/ddl_controller.rs | 9 +- src/meta/src/stream/scale.rs | 3 +- src/meta/src/stream/source_manager.rs | 217 +++++++++++++++--- src/meta/src/stream/stream_graph/actor.rs | 41 +++- src/meta/src/stream/stream_graph/fragment.rs | 145 ++++++++---- src/meta/src/stream/stream_graph/schedule.rs | 1 + src/meta/src/stream/stream_manager.rs | 18 +- src/prost/src/lib.rs | 55 +++++ .../source/kafka_backfill_executor.rs | 10 +- .../source/kafka_backfill_state_table.rs | 1 - 42 files changed, 1169 insertions(+), 286 deletions(-) create mode 100644 src/frontend/src/optimizer/plan_node/logical_source_backfill.rs create mode 100644 src/frontend/src/optimizer/plan_node/stream_source_backfill.rs diff --git a/.git-blame-ignore-revs b/.git-blame-ignore-revs index 6efd862273624..b8ca322d767a8 100644 --- a/.git-blame-ignore-revs +++ b/.git-blame-ignore-revs @@ -39,3 +39,6 @@ d70dba827c303373f3220c9733f7c7443e5c2d37 # chore: cargo +nightly fmt (#13162) (format let-chains) c583e2c6c054764249acf484438c7bf7197765f4 + +# chore: replace all ProstXxx with PbXxx (#8621) +6fd8821f2e053957b183d648bea9c95b6703941f diff --git a/Cargo.lock b/Cargo.lock index 1c074e276553a..727333377a691 100644 --- a/Cargo.lock +++ b/Cargo.lock @@ -968,7 +968,7 @@ dependencies = [ "aws-smithy-types", "aws-types", "bytes", - "fastrand 2.0.1", + "fastrand 2.0.0", "http 0.2.9", "hyper", "time", @@ -1019,7 +1019,7 @@ dependencies = [ "aws-smithy-runtime-api", "aws-smithy-types", "aws-types", - "fastrand 2.0.1", + "fastrand 2.0.0", "http 0.2.9", "percent-encoding", "tracing", @@ -1044,7 +1044,7 @@ dependencies = [ "aws-smithy-types", "aws-smithy-xml", "aws-types", - "fastrand 2.0.1", + "fastrand 2.0.0", "http 0.2.9", "regex", "tracing", @@ -1247,7 +1247,7 @@ dependencies = [ "aws-smithy-runtime-api", "aws-smithy-types", "bytes", - "fastrand 2.0.1", + "fastrand 2.0.0", "http 0.2.9", "http-body", "hyper", @@ -1853,7 +1853,7 @@ checksum = "6ffc30dee200c20b4dcb80572226f42658e1d9c4b668656d7cc59c33d50e396e" dependencies = [ "cap-primitives", "cap-std", - "rustix 0.38.31", + "rustix 0.38.28", "smallvec", ] @@ -1869,7 +1869,7 @@ dependencies = [ "io-lifetimes 2.0.3", "ipnet", "maybe-owned", - "rustix 0.38.31", + "rustix 0.38.28", "windows-sys 0.48.0", "winx", ] @@ -1893,7 +1893,7 @@ dependencies = [ "cap-primitives", "io-extras", "io-lifetimes 2.0.3", - "rustix 0.38.31", + "rustix 0.38.28", ] [[package]] @@ -1904,7 +1904,7 @@ checksum = "f8f52b3c8f4abfe3252fd0a071f3004aaa3b18936ec97bdbd8763ce03aff6247" dependencies = [ "cap-primitives", "once_cell", - "rustix 0.38.31", + "rustix 0.38.28", "winx", ] @@ -3903,9 +3903,9 @@ dependencies = [ [[package]] name = "fastrand" -version = "2.0.1" +version = "2.0.0" source = "registry+https://github.com/rust-lang/crates.io-index" -checksum = "25cbce373ec4653f1a01a31e8a5e5ec0c622dc27ff9c4e6606eefef5cbbed4a5" +checksum = "6999dc1837253364c2ebb0704ba97994bd874e8f195d665c50b7548f6ea92764" [[package]] name = "fd-lock" @@ -3914,7 +3914,7 @@ source = "registry+https://github.com/rust-lang/crates.io-index" checksum = "b93f7a0db71c99f68398f80653ed05afb0b00e062e1a20c7ff849c4edfabbbcc" dependencies = [ "cfg-if", - "rustix 0.38.31", + "rustix 0.38.28", "windows-sys 0.52.0", ] @@ -4278,7 +4278,7 @@ source = "registry+https://github.com/rust-lang/crates.io-index" checksum = "033b337d725b97690d86893f9de22b67b80dcc4e9ad815f348254c38119db8fb" dependencies = [ "io-lifetimes 2.0.3", - "rustix 0.38.31", + "rustix 0.38.28", "windows-sys 0.52.0", ] @@ -5265,7 +5265,7 @@ source = "registry+https://github.com/rust-lang/crates.io-index" checksum = "cb0889898416213fab133e1d33a0e5858a48177452750691bde3666d0fdbaf8b" dependencies = [ "hermit-abi", - "rustix 0.38.31", + "rustix 0.38.28", "windows-sys 0.48.0", ] @@ -5544,9 +5544,9 @@ dependencies = [ [[package]] name = "libc" -version = "0.2.153" +version = "0.2.150" source = "registry+https://github.com/rust-lang/crates.io-index" -checksum = "9c198f91728a82281a64e1f4f9eeb25d82cb32a5de251c6bd1b5154d63a8e7bd" +checksum = "89d92a4743f9a61002fae18374ed11e7973f530cb3a3255fb354818118b2203c" [[package]] name = "libflate" @@ -6048,7 +6048,7 @@ version = "0.6.4" source = "registry+https://github.com/rust-lang/crates.io-index" checksum = "b2cffa4ad52c6f791f4f8b15f0c05f9824b2ced1160e88cc393d64fff9a8ac64" dependencies = [ - "rustix 0.38.31", + "rustix 0.38.28", ] [[package]] @@ -7734,7 +7734,7 @@ dependencies = [ "hex", "lazy_static", "procfs-core", - "rustix 0.38.31", + "rustix 0.38.28", ] [[package]] @@ -10268,9 +10268,9 @@ dependencies = [ [[package]] name = "rustix" -version = "0.38.31" +version = "0.38.28" source = "registry+https://github.com/rust-lang/crates.io-index" -checksum = "6ea3e1a662af26cd7a3ba09c0297a31af215563ecf42817c98df621387f4e949" +checksum = "72e572a5e8ca657d7366229cdde4bd14c4eb5499a9573d4d366fe1b599daa316" dependencies = [ "bitflags 2.4.0", "errno", @@ -11754,7 +11754,7 @@ dependencies = [ "cap-std", "fd-lock", "io-lifetimes 2.0.3", - "rustix 0.38.31", + "rustix 0.38.28", "windows-sys 0.48.0", "winx", ] @@ -11787,13 +11787,14 @@ dependencies = [ [[package]] name = "tempfile" -version = "3.10.0" +version = "3.9.0" source = "registry+https://github.com/rust-lang/crates.io-index" -checksum = "a365e8cd18e44762ef95d87f284f4b5cd04107fec2ff3052bd6a3e6069669e67" +checksum = "01ce4141aa927a6d1bd34a041795abd0db1cccba5d5f24b009f694bdf3a1f3fa" dependencies = [ "cfg-if", - "fastrand 2.0.1", - "rustix 0.38.31", + "fastrand 2.0.0", + "redox_syscall 0.4.1", + "rustix 0.38.28", "windows-sys 0.52.0", ] @@ -12778,7 +12779,7 @@ dependencies = [ "io-extras", "io-lifetimes 2.0.3", "once_cell", - "rustix 0.38.31", + "rustix 0.38.28", "system-interface", "tracing", "wasi-common", @@ -12797,7 +12798,7 @@ dependencies = [ "cap-std", "io-extras", "log", - "rustix 0.38.31", + "rustix 0.38.28", "thiserror", "tracing", "wasmtime", @@ -12972,7 +12973,7 @@ dependencies = [ "bincode 1.3.3", "directories-next", "log", - "rustix 0.38.31", + "rustix 0.38.28", "serde", "serde_derive", "sha2", @@ -13075,7 +13076,7 @@ dependencies = [ "anyhow", "cc", "cfg-if", - "rustix 0.38.31", + "rustix 0.38.28", "wasmtime-asm-macros", "wasmtime-versioned-export-macros", "windows-sys 0.52.0", @@ -13097,7 +13098,7 @@ dependencies = [ "log", "object", "rustc-demangle", - "rustix 0.38.31", + "rustix 0.38.28", "serde", "serde_derive", "target-lexicon", @@ -13116,7 +13117,7 @@ checksum = "dd21fd0f5ca68681d3d5b636eea00f182d0f9d764144469e9257fd7e3f55ae0e" dependencies = [ "object", "once_cell", - "rustix 0.38.31", + "rustix 0.38.28", "wasmtime-versioned-export-macros", ] @@ -13149,7 +13150,7 @@ dependencies = [ "memoffset", "paste", "psm", - "rustix 0.38.31", + "rustix 0.38.28", "sptr", "wasm-encoder", "wasmtime-asm-macros", @@ -13207,7 +13208,7 @@ dependencies = [ "libc", "log", "once_cell", - "rustix 0.38.31", + "rustix 0.38.28", "system-interface", "thiserror", "tokio", @@ -13320,7 +13321,7 @@ dependencies = [ "either", "home", "once_cell", - "rustix 0.38.31", + "rustix 0.38.28", ] [[package]] @@ -13827,7 +13828,7 @@ dependencies = [ "ring 0.16.20", "rust_decimal", "rustc-hash", - "rustix 0.38.31", + "rustix 0.38.28", "scopeguard", "sea-orm", "sea-query", diff --git a/proto/catalog.proto b/proto/catalog.proto index 99fd1b0a69514..60368f4b7dba4 100644 --- a/proto/catalog.proto +++ b/proto/catalog.proto @@ -62,9 +62,16 @@ message StreamSourceInfo { SchemaRegistryNameStrategy name_strategy = 10; optional string key_message_name = 11; plan_common.ExternalTableDesc external_table = 12; - // Whether the stream source is a cdc source streaming job. - // We need this field to differentiate the cdc source job until we fully implement risingwavelabs/rfcs#72. - bool cdc_source_job = 13; + // Whether the stream source has a streaming job. + // This is related with [RFC: Reusable Source Executor](https://github.com/risingwavelabs/rfcs/pull/72). + // Currently, the following sources have streaming jobs: + // - Direct CDC sources (mysql & postgresql) + // - MQ sources (Kafka, Pulsar, Kinesis, etc.) + bool has_streaming_job = 13; + // Only used when `has_streaming_job` is `true`. + // If `false`, `requires_singleton` will be set in the stream plan. + bool is_distributed = 15; + reserved "cdc_source_job"; // deprecated // Options specified by user in the FORMAT ENCODE clause. map format_encode_options = 14; } diff --git a/src/common/src/monitor/connection.rs b/src/common/src/monitor/connection.rs index 2e28102bf5077..5086cd36f81ca 100644 --- a/src/common/src/monitor/connection.rs +++ b/src/common/src/monitor/connection.rs @@ -34,6 +34,7 @@ use prometheus::{ register_int_counter_vec_with_registry, register_int_gauge_vec_with_registry, IntCounter, IntCounterVec, IntGauge, IntGaugeVec, Registry, }; +use thiserror_ext::AsReport; use tokio::io::{AsyncRead, AsyncWrite, ReadBuf}; use tonic::transport::{Channel, Endpoint}; use tracing::{debug, info, warn}; @@ -549,7 +550,7 @@ impl tonic::transport::server::Router { config.tcp_nodelay, config.keepalive_duration, ) - .unwrap(); + .unwrap_or_else(|err| panic!("failed to connect to {listen_addr}: {}", err.as_report())); let incoming = MonitoredConnection::new( incoming, MonitorNewConnectionImpl { diff --git a/src/common/src/util/iter_util.rs b/src/common/src/util/iter_util.rs index 92f19a0ee46fc..7588171ad2f73 100644 --- a/src/common/src/util/iter_util.rs +++ b/src/common/src/util/iter_util.rs @@ -54,3 +54,29 @@ where { a.into_iter().zip_eq_fast(b) } + +pub trait IntoIteratorExt +where + for<'a> &'a Self: IntoIterator, +{ + /// Shorter version of `self.iter().map(f).collect()`. + fn map_collect(&self, f: F) -> BCollection + where + F: FnMut(&A) -> B, + for<'a> &'a Self: IntoIterator, + BCollection: FromIterator, + { + self.into_iter().map(f).collect() + } + + /// Shorter version of `self.iter().map(f).collect_vec()`. + fn map_to_vec(&self, f: F) -> Vec + where + F: FnMut(&A) -> B, + for<'a> &'a Self: IntoIterator, + { + self.map_collect(f) + } +} + +impl IntoIteratorExt for T where for<'a> &'a Self: IntoIterator {} diff --git a/src/common/src/util/stream_graph_visitor.rs b/src/common/src/util/stream_graph_visitor.rs index ce2820752f120..c9518a03c2623 100644 --- a/src/common/src/util/stream_graph_visitor.rs +++ b/src/common/src/util/stream_graph_visitor.rs @@ -187,6 +187,9 @@ pub fn visit_stream_node_tables_inner( always!(source.state_table, "FsFetch"); } } + NodeBody::SourceBackfill(node) => { + always!(node.state_table, "SourceBackfill") + } // Sink NodeBody::Sink(node) => { diff --git a/src/connector/src/parser/additional_columns.rs b/src/connector/src/parser/additional_columns.rs index c1da30f788b3e..214a5a7484e0f 100644 --- a/src/connector/src/parser/additional_columns.rs +++ b/src/connector/src/parser/additional_columns.rs @@ -184,6 +184,67 @@ pub fn build_additional_column_catalog( Ok(catalog) } +pub fn add_partition_offset_cols( + columns: &[ColumnCatalog], + connector_name: &str, +) -> ([bool; 2], [ColumnCatalog; 2]) { + let mut columns_exist = [false; 2]; + let mut last_column_id = columns + .iter() + .map(|c| c.column_desc.column_id) + .max() + .unwrap_or(ColumnId::placeholder()); + + let additional_columns: Vec<_> = { + let compat_col_types = COMPATIBLE_ADDITIONAL_COLUMNS + .get(&*connector_name) + .unwrap_or(&COMMON_COMPATIBLE_ADDITIONAL_COLUMNS); + ["partition", "file", "offset"] + .iter() + .filter_map(|col_type| { + last_column_id = last_column_id.next(); + if compat_col_types.contains(col_type) { + Some( + build_additional_column_catalog( + last_column_id, + &connector_name, + col_type, + None, + None, + None, + false, + ) + .unwrap(), + ) + } else { + None + } + }) + .collect() + }; + assert_eq!(additional_columns.len(), 2); + + // Check if partition/file/offset columns are included explicitly. + for col in columns { + use risingwave_pb::plan_common::additional_column::ColumnType; + match col.column_desc.additional_column { + AdditionalColumn { + column_type: Some(ColumnType::Partition(_) | ColumnType::Filename(_)), + } => { + columns_exist[0] = true; + } + AdditionalColumn { + column_type: Some(ColumnType::Offset(_)), + } => { + columns_exist[1] = true; + } + _ => (), + } + } + + (columns_exist, additional_columns.try_into().unwrap()) +} + fn build_header_catalog( column_id: ColumnId, col_name: &str, diff --git a/src/connector/src/source/base.rs b/src/connector/src/source/base.rs index 7cf1e83372f78..9ed5ffd3177ab 100644 --- a/src/connector/src/source/base.rs +++ b/src/connector/src/source/base.rs @@ -73,8 +73,10 @@ pub trait SourceProperties: TryFromHashmap + Clone + WithOptions { type SplitEnumerator: SplitEnumerator; type SplitReader: SplitReader; + /// Load additional info from `PbSource`. Currently only used by CDC. fn init_from_pb_source(&mut self, _source: &PbSource) {} + /// Load additional info from `ExternalTableDesc`. Currently only used by CDC. fn init_from_pb_cdc_table_desc(&mut self, _table_desc: &ExternalTableDesc) {} } @@ -432,10 +434,12 @@ impl ConnectorProperties { matches!(self, ConnectorProperties::Kinesis(_)) } + /// Load additional info from `PbSource`. Currently only used by CDC. pub fn init_from_pb_source(&mut self, source: &PbSource) { dispatch_source_prop!(self, prop, prop.init_from_pb_source(source)) } + /// Load additional info from `ExternalTableDesc`. Currently only used by CDC. pub fn init_from_pb_cdc_table_desc(&mut self, cdc_table_desc: &ExternalTableDesc) { dispatch_source_prop!(self, prop, prop.init_from_pb_cdc_table_desc(cdc_table_desc)) } diff --git a/src/connector/src/source/cdc/mod.rs b/src/connector/src/source/cdc/mod.rs index 5fc6aefdfefdd..7e20d9cb81778 100644 --- a/src/connector/src/source/cdc/mod.rs +++ b/src/connector/src/source/cdc/mod.rs @@ -140,7 +140,7 @@ where }; self.table_schema = table_schema; if let Some(info) = source.info.as_ref() { - self.is_multi_table_shared = info.cdc_source_job; + self.is_multi_table_shared = info.has_streaming_job; } } diff --git a/src/connector/src/source/reader/desc.rs b/src/connector/src/source/reader/desc.rs index a842b091ab928..e54c6c042a961 100644 --- a/src/connector/src/source/reader/desc.rs +++ b/src/connector/src/source/reader/desc.rs @@ -15,20 +15,17 @@ use std::collections::HashMap; use std::sync::Arc; +use itertools::Itertools; use risingwave_common::bail; -use risingwave_common::catalog::{ColumnDesc, ColumnId}; +use risingwave_common::catalog::{ColumnCatalog, ColumnDesc}; use risingwave_common::util::iter_util::ZipEqFast; use risingwave_pb::catalog::PbStreamSourceInfo; -use risingwave_pb::plan_common::additional_column::ColumnType; -use risingwave_pb::plan_common::{AdditionalColumn, PbColumnCatalog}; +use risingwave_pb::plan_common::PbColumnCatalog; #[expect(deprecated)] use super::fs_reader::FsSourceReader; use super::reader::SourceReader; -use crate::parser::additional_columns::{ - build_additional_column_catalog, COMMON_COMPATIBLE_ADDITIONAL_COLUMNS, - COMPATIBLE_ADDITIONAL_COLUMNS, -}; +use crate::parser::additional_columns::add_partition_offset_cols; use crate::parser::{EncodingProperties, ProtocolProperties, SpecificParserConfig}; use crate::source::monitor::SourceMetrics; use crate::source::{SourceColumnDesc, SourceColumnType, UPSTREAM_SOURCE_KEY}; @@ -93,65 +90,18 @@ impl SourceDescBuilder { /// This function builds `SourceColumnDesc` from `ColumnCatalog`, and handle the creation /// of hidden columns like partition/file, offset that are not specified by user. pub fn column_catalogs_to_source_column_descs(&self) -> Vec { - let mut columns_exist = [false; 2]; - let mut last_column_id = self - .columns - .iter() - .map(|c| c.column_desc.as_ref().unwrap().column_id.into()) - .max() - .unwrap_or(ColumnId::placeholder()); let connector_name = self .with_properties .get(UPSTREAM_SOURCE_KEY) .map(|s| s.to_lowercase()) .unwrap(); - - let additional_columns: Vec<_> = { - let compat_col_types = COMPATIBLE_ADDITIONAL_COLUMNS - .get(&*connector_name) - .unwrap_or(&COMMON_COMPATIBLE_ADDITIONAL_COLUMNS); - ["partition", "file", "offset"] - .iter() - .filter_map(|col_type| { - last_column_id = last_column_id.next(); - if compat_col_types.contains(col_type) { - Some( - build_additional_column_catalog( - last_column_id, - &connector_name, - col_type, - None, - None, - None, - false, - ) - .unwrap() - .to_protobuf(), - ) - } else { - None - } - }) - .collect() - }; - assert_eq!(additional_columns.len(), 2); - - // Check if partition/file/offset columns are included explicitly. - for col in &self.columns { - match col.column_desc.as_ref().unwrap().get_additional_column() { - Ok(AdditionalColumn { - column_type: Some(ColumnType::Partition(_) | ColumnType::Filename(_)), - }) => { - columns_exist[0] = true; - } - Ok(AdditionalColumn { - column_type: Some(ColumnType::Offset(_)), - }) => { - columns_exist[1] = true; - } - _ => (), - } - } + let columns = self + .columns + .iter() + .map(|c| ColumnCatalog::from(c.clone())) + .collect_vec(); + let (columns_exist, additional_columns) = + add_partition_offset_cols(&columns, &connector_name); let mut columns: Vec<_> = self .columns @@ -162,7 +112,7 @@ impl SourceDescBuilder { for (existed, c) in columns_exist.iter().zip_eq_fast(&additional_columns) { if !existed { columns.push(SourceColumnDesc::hidden_addition_col_from_column_desc( - &ColumnDesc::from(c.column_desc.as_ref().unwrap()), + &c.column_desc, )); } } diff --git a/src/frontend/src/binder/relation/table_or_source.rs b/src/frontend/src/binder/relation/table_or_source.rs index a459efd39f016..01f6cb7619841 100644 --- a/src/frontend/src/binder/relation/table_or_source.rs +++ b/src/frontend/src/binder/relation/table_or_source.rs @@ -58,6 +58,12 @@ impl From<&SourceCatalog> for BoundSource { } } +impl BoundSource { + pub fn can_backfill(&self) -> bool { + self.catalog.info.has_streaming_job + } +} + impl Binder { /// Binds table or source, or logical view according to what we get from the catalog. pub fn bind_relation_by_name_inner( diff --git a/src/frontend/src/catalog/source_catalog.rs b/src/frontend/src/catalog/source_catalog.rs index 59f77bba9fa42..6e9ee89e8d2e1 100644 --- a/src/frontend/src/catalog/source_catalog.rs +++ b/src/frontend/src/catalog/source_catalog.rs @@ -21,6 +21,7 @@ use risingwave_pb::catalog::{PbSource, StreamSourceInfo, WatermarkDesc}; use super::{ColumnId, ConnectionId, DatabaseId, OwnedByUserCatalog, SchemaId, SourceId}; use crate::catalog::TableId; +use crate::handler::create_source::UPSTREAM_SOURCE_KEY; use crate::user::UserId; /// This struct `SourceCatalog` is used in frontend. @@ -83,6 +84,13 @@ impl SourceCatalog { pub fn version(&self) -> SourceVersionId { self.version } + + pub fn connector_name(&self) -> String { + self.with_properties + .get(UPSTREAM_SOURCE_KEY) + .map(|s| s.to_lowercase()) + .unwrap() + } } impl From<&PbSource> for SourceCatalog { diff --git a/src/frontend/src/handler/create_source.rs b/src/frontend/src/handler/create_source.rs index 33cb74a0bf8e7..bc7d32216c8c4 100644 --- a/src/frontend/src/handler/create_source.rs +++ b/src/frontend/src/handler/create_source.rs @@ -479,7 +479,7 @@ fn bind_columns_from_source_for_cdc( row_encode: row_encode_to_prost(&source_schema.row_encode) as i32, format_encode_options, use_schema_registry: json_schema_infer_use_schema_registry(&schema_config), - cdc_source_job: true, + has_streaming_job: true, ..Default::default() }; if !format_encode_options_to_consume.is_empty() { @@ -1161,18 +1161,22 @@ pub async fn handle_create_source( ensure_table_constraints_supported(&stmt.constraints)?; let sql_pk_names = bind_sql_pk_names(&stmt.columns, &stmt.constraints)?; - // gated the feature with a session variable let create_cdc_source_job = if is_cdc_connector(&with_properties) { CdcTableType::from_properties(&with_properties).can_backfill() } else { false }; + let has_streaming_job = create_cdc_source_job || is_kafka_connector(&with_properties); - let (columns_from_resolve_source, source_info) = if create_cdc_source_job { + let (columns_from_resolve_source, mut source_info) = if create_cdc_source_job { bind_columns_from_source_for_cdc(&session, &source_schema, &with_properties)? } else { bind_columns_from_source(&session, &source_schema, &with_properties).await? }; + if has_streaming_job { + source_info.has_streaming_job = true; + source_info.is_distributed = !create_cdc_source_job; + } let columns_from_sql = bind_sql_columns(&stmt.columns)?; let mut columns = bind_all_columns( @@ -1269,18 +1273,15 @@ pub async fn handle_create_source( let catalog_writer = session.catalog_writer()?; - if create_cdc_source_job { - // create a streaming job for the cdc source, which will mark as *singleton* in the Fragmenter + if has_streaming_job { let graph = { let context = OptimizerContext::from_handler_args(handler_args); - // cdc source is an append-only source in plain json format let source_node = LogicalSource::with_catalog( Rc::new(SourceCatalog::from(&source)), SourceNodeKind::CreateSourceWithStreamjob, context.into(), )?; - // generate stream graph for cdc source job let stream_plan = source_node.to_stream(&mut ToStreamContext::new(false))?; let mut graph = build_graph(stream_plan)?; graph.parallelism = diff --git a/src/frontend/src/optimizer/plan_node/logical_source.rs b/src/frontend/src/optimizer/plan_node/logical_source.rs index fa7ad908d01d4..3c330147661b7 100644 --- a/src/frontend/src/optimizer/plan_node/logical_source.rs +++ b/src/frontend/src/optimizer/plan_node/logical_source.rs @@ -23,6 +23,8 @@ use risingwave_common::bail_not_implemented; use risingwave_common::catalog::{ ColumnCatalog, ColumnDesc, Field, Schema, KAFKA_TIMESTAMP_COLUMN_NAME, }; +use risingwave_common::util::iter_util::ZipEqFast; +use risingwave_connector::parser::additional_columns::add_partition_offset_cols; use risingwave_connector::source::DataType; use risingwave_pb::plan_common::column_desc::GeneratedOrDefaultColumn; use risingwave_pb::plan_common::GeneratedColumnDesc; @@ -67,12 +69,27 @@ pub struct LogicalSource { impl LogicalSource { pub fn new( source_catalog: Option>, - column_catalog: Vec, + mut column_catalog: Vec, row_id_index: Option, kind: SourceNodeKind, ctx: OptimizerContextRef, ) -> Result { let kafka_timestamp_range = (Bound::Unbounded, Bound::Unbounded); + + // for sources with streaming job, we will include partition and offset cols in the output. + if let Some(source_catalog) = &source_catalog + && matches!(kind, SourceNodeKind::CreateSourceWithStreamjob) + { + let (columns_exist, additional_columns) = + add_partition_offset_cols(&column_catalog, &source_catalog.connector_name()); + for (existed, mut c) in columns_exist.into_iter().zip_eq_fast(additional_columns) { + c.is_hidden = true; + if !existed { + column_catalog.push(c); + } + } + } + let core = generic::Source { catalog: source_catalog, column_catalog, @@ -506,9 +523,11 @@ impl ToBatch for LogicalSource { impl ToStream for LogicalSource { fn to_stream(&self, _ctx: &mut ToStreamContext) -> Result { let mut plan: PlanRef; + match self.core.kind { SourceNodeKind::CreateTable | SourceNodeKind::CreateSourceWithStreamjob => { - // Note: for create table, row_id and generated columns is created in plan_root.gen_table_plan + // Note: for create table, row_id and generated columns is created in plan_root.gen_table_plan. + // for backfill-able source, row_id and generated columns is created after SourceBackfill node. if self.core.is_new_fs_connector() { plan = Self::create_fs_list_plan(self.core.clone())?; plan = StreamFsFetch::new(plan, self.core.clone()).into(); diff --git a/src/frontend/src/optimizer/plan_node/logical_source_backfill.rs b/src/frontend/src/optimizer/plan_node/logical_source_backfill.rs new file mode 100644 index 0000000000000..4a376794af57c --- /dev/null +++ b/src/frontend/src/optimizer/plan_node/logical_source_backfill.rs @@ -0,0 +1,207 @@ +// 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 std::ops::Bound; +use std::rc::Rc; + +use pretty_xmlish::{Pretty, XmlNode}; +use risingwave_common::util::iter_util::ZipEqFast; +use risingwave_connector::parser::additional_columns::add_partition_offset_cols; + +use super::generic::{GenericPlanRef, SourceNodeKind}; +use super::stream_watermark_filter::StreamWatermarkFilter; +use super::utils::{childless_record, Distill}; +use super::{ + generic, BatchProject, BatchSource, ColPrunable, ExprRewritable, Logical, LogicalFilter, + LogicalProject, LogicalSource, PlanBase, PlanRef, PredicatePushdown, StreamProject, + StreamRowIdGen, ToBatch, ToStream, +}; +use crate::catalog::source_catalog::SourceCatalog; +use crate::error::Result; +use crate::expr::{ExprImpl, ExprRewriter, ExprVisitor}; +use crate::optimizer::optimizer_context::OptimizerContextRef; +use crate::optimizer::plan_node::expr_visitable::ExprVisitable; +use crate::optimizer::plan_node::utils::column_names_pretty; +use crate::optimizer::plan_node::{ + ColumnPruningContext, PredicatePushdownContext, RewriteStreamContext, StreamSourceBackfill, + ToStreamContext, +}; +use crate::optimizer::property::Distribution::HashShard; +use crate::utils::{ColIndexMapping, Condition}; + +#[derive(Debug, Clone, PartialEq, Eq, Hash)] +pub struct LogicalSourceBackfill { + pub base: PlanBase, + pub core: generic::Source, + + /// Expressions to output. This field presents and will be turned to a `Project` when + /// converting to a physical plan, only if there are generated columns. + output_exprs: Option>, + /// When there are generated columns, the `StreamRowIdGen`'s row_id_index is different from + /// the one in `core`. So we store the one in `output_exprs` here. + output_row_id_index: Option, +} + +impl LogicalSourceBackfill { + pub fn new(source_catalog: Rc, ctx: OptimizerContextRef) -> Result { + let mut column_catalog = source_catalog.columns.clone(); + let row_id_index = source_catalog.row_id_index; + + let kafka_timestamp_range = (Bound::Unbounded, Bound::Unbounded); + + let (columns_exist, additional_columns) = + add_partition_offset_cols(&column_catalog, &source_catalog.connector_name()); + for (existed, mut c) in columns_exist.into_iter().zip_eq_fast(additional_columns) { + c.is_hidden = true; + if !existed { + column_catalog.push(c); + } + } + let core = generic::Source { + catalog: Some(source_catalog), + column_catalog, + row_id_index, + // FIXME: this field is not useful for backfill. + kind: SourceNodeKind::CreateMViewOrBatch, + ctx, + kafka_timestamp_range, + }; + + let base = PlanBase::new_logical_with_core(&core); + + let output_exprs = + LogicalSource::derive_output_exprs_from_generated_columns(&core.column_catalog)?; + let (core, output_row_id_index) = core.exclude_generated_columns(); + + Ok(LogicalSourceBackfill { + base, + core, + output_exprs, + output_row_id_index, + }) + } + + pub fn source_catalog(&self) -> Rc { + self.core + .catalog + .clone() + .expect("source catalog should exist for LogicalSourceBackfill") + } +} + +impl_plan_tree_node_for_leaf! {LogicalSourceBackfill} +impl Distill for LogicalSourceBackfill { + fn distill<'a>(&self) -> XmlNode<'a> { + let src = Pretty::from(self.source_catalog().name.clone()); + let time = Pretty::debug(&self.core.kafka_timestamp_range); + let fields = vec![ + ("source", src), + ("columns", column_names_pretty(self.schema())), + ("time_range", time), + ]; + + childless_record("LogicalSourceBackfill", fields) + } +} + +impl ColPrunable for LogicalSourceBackfill { + fn prune_col(&self, required_cols: &[usize], _ctx: &mut ColumnPruningContext) -> PlanRef { + let mapping = ColIndexMapping::with_remaining_columns(required_cols, self.schema().len()); + LogicalProject::with_mapping(self.clone().into(), mapping).into() + } +} + +impl ExprRewritable for LogicalSourceBackfill { + fn has_rewritable_expr(&self) -> bool { + self.output_exprs.is_some() + } + + fn rewrite_exprs(&self, r: &mut dyn ExprRewriter) -> PlanRef { + let mut output_exprs = self.output_exprs.clone(); + + for expr in output_exprs.iter_mut().flatten() { + *expr = r.rewrite_expr(expr.clone()); + } + + Self { + output_exprs, + ..self.clone() + } + .into() + } +} + +impl ExprVisitable for LogicalSourceBackfill { + fn visit_exprs(&self, v: &mut dyn ExprVisitor) { + self.output_exprs + .iter() + .flatten() + .for_each(|e| v.visit_expr(e)); + } +} + +impl PredicatePushdown for LogicalSourceBackfill { + fn predicate_pushdown( + &self, + predicate: Condition, + _ctx: &mut PredicatePushdownContext, + ) -> PlanRef { + LogicalFilter::create(self.clone().into(), predicate) + } +} + +impl ToBatch for LogicalSourceBackfill { + fn to_batch(&self) -> Result { + let mut plan: PlanRef = BatchSource::new(self.core.clone()).into(); + + if let Some(exprs) = &self.output_exprs { + let logical_project = generic::Project::new(exprs.to_vec(), plan); + plan = BatchProject::new(logical_project).into(); + } + + Ok(plan) + } +} + +impl ToStream for LogicalSourceBackfill { + fn to_stream(&self, _ctx: &mut ToStreamContext) -> Result { + let mut plan = StreamSourceBackfill::new(self.core.clone()).into(); + + if let Some(exprs) = &self.output_exprs { + let logical_project = generic::Project::new(exprs.to_vec(), plan); + plan = StreamProject::new(logical_project).into(); + } + + let catalog = self.source_catalog(); + if !catalog.watermark_descs.is_empty() { + plan = StreamWatermarkFilter::new(plan, catalog.watermark_descs.clone()).into(); + } + + if let Some(row_id_index) = self.output_row_id_index { + plan = StreamRowIdGen::new_with_dist(plan, row_id_index, HashShard(vec![row_id_index])) + .into(); + } + Ok(plan) + } + + fn logical_rewrite_for_stream( + &self, + _ctx: &mut RewriteStreamContext, + ) -> Result<(PlanRef, ColIndexMapping)> { + Ok(( + self.clone().into(), + ColIndexMapping::identity(self.schema().len()), + )) + } +} diff --git a/src/frontend/src/optimizer/plan_node/mod.rs b/src/frontend/src/optimizer/plan_node/mod.rs index e4dfb0e8f2fe1..0b6e007b06b11 100644 --- a/src/frontend/src/optimizer/plan_node/mod.rs +++ b/src/frontend/src/optimizer/plan_node/mod.rs @@ -676,8 +676,8 @@ impl dyn PlanNode { impl dyn PlanNode { /// Serialize the plan node and its children to a stream plan proto. /// - /// Note that [`StreamTableScan`] has its own implementation of `to_stream_prost`. We have a - /// hook inside to do some ad-hoc thing for [`StreamTableScan`]. + /// Note that some operators has their own implementation of `to_stream_prost`. We have a + /// hook inside to do some ad-hoc things. pub fn to_stream_prost( &self, state: &mut BuildFragmentGraphState, @@ -690,6 +690,9 @@ impl dyn PlanNode { if let Some(stream_cdc_table_scan) = self.as_stream_cdc_table_scan() { return stream_cdc_table_scan.adhoc_to_stream_prost(state); } + if let Some(stream_source_backfill) = self.as_stream_source_backfill() { + return stream_source_backfill.adhoc_to_stream_prost(state); + } if let Some(stream_share) = self.as_stream_share() { return stream_share.adhoc_to_stream_prost(state); } @@ -824,6 +827,7 @@ mod logical_project_set; mod logical_scan; mod logical_share; mod logical_source; +mod logical_source_backfill; mod logical_sys_scan; mod logical_table_function; mod logical_topn; @@ -853,6 +857,7 @@ mod stream_simple_agg; mod stream_sink; mod stream_sort; mod stream_source; +mod stream_source_backfill; mod stream_stateless_simple_agg; mod stream_table_scan; mod stream_topn; @@ -915,6 +920,7 @@ pub use logical_project_set::LogicalProjectSet; pub use logical_scan::LogicalScan; pub use logical_share::LogicalShare; pub use logical_source::LogicalSource; +pub use logical_source_backfill::LogicalSourceBackfill; pub use logical_sys_scan::LogicalSysScan; pub use logical_table_function::LogicalTableFunction; pub use logical_topn::LogicalTopN; @@ -946,6 +952,7 @@ pub use stream_simple_agg::StreamSimpleAgg; pub use stream_sink::{IcebergPartitionInfo, PartitionComputeInfo, StreamSink}; pub use stream_sort::StreamEowcSort; pub use stream_source::StreamSource; +pub use stream_source_backfill::StreamSourceBackfill; pub use stream_stateless_simple_agg::StreamStatelessSimpleAgg; pub use stream_table_scan::StreamTableScan; pub use stream_temporal_join::StreamTemporalJoin; @@ -987,6 +994,7 @@ macro_rules! for_all_plan_nodes { , { Logical, CdcScan } , { Logical, SysScan } , { Logical, Source } + , { Logical, SourceBackfill } , { Logical, Insert } , { Logical, Delete } , { Logical, Update } @@ -1040,6 +1048,7 @@ macro_rules! for_all_plan_nodes { , { Stream, CdcTableScan } , { Stream, Sink } , { Stream, Source } + , { Stream, SourceBackfill } , { Stream, HashJoin } , { Stream, Exchange } , { Stream, HashAgg } @@ -1083,6 +1092,7 @@ macro_rules! for_logical_plan_nodes { , { Logical, CdcScan } , { Logical, SysScan } , { Logical, Source } + , { Logical, SourceBackfill } , { Logical, Insert } , { Logical, Delete } , { Logical, Update } @@ -1156,6 +1166,7 @@ macro_rules! for_stream_plan_nodes { , { Stream, CdcTableScan } , { Stream, Sink } , { Stream, Source } + , { Stream, SourceBackfill } , { Stream, HashAgg } , { Stream, SimpleAgg } , { Stream, StatelessSimpleAgg } diff --git a/src/frontend/src/optimizer/plan_node/stream_cdc_table_scan.rs b/src/frontend/src/optimizer/plan_node/stream_cdc_table_scan.rs index 8bfecb64b03d5..0c34d963869d9 100644 --- a/src/frontend/src/optimizer/plan_node/stream_cdc_table_scan.rs +++ b/src/frontend/src/optimizer/plan_node/stream_cdc_table_scan.rs @@ -132,6 +132,7 @@ impl StreamNode for StreamCdcTableScan { } impl StreamCdcTableScan { + /// plan: merge -> filter -> exchange(simple) -> `stream_scan` pub fn adhoc_to_stream_prost( &self, state: &mut BuildFragmentGraphState, @@ -241,10 +242,10 @@ impl StreamCdcTableScan { .collect_vec(); tracing::debug!( - "output_column_ids: {:?}, upstream_column_ids: {:?}, output_indices: {:?}", - self.core.output_column_ids(), - upstream_column_ids, - output_indices + output_column_ids=?self.core.output_column_ids(), + ?upstream_column_ids, + ?output_indices, + "stream cdc table scan output indices" ); let stream_scan_body = PbNodeBody::StreamCdcScan(StreamCdcScanNode { diff --git a/src/frontend/src/optimizer/plan_node/stream_source_backfill.rs b/src/frontend/src/optimizer/plan_node/stream_source_backfill.rs new file mode 100644 index 0000000000000..6cc93c64d8af2 --- /dev/null +++ b/src/frontend/src/optimizer/plan_node/stream_source_backfill.rs @@ -0,0 +1,188 @@ +// 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 std::rc::Rc; + +use fixedbitset::FixedBitSet; +use itertools::Itertools; +use pretty_xmlish::{Pretty, XmlNode}; +use risingwave_common::catalog::Field; +use risingwave_common::types::DataType; +use risingwave_common::util::sort_util::OrderType; +use risingwave_pb::stream_plan::stream_node::{NodeBody, PbNodeBody}; +use risingwave_pb::stream_plan::PbStreamNode; + +use super::stream::prelude::*; +use super::utils::TableCatalogBuilder; +use super::{PlanBase, PlanRef}; +use crate::catalog::source_catalog::SourceCatalog; +use crate::optimizer::plan_node::expr_visitable::ExprVisitable; +use crate::optimizer::plan_node::utils::{childless_record, Distill}; +use crate::optimizer::plan_node::{generic, ExprRewritable, StreamNode}; +use crate::optimizer::property::Distribution; +use crate::scheduler::SchedulerResult; +use crate::stream_fragmenter::BuildFragmentGraphState; +use crate::{Explain, TableCatalog}; + +#[derive(Debug, Clone, PartialEq, Eq, Hash)] +pub struct StreamSourceBackfill { + pub base: PlanBase, + core: generic::Source, +} + +impl_plan_tree_node_for_leaf! { StreamSourceBackfill } + +impl StreamSourceBackfill { + pub fn new(source: generic::Source) -> Self { + let base = PlanBase::new_stream_with_core( + &source, + Distribution::SomeShard, + source.catalog.as_ref().map_or(true, |s| s.append_only), + false, + FixedBitSet::with_capacity(source.column_catalog.len()), + ); + + Self { base, core: source } + } + + fn get_columns(&self) -> Vec<&str> { + self.core + .column_catalog + .iter() + .map(|column| column.name()) + .collect() + } + + pub fn source_catalog(&self) -> Rc { + self.core + .catalog + .clone() + .expect("source backfill should have source cataglog") + } + + pub fn infer_internal_table_catalog() -> TableCatalog { + // note that source's internal table is to store partition_id -> offset mapping and its + // schema is irrelevant to input schema + // On the premise of ensuring that the materialized_source data can be cleaned up, keep the + // state in source. + // Source state doesn't maintain retention_seconds, internal_table_subset function only + // returns retention_seconds so default is used here + let mut builder = TableCatalogBuilder::default(); + + let key = Field { + data_type: DataType::Varchar, + name: "partition_id".to_string(), + sub_fields: vec![], + type_name: "".to_string(), + }; + let value = Field { + data_type: DataType::Jsonb, + name: "backfill_progress".to_string(), + sub_fields: vec![], + type_name: "".to_string(), + }; + + let ordered_col_idx = builder.add_column(&key); + builder.add_column(&value); + builder.add_order_column(ordered_col_idx, OrderType::ascending()); + // read prefix hint is 0. We need to scan all data in the state table. + builder.build(vec![], 0) + } + + pub fn adhoc_to_stream_prost( + &self, + state: &mut BuildFragmentGraphState, + ) -> SchedulerResult { + use risingwave_pb::stream_plan::*; + + let stream_key = self + .stream_key() + .unwrap_or_else(|| { + panic!( + "should always have a stream key in the stream plan but not, sub plan: {}", + PlanRef::from(self.clone()).explain_to_string() + ) + }) + .iter() + .map(|x| *x as u32) + .collect_vec(); + + let source_catalog = self.source_catalog(); + let source_inner = SourceBackfillNode { + source_id: source_catalog.id, + source_name: source_catalog.name.clone(), + state_table: Some( + Self::infer_internal_table_catalog() + .with_id(state.gen_table_id_wrapped()) + .to_internal_table_prost(), + ), + info: Some(source_catalog.info.clone()), + // XXX: what's the usage of this? + row_id_index: self.core.row_id_index.map(|index| index as _), + columns: self + .core + .column_catalog + .iter() + .map(|c| c.to_protobuf()) + .collect_vec(), + with_properties: source_catalog.with_properties.clone().into_iter().collect(), + // rate_limit: self.base.ctx().overwrite_options().streaming_rate_limit, + }; + + let stream_scan_body = PbNodeBody::SourceBackfill(source_inner); + + let fields = self.schema().to_prost(); + // plan: merge -> backfill + Ok(PbStreamNode { + fields: fields.clone(), + input: vec![ + // The merge node body will be filled by the `ActorBuilder` on the meta service. + PbStreamNode { + node_body: Some(PbNodeBody::Merge(Default::default())), + identity: "Upstream".into(), + fields, + stream_key: vec![], // not used + ..Default::default() + }, + ], + node_body: Some(stream_scan_body), + stream_key, + operator_id: self.base.id().0 as u64, + identity: self.distill_to_string(), + append_only: self.append_only(), + }) + } +} + +impl Distill for StreamSourceBackfill { + fn distill<'a>(&self) -> XmlNode<'a> { + let columns = self + .get_columns() + .iter() + .map(|ele| Pretty::from(ele.to_string())) + .collect(); + let col = Pretty::Array(columns); + childless_record("StreamSourceBackfill", vec![("columns", col)]) + } +} + +impl ExprRewritable for StreamSourceBackfill {} + +impl ExprVisitable for StreamSourceBackfill {} + +impl StreamNode for StreamSourceBackfill { + fn to_stream_prost_body(&self, _state: &mut BuildFragmentGraphState) -> NodeBody { + unreachable!("stream source backfill cannot be converted into a prost body -- call `adhoc_to_stream_prost` instead.") + } +} diff --git a/src/frontend/src/planner/relation.rs b/src/frontend/src/planner/relation.rs index 3f64a8fde4405..a80973cf6e7f1 100644 --- a/src/frontend/src/planner/relation.rs +++ b/src/frontend/src/planner/relation.rs @@ -28,7 +28,8 @@ use crate::expr::{Expr, ExprImpl, ExprType, FunctionCall, InputRef}; use crate::optimizer::plan_node::generic::SourceNodeKind; use crate::optimizer::plan_node::{ LogicalApply, LogicalHopWindow, LogicalJoin, LogicalProject, LogicalScan, LogicalShare, - LogicalSource, LogicalSysScan, LogicalTableFunction, LogicalValues, PlanRef, + LogicalSource, LogicalSourceBackfill, LogicalSysScan, LogicalTableFunction, LogicalValues, + PlanRef, }; use crate::optimizer::property::Cardinality; use crate::planner::Planner; @@ -86,12 +87,16 @@ impl Planner { } pub(super) fn plan_source(&mut self, source: BoundSource) -> Result { - Ok(LogicalSource::with_catalog( - Rc::new(source.catalog), - SourceNodeKind::CreateMViewOrBatch, - self.ctx(), - )? - .into()) + if source.can_backfill() { + Ok(LogicalSourceBackfill::new(Rc::new(source.catalog), self.ctx())?.into()) + } else { + Ok(LogicalSource::with_catalog( + Rc::new(source.catalog), + SourceNodeKind::CreateMViewOrBatch, + self.ctx(), + )? + .into()) + } } pub(super) fn plan_join(&mut self, join: BoundJoin) -> Result { diff --git a/src/frontend/src/stream_fragmenter/mod.rs b/src/frontend/src/stream_fragmenter/mod.rs index a3d18a2c6dc17..87cb9a6703f18 100644 --- a/src/frontend/src/stream_fragmenter/mod.rs +++ b/src/frontend/src/stream_fragmenter/mod.rs @@ -264,9 +264,9 @@ fn build_fragment( if let Some(source) = node.source_inner.as_ref() && let Some(source_info) = source.info.as_ref() - && source_info.cdc_source_job + && source_info.has_streaming_job + && !source_info.is_distributed { - tracing::debug!("mark cdc source job as singleton"); current_fragment.requires_singleton = true; } } @@ -294,6 +294,7 @@ fn build_fragment( } NodeBody::StreamCdcScan(_) => { + // XXX: Should we use a different flag for CDC scan? current_fragment.fragment_type_mask |= FragmentTypeFlag::StreamScan as u32; // the backfill algorithm is not parallel safe current_fragment.requires_singleton = true; @@ -309,6 +310,13 @@ fn build_fragment( .upstream_table_ids .push(node.upstream_source_id); } + NodeBody::SourceBackfill(node) => { + current_fragment.fragment_type_mask |= FragmentTypeFlag::SourceBackfill as u32; + // memorize upstream source id for later use + let source_id = node.source_id; + state.dependent_table_ids.insert(source_id.into()); + current_fragment.upstream_table_ids.push(source_id); + } NodeBody::Now(_) => { // TODO: Remove this and insert a `BarrierRecv` instead. diff --git a/src/meta/src/barrier/command.rs b/src/meta/src/barrier/command.rs index 07765fe840c38..c2ad921244bcb 100644 --- a/src/meta/src/barrier/command.rs +++ b/src/meta/src/barrier/command.rs @@ -31,9 +31,9 @@ use risingwave_pb::stream_plan::barrier_mutation::Mutation; use risingwave_pb::stream_plan::throttle_mutation::RateLimit; use risingwave_pb::stream_plan::update_mutation::*; use risingwave_pb::stream_plan::{ - AddMutation, BarrierMutation, CombinedMutation, Dispatcher, Dispatchers, FragmentTypeFlag, - PauseMutation, ResumeMutation, SourceChangeSplitMutation, StopMutation, StreamActor, - ThrottleMutation, UpdateMutation, + AddMutation, BarrierMutation, CombinedMutation, Dispatcher, Dispatchers, PauseMutation, + ResumeMutation, SourceChangeSplitMutation, StopMutation, StreamActor, ThrottleMutation, + UpdateMutation, }; use risingwave_pb::stream_service::WaitEpochCommitRequest; use thiserror_ext::AsReport; @@ -687,24 +687,11 @@ impl CommandContext { pub fn actors_to_track(&self) -> HashSet { match &self.command { Command::CreateStreamingJob { - dispatchers, - table_fragments, - .. - } => { - // cdc backfill table job doesn't need to be tracked - if table_fragments.fragments().iter().any(|fragment| { - fragment.fragment_type_mask & FragmentTypeFlag::CdcFilter as u32 != 0 - }) { - Default::default() - } else { - dispatchers - .values() - .flatten() - .flat_map(|dispatcher| dispatcher.downstream_actor_id.iter().copied()) - .chain(table_fragments.values_actor_ids()) - .collect() - } - } + table_fragments, .. + } => table_fragments + .tracking_progress_actor_ids() + .into_iter() + .collect(), _ => Default::default(), } } @@ -794,7 +781,7 @@ impl CommandContext { .await?; self.barrier_manager_context .source_manager - .apply_source_change(None, Some(split_assignment.clone()), None) + .apply_source_change(None, None, Some(split_assignment.clone()), None) .await; } @@ -954,11 +941,12 @@ impl CommandContext { // Extract the fragments that include source operators. let source_fragments = table_fragments.stream_source_fragments(); - + let backfill_fragments = table_fragments.source_backfill_fragments()?; self.barrier_manager_context .source_manager .apply_source_change( Some(source_fragments), + Some(backfill_fragments), Some(init_split_assignment.clone()), None, ) @@ -1021,10 +1009,13 @@ impl CommandContext { .drop_source_fragments(std::slice::from_ref(old_table_fragments)) .await; let source_fragments = new_table_fragments.stream_source_fragments(); + // XXX: is it possbile to have backfill fragments here? + let backfill_fragments = new_table_fragments.source_backfill_fragments()?; self.barrier_manager_context .source_manager .apply_source_change( Some(source_fragments), + Some(backfill_fragments), Some(init_split_assignment.clone()), None, ) diff --git a/src/meta/src/barrier/mod.rs b/src/meta/src/barrier/mod.rs index 47bef49c66574..963245feb9597 100644 --- a/src/meta/src/barrier/mod.rs +++ b/src/meta/src/barrier/mod.rs @@ -870,6 +870,7 @@ impl GlobalBarrierManager { } commands }; + tracing::trace!("finished_commands: {}", finished_commands.len()); for command in finished_commands { self.checkpoint_control.stash_command_to_finish(command); diff --git a/src/meta/src/barrier/progress.rs b/src/meta/src/barrier/progress.rs index f22c5a2bbb216..2ea9a99aa848e 100644 --- a/src/meta/src/barrier/progress.rs +++ b/src/meta/src/barrier/progress.rs @@ -206,6 +206,11 @@ impl TrackingJob { pub(crate) fn notify_finished(self) { match self { TrackingJob::New(command) => { + tracing::trace!( + "notify finished, command: {:?}, curr_epoch: {:?}", + command.context.command, + command.context.curr_epoch + ); command .notifiers .into_iter() @@ -368,6 +373,7 @@ impl CreateMviewProgressTracker { version_stats: &HummockVersionStats, ) -> Option { let actors = command.context.actors_to_track(); + tracing::trace!("add actors to track: {:?}", actors); if actors.is_empty() { // The command can be finished immediately. return Some(TrackingJob::New(command)); @@ -426,6 +432,7 @@ impl CreateMviewProgressTracker { upstream_total_key_count, definition, ); + tracing::trace!("add progress: {:?}", progress); if *ddl_type == DdlType::Sink { // We return the original tracking job immediately. // This is because sink can be decoupled with backfill progress. @@ -450,6 +457,7 @@ impl CreateMviewProgressTracker { progress: &CreateMviewProgress, version_stats: &HummockVersionStats, ) -> Option { + tracing::trace!("update progress: {:?}", progress); let actor = progress.backfill_actor_id; let Some(table_id) = self.actor_map.get(&actor).copied() else { // On restart, backfill will ALWAYS notify CreateMviewProgressTracker, diff --git a/src/meta/src/barrier/schedule.rs b/src/meta/src/barrier/schedule.rs index c481b756b1828..0e616b305dabc 100644 --- a/src/meta/src/barrier/schedule.rs +++ b/src/meta/src/barrier/schedule.rs @@ -276,9 +276,11 @@ impl BarrierScheduler { let mut infos = Vec::with_capacity(contexts.len()); for (injected_rx, collect_rx, finish_rx) in contexts { + tracing::trace!("waiting for command to be injected"); // Wait for this command to be injected, and record the result. let info = injected_rx.await.ok().context("failed to inject barrier")?; infos.push(info); + tracing::trace!("injected_rx finished"); // Throw the error if it occurs when collecting this barrier. collect_rx diff --git a/src/meta/src/controller/catalog.rs b/src/meta/src/controller/catalog.rs index 6077efa7f88c1..44c27c414cb77 100644 --- a/src/meta/src/controller/catalog.rs +++ b/src/meta/src/controller/catalog.rs @@ -1501,7 +1501,7 @@ impl CatalogController { .map(|obj| obj.oid) .collect_vec(); - // cdc source streaming job. + // source streaming job. if object_type == ObjectType::Source { let source_info: Option = Source::find_by_id(object_id) .select_only() @@ -1511,7 +1511,7 @@ impl CatalogController { .await? .ok_or_else(|| MetaError::catalog_id_not_found("source", object_id))?; if let Some(source_info) = source_info - && source_info.into_inner().cdc_source_job + && source_info.into_inner().has_streaming_job { to_drop_streaming_jobs.push(object_id); } diff --git a/src/meta/src/controller/fragment.rs b/src/meta/src/controller/fragment.rs index 833e642a83e74..9f00432c18e76 100644 --- a/src/meta/src/controller/fragment.rs +++ b/src/meta/src/controller/fragment.rs @@ -51,8 +51,7 @@ use sea_orm::{ use crate::controller::catalog::{CatalogController, CatalogControllerInner}; use crate::controller::utils::{ - find_stream_source, get_actor_dispatchers, FragmentDesc, PartialActorLocation, - PartialFragmentStateTables, + get_actor_dispatchers, FragmentDesc, PartialActorLocation, PartialFragmentStateTables, }; use crate::manager::{ActorInfos, LocalNotification}; use crate::model::TableParallelism; @@ -1187,9 +1186,9 @@ impl CatalogController { let mut source_fragment_ids = HashMap::new(); for (fragment_id, _, stream_node) in fragments { - if let Some(source) = find_stream_source(&stream_node.to_protobuf()) { + if let Some(source_id) = stream_node.to_protobuf().find_stream_source() { source_fragment_ids - .entry(source.source_id as SourceId) + .entry(source_id as SourceId) .or_insert_with(BTreeSet::new) .insert(fragment_id); } @@ -1197,31 +1196,33 @@ impl CatalogController { Ok(source_fragment_ids) } - pub async fn get_stream_source_fragment_ids( + pub async fn load_backfill_fragment_ids( &self, - job_id: ObjectId, - ) -> MetaResult>> { + ) -> MetaResult>> { let inner = self.inner.read().await; - let mut fragments: Vec<(FragmentId, i32, StreamNode)> = Fragment::find() + let mut fragments: Vec<(FragmentId, Vec, i32, StreamNode)> = Fragment::find() .select_only() .columns([ fragment::Column::FragmentId, + fragment::Column::UpstreamFragmentId, fragment::Column::FragmentTypeMask, fragment::Column::StreamNode, ]) - .filter(fragment::Column::JobId.eq(job_id)) .into_tuple() .all(&inner.db) .await?; - fragments.retain(|(_, mask, _)| *mask & PbFragmentTypeFlag::Source as i32 != 0); + fragments.retain(|(_, _, mask, _)| *mask & PbFragmentTypeFlag::SourceBackfill as i32 != 0); let mut source_fragment_ids = HashMap::new(); - for (fragment_id, _, stream_node) in fragments { - if let Some(source) = find_stream_source(&stream_node.to_protobuf()) { + for (fragment_id, upstream_fragment_id, _, stream_node) in fragments { + if let Some(source_id) = stream_node.to_protobuf().find_source_backfill() { + if upstream_fragment_id.len() != 1 { + bail!("SourceBackfill should have only one upstream fragment, found {} for fragment {}", upstream_fragment_id.len(), fragment_id); + } source_fragment_ids - .entry(source.source_id as SourceId) + .entry(source_id as SourceId) .or_insert_with(BTreeSet::new) - .insert(fragment_id); + .insert((fragment_id, upstream_fragment_id[0])); } } Ok(source_fragment_ids) diff --git a/src/meta/src/controller/streaming_job.rs b/src/meta/src/controller/streaming_job.rs index 9bb8af6172469..23ee87a4e2f45 100644 --- a/src/meta/src/controller/streaming_job.rs +++ b/src/meta/src/controller/streaming_job.rs @@ -784,7 +784,7 @@ impl CatalogController { if let Some(table_id) = source.optional_associated_table_id { vec![table_id] } else if let Some(source_info) = &source.source_info - && source_info.inner_ref().cdc_source_job + && source_info.inner_ref().has_streaming_job { vec![source_id] } else { @@ -819,6 +819,7 @@ impl CatalogController { .map(|(id, mask, stream_node)| (id, mask, stream_node.to_protobuf())) .collect_vec(); + // TODO: limit source backfill? fragments.retain_mut(|(_, fragment_type_mask, stream_node)| { let mut found = false; if *fragment_type_mask & PbFragmentTypeFlag::Source as i32 != 0 { diff --git a/src/meta/src/manager/catalog/database.rs b/src/meta/src/manager/catalog/database.rs index cd2f0bbb13b94..3d8c0a1674632 100644 --- a/src/meta/src/manager/catalog/database.rs +++ b/src/meta/src/manager/catalog/database.rs @@ -361,11 +361,13 @@ impl DatabaseManager { .chain(self.indexes.keys().copied()) .chain(self.sources.keys().copied()) .chain( - // filter cdc source jobs self.sources .iter() .filter(|(_, source)| { - source.info.as_ref().is_some_and(|info| info.cdc_source_job) + source + .info + .as_ref() + .is_some_and(|info| info.has_streaming_job) }) .map(|(id, _)| id) .copied(), diff --git a/src/meta/src/manager/catalog/fragment.rs b/src/meta/src/manager/catalog/fragment.rs index fdae16efe5a7b..78370fa754858 100644 --- a/src/meta/src/manager/catalog/fragment.rs +++ b/src/meta/src/manager/catalog/fragment.rs @@ -1006,6 +1006,29 @@ impl FragmentManager { bail!("fragment not found: {}", fragment_id) } + pub async fn get_running_actors_and_upstream_fragment_of_fragment( + &self, + fragment_id: FragmentId, + ) -> MetaResult)>> { + let map = &self.core.read().await.table_fragments; + + for table_fragment in map.values() { + if let Some(fragment) = table_fragment.fragments.get(&fragment_id) { + let running_actors = fragment + .actors + .iter() + .filter(|a| { + table_fragment.actor_status[&a.actor_id].state == ActorState::Running as i32 + }) + .map(|a| (a.actor_id, a.upstream_actor_id.clone())) + .collect(); + return Ok(running_actors); + } + } + + bail!("fragment not found: {}", fragment_id) + } + /// Add the newly added Actor to the `FragmentManager` pub async fn pre_apply_reschedules( &self, diff --git a/src/meta/src/manager/catalog/mod.rs b/src/meta/src/manager/catalog/mod.rs index 33990548f01b8..f0ee31788acc4 100644 --- a/src/meta/src/manager/catalog/mod.rs +++ b/src/meta/src/manager/catalog/mod.rs @@ -1145,7 +1145,7 @@ impl CatalogManager { let mut all_sink_ids: HashSet = HashSet::default(); let mut all_source_ids: HashSet = HashSet::default(); let mut all_view_ids: HashSet = HashSet::default(); - let mut all_cdc_source_ids: HashSet = HashSet::default(); + let mut all_streaming_job_source_ids: HashSet = HashSet::default(); let relations_depend_on = |relation_id: RelationId| -> Vec { let tables_depend_on = tables @@ -1408,11 +1408,10 @@ impl CatalogManager { continue; } - // cdc source streaming job if let Some(info) = source.info - && info.cdc_source_job + && info.has_streaming_job { - all_cdc_source_ids.insert(source.id); + all_streaming_job_source_ids.insert(source.id); let source_table_fragments = fragment_manager .select_table_fragments_by_table_id(&source.id.into()) .await?; @@ -1669,7 +1668,7 @@ impl CatalogManager { .into_iter() .map(|id| id.into()) .chain(all_sink_ids.into_iter().map(|id| id.into())) - .chain(all_cdc_source_ids.into_iter().map(|id| id.into())) + .chain(all_streaming_job_source_ids.into_iter().map(|id| id.into())) .collect_vec(); Ok((version, catalog_deleted_ids)) diff --git a/src/meta/src/manager/metadata.rs b/src/meta/src/manager/metadata.rs index eef1c0c101256..291a3de6e0114 100644 --- a/src/meta/src/manager/metadata.rs +++ b/src/meta/src/manager/metadata.rs @@ -590,6 +590,22 @@ impl MetadataManager { } } + pub async fn get_running_actors_and_upstream_actors_of_fragment( + &self, + id: FragmentId, + ) -> MetaResult)>> { + match self { + MetadataManager::V1(mgr) => { + mgr.fragment_manager + .get_running_actors_and_upstream_fragment_of_fragment(id) + .await + } + MetadataManager::V2(_mgr) => { + todo!() + } + } + } + pub async fn get_job_fragments_by_ids( &self, ids: &[TableId], diff --git a/src/meta/src/model/stream.rs b/src/meta/src/model/stream.rs index ef55f78493f85..da1a310ee8d86 100644 --- a/src/meta/src/model/stream.rs +++ b/src/meta/src/model/stream.rs @@ -30,7 +30,7 @@ use risingwave_pb::meta::{PbTableFragments, PbTableParallelism}; use risingwave_pb::plan_common::PbExprContext; use risingwave_pb::stream_plan::stream_node::NodeBody; use risingwave_pb::stream_plan::{ - FragmentTypeFlag, PbFragmentTypeFlag, PbStreamContext, StreamActor, StreamNode, StreamSource, + FragmentTypeFlag, PbFragmentTypeFlag, PbStreamContext, StreamActor, StreamNode, }; use super::{ActorId, FragmentId}; @@ -337,7 +337,7 @@ impl TableFragments { } /// Returns the actor ids with the given fragment type. - fn filter_actor_ids(&self, check_type: impl Fn(u32) -> bool) -> Vec { + pub fn filter_actor_ids(&self, check_type: impl Fn(u32) -> bool) -> Vec { self.fragments .values() .filter(|fragment| check_type(fragment.get_fragment_type_mask())) @@ -367,10 +367,12 @@ impl TableFragments { }) } - /// Returns values actor ids. - pub fn values_actor_ids(&self) -> Vec { + /// Returns actor ids that need to be tracked when creating MV. + pub fn tracking_progress_actor_ids(&self) -> Vec { Self::filter_actor_ids(self, |fragment_type_mask| { - (fragment_type_mask & FragmentTypeFlag::Values as u32) != 0 + (fragment_type_mask + & (FragmentTypeFlag::Values as u32 | FragmentTypeFlag::StreamScan as u32)) + != 0 }) } @@ -411,23 +413,6 @@ impl TableFragments { .collect() } - /// Find the external stream source info inside the stream node, if any. - pub fn find_stream_source(stream_node: &StreamNode) -> Option<&StreamSource> { - if let Some(NodeBody::Source(source)) = stream_node.node_body.as_ref() { - if let Some(inner) = &source.source_inner { - return Some(inner); - } - } - - for child in &stream_node.input { - if let Some(source) = Self::find_stream_source(child) { - return Some(source); - } - } - - None - } - /// Extract the fragments that include source executors that contains an external stream source, /// grouping by source id. pub fn stream_source_fragments(&self) -> HashMap> { @@ -435,10 +420,7 @@ impl TableFragments { for fragment in self.fragments() { for actor in &fragment.actors { - if let Some(source_id) = - TableFragments::find_stream_source(actor.nodes.as_ref().unwrap()) - .map(|s| s.source_id) - { + if let Some(source_id) = actor.nodes.as_ref().unwrap().find_stream_source() { source_fragments .entry(source_id) .or_insert(BTreeSet::new()) @@ -451,6 +433,29 @@ impl TableFragments { source_fragments } + pub fn source_backfill_fragments( + &self, + ) -> MetadataModelResult>> { + let mut source_fragments = HashMap::new(); + + for fragment in self.fragments() { + for actor in &fragment.actors { + if let Some(source_id) = actor.nodes.as_ref().unwrap().find_source_backfill() { + if fragment.upstream_fragment_ids.len() != 1 { + return Err(anyhow::anyhow!("SourceBackfill should have only one upstream fragment, found {:?} for fragment {}", fragment.upstream_fragment_ids, fragment.fragment_id).into()); + } + source_fragments + .entry(source_id) + .or_insert(BTreeSet::new()) + .insert((fragment.fragment_id, fragment.upstream_fragment_ids[0])); + + break; + } + } + } + Ok(source_fragments) + } + /// Resolve dependent table fn resolve_dependent_table(stream_node: &StreamNode, table_ids: &mut HashMap) { let table_id = match stream_node.node_body.as_ref() { diff --git a/src/meta/src/rpc/ddl_controller.rs b/src/meta/src/rpc/ddl_controller.rs index 7cff9dc4a9b7a..35ea9a30cbdf0 100644 --- a/src/meta/src/rpc/ddl_controller.rs +++ b/src/meta/src/rpc/ddl_controller.rs @@ -462,6 +462,7 @@ impl DdlController { .await; }; // 1. Drop source in catalog. + // If the source has a streaming job, it's also dropped here. let (version, streaming_job_ids) = mgr .catalog_manager .drop_relation( @@ -1337,8 +1338,12 @@ impl DdlController { .get_upstream_root_fragments(fragment_graph.dependent_table_ids()) .await?; - let upstream_actors: HashMap<_, _> = upstream_root_fragments + // XXX: do we need to filter here? + let upstream_mview_actors: HashMap<_, _> = upstream_root_fragments .iter() + // .filter(|(_, fragment)| { + // fragment.fragment_type_mask & FragmentTypeFlag::Mview as u32 != 0 + // }) .map(|(&table_id, fragment)| { ( table_id, @@ -1435,7 +1440,7 @@ impl DdlController { let ctx = CreateStreamingJobContext { dispatchers, - upstream_mview_actors: upstream_actors, + upstream_mview_actors, internal_tables, building_locations, existing_locations, diff --git a/src/meta/src/stream/scale.rs b/src/meta/src/stream/scale.rs index 7f40f8e3da033..f51fab1e7cd20 100644 --- a/src/meta/src/stream/scale.rs +++ b/src/meta/src/stream/scale.rs @@ -609,7 +609,7 @@ impl ScaleController { if (fragment.get_fragment_type_mask() & FragmentTypeFlag::Source as u32) != 0 { let stream_node = fragment.actors.first().unwrap().get_nodes().unwrap(); - if TableFragments::find_stream_source(stream_node).is_some() { + if stream_node.find_stream_source().is_some() { stream_source_fragment_ids.insert(*fragment_id); } } @@ -1613,6 +1613,7 @@ impl ScaleController { if !stream_source_actor_splits.is_empty() { self.source_manager .apply_source_change( + None, None, Some(stream_source_actor_splits), Some(stream_source_dropped_actors), diff --git a/src/meta/src/stream/source_manager.rs b/src/meta/src/stream/source_manager.rs index 99d6602cb4e85..5d84585b73ad2 100644 --- a/src/meta/src/stream/source_manager.rs +++ b/src/meta/src/stream/source_manager.rs @@ -30,6 +30,7 @@ use risingwave_connector::source::{ }; use risingwave_pb::catalog::Source; use risingwave_pb::source::{ConnectorSplit, ConnectorSplits}; +use risingwave_pb::stream_plan::Dispatcher; use risingwave_rpc_client::ConnectorClient; use thiserror_ext::AsReport; use tokio::sync::mpsc::{UnboundedReceiver, UnboundedSender}; @@ -229,8 +230,8 @@ pub struct SourceManagerCore { managed_sources: HashMap, /// Fragments associated with each source source_fragments: HashMap>, - /// Revert index for source_fragments - fragment_sources: HashMap, + /// `source_id` -> `(fragment_id, upstream_fragment_id)` + backfill_fragments: HashMap>, /// Splits assigned per actor actor_splits: HashMap>, @@ -241,20 +242,14 @@ impl SourceManagerCore { metadata_manager: MetadataManager, managed_sources: HashMap, source_fragments: HashMap>, + backfill_fragments: HashMap>, actor_splits: HashMap>, ) -> Self { - let mut fragment_sources = HashMap::new(); - for (source_id, fragment_ids) in &source_fragments { - for fragment_id in fragment_ids { - fragment_sources.insert(*fragment_id, *source_id); - } - } - Self { metadata_manager, managed_sources, source_fragments, - fragment_sources, + backfill_fragments, actor_splits, } } @@ -268,12 +263,13 @@ impl SourceManagerCore { let mut split_assignment: SplitAssignment = HashMap::new(); for (source_id, handle) in &self.managed_sources { - let fragment_ids = match self.source_fragments.get(source_id) { + let source_fragment_ids = match self.source_fragments.get(source_id) { Some(fragment_ids) if !fragment_ids.is_empty() => fragment_ids, _ => { continue; } }; + let backfill_fragment_ids = self.backfill_fragments.get(source_id); let Some(discovered_splits) = handle.discovered_splits().await else { return Ok(split_assignment); @@ -282,20 +278,26 @@ impl SourceManagerCore { tracing::warn!("No splits discovered for source {}", source_id); } - for fragment_id in fragment_ids { - let actor_ids = match self + for &fragment_id in source_fragment_ids { + let actors = match self .metadata_manager - .get_running_actors_of_fragment(*fragment_id) + .get_running_actors_of_fragment(fragment_id) .await { - Ok(actor_ids) => actor_ids, + Ok(actors) => { + if actors.is_empty() { + tracing::warn!("No actors found for fragment {}", fragment_id); + continue; + } + actors + } Err(err) => { tracing::warn!(error = %err.as_report(), "Failed to get the actor of the fragment, maybe the fragment doesn't exist anymore"); continue; } }; - let prev_actor_splits: HashMap<_, _> = actor_ids + let prev_actor_splits: HashMap<_, _> = actors .into_iter() .map(|actor_id| { ( @@ -309,14 +311,51 @@ impl SourceManagerCore { .collect(); if let Some(new_assignment) = reassign_splits( - *fragment_id, + fragment_id, prev_actor_splits, &discovered_splits, SplitDiffOptions { enable_scale_in: handle.enable_scale_in, }, ) { - split_assignment.insert(*fragment_id, new_assignment); + split_assignment.insert(fragment_id, new_assignment); + } + } + + if let Some(backfill_fragment_ids) = backfill_fragment_ids { + // align splits for backfill fragments with its upstream source fragment + for (fragment_id, upstream_fragment_id) in backfill_fragment_ids { + let Some(upstream_assignment) = split_assignment.get(upstream_fragment_id) + else { + // upstream fragment unchanged, do not update backfill fragment too + continue; + }; + let actors = match self + .metadata_manager + .get_running_actors_and_upstream_actors_of_fragment(*fragment_id) + .await + { + Ok(actors) => { + if actors.is_empty() { + tracing::warn!("No actors found for fragment {}", fragment_id); + continue; + } + actors + } + Err(err) => { + tracing::warn!("Failed to get the actor of the fragment {}, maybe the fragment doesn't exist anymore", err.to_string()); + continue; + } + }; + split_assignment.insert( + *fragment_id, + align_backfill_splits( + actors, + upstream_assignment, + *fragment_id, + *upstream_fragment_id, + )?, + ); } } } @@ -326,26 +365,32 @@ impl SourceManagerCore { fn apply_source_change( &mut self, - source_fragments: Option>>, + added_source_fragments: Option>>, + added_backfill_fragments: Option>>, split_assignment: Option, dropped_actors: Option>, ) { - if let Some(source_fragments) = source_fragments { + if let Some(source_fragments) = added_source_fragments { for (source_id, mut fragment_ids) in source_fragments { - for fragment_id in &fragment_ids { - self.fragment_sources.insert(*fragment_id, source_id); - } - self.source_fragments .entry(source_id) .or_default() .append(&mut fragment_ids); } } + if let Some(backfill_fragments) = added_backfill_fragments { + for (source_id, mut fragment_ids) in backfill_fragments { + self.backfill_fragments + .entry(source_id) + .or_default() + .append(&mut fragment_ids); + } + } if let Some(assignment) = split_assignment { for (_, actor_splits) in assignment { for (actor_id, splits) in actor_splits { + // override previous splits info self.actor_splits.insert(actor_id, splits); } } @@ -374,10 +419,6 @@ impl SourceManagerCore { entry.remove(); } } - - for fragment_id in &fragment_ids { - self.fragment_sources.remove(fragment_id); - } } for actor_id in removed_actors { @@ -433,6 +474,10 @@ impl Default for SplitDiffOptions { /// /// The existing splits will remain unmoved in their currently assigned actor. /// +/// If an actor has an upstream actor, it should be a backfill executor, +/// and its splits should be aligned with the upstream actor. `reassign_splits` should not be used in this case. +/// Use `align_backfill_splits` instead. +/// /// - `fragment_id`: just for logging /// /// ## Different connectors' behavior of split change @@ -545,6 +590,32 @@ where ) } +fn align_backfill_splits( + backfill_actors: impl IntoIterator)>, + upstream_assignment: &HashMap>, + fragment_id: FragmentId, + upstream_fragment_id: FragmentId, +) -> anyhow::Result>> { + backfill_actors + .into_iter() + .map(|(actor_id, upstream_actor_id)| { + let err = || anyhow::anyhow!("source backfill actor should have upstream actor, fragment_id: {fragment_id}, upstream_fragment_id: {upstream_fragment_id}, actor_id: {actor_id}, upstream_assignment: {upstream_assignment:?}, upstream_actor_id: {upstream_actor_id:?}"); + if upstream_actor_id.len() != 1 { + return Err(err()); + } + let Some(splits ) = upstream_assignment + .get(&upstream_actor_id[0]) + else { + return Err(err()); + }; + Ok(( + actor_id, + splits.clone(), + )) + }) + .collect() +} + impl SourceManager { const DEFAULT_SOURCE_TICK_INTERVAL: Duration = Duration::from_secs(10); const DEFAULT_SOURCE_TICK_TIMEOUT: Duration = Duration::from_secs(10); @@ -570,6 +641,7 @@ impl SourceManager { let mut actor_splits = HashMap::new(); let mut source_fragments = HashMap::new(); + let mut backfill_fragments = HashMap::new(); match &metadata_manager { MetadataManager::V1(mgr) => { @@ -581,6 +653,7 @@ impl SourceManager { .values() { source_fragments.extend(table_fragments.stream_source_fragments()); + backfill_fragments.extend(table_fragments.source_backfill_fragments()?); actor_splits.extend(table_fragments.actor_splits.clone()); } } @@ -597,6 +670,21 @@ impl SourceManager { ) }) .collect(); + backfill_fragments = mgr + .catalog_controller + .load_backfill_fragment_ids() + .await? + .into_iter() + .map(|(source_id, fragment_ids)| { + ( + source_id as SourceId, + fragment_ids + .into_iter() + .map(|(id, up_id)| (id as _, up_id as _)) + .collect(), + ) + }) + .collect(); actor_splits = mgr .catalog_controller .load_actor_splits() @@ -621,6 +709,7 @@ impl SourceManager { metadata_manager, managed_sources, source_fragments, + backfill_fragments, actor_splits, )); @@ -670,12 +759,18 @@ impl SourceManager { /// Updates states after split change (`post_collect` barrier) or scaling (`post_apply_reschedule`). pub async fn apply_source_change( &self, - source_fragments: Option>>, + added_source_fragments: Option>>, + added_backfill_fragments: Option>>, split_assignment: Option, dropped_actors: Option>, ) { let mut core = self.core.lock().await; - core.apply_source_change(source_fragments, split_assignment, dropped_actors); + core.apply_source_change( + added_source_fragments, + added_backfill_fragments, + split_assignment, + dropped_actors, + ); } /// Migrates splits from previous actors to the new actors for a rescheduled fragment. @@ -777,6 +872,68 @@ impl SourceManager { Ok(assigned) } + pub async fn allocate_splits_for_backfill( + &self, + table_id: &TableId, + dispatchers: &HashMap>, + ) -> MetaResult { + let core = self.core.lock().await; + let table_fragments = core + .metadata_manager + .get_job_fragments_by_id(table_id) + .await?; + + let upstream_assignment = &core.actor_splits; + let source_backfill_fragments = table_fragments.source_backfill_fragments()?; + tracing::debug!( + ?source_backfill_fragments, + ?table_fragments, + "allocate_splits_for_backfill source backfill fragments" + ); + + let mut assigned = HashMap::new(); + + for (_source_id, fragments) in source_backfill_fragments { + for (fragment_id, upstream_fragment_id) in fragments { + let upstream_actors = core + .metadata_manager + .get_running_actors_of_fragment(upstream_fragment_id) + .await?; + let mut backfill_actors = vec![]; + for upstream_actor in upstream_actors { + if let Some(dispatchers) = dispatchers.get(&upstream_actor) { + let err = || { + anyhow::anyhow!( + "source backfill fragment's upstream fragment should have one dispatcher, fragment_id: {fragment_id}, upstream_fragment_id: {upstream_fragment_id}, upstream_actor: {upstream_actor}, dispatchers: {dispatchers:?}", + fragment_id = fragment_id, + upstream_fragment_id = upstream_fragment_id, + upstream_actor = upstream_actor, + dispatchers = dispatchers + ) + }; + if dispatchers.len() != 1 || dispatchers[0].downstream_actor_id.len() != 1 { + return Err(err().into()); + } + + backfill_actors + .push((dispatchers[0].downstream_actor_id[0], vec![upstream_actor])); + } + } + assigned.insert( + fragment_id, + align_backfill_splits( + backfill_actors, + upstream_assignment, + fragment_id, + upstream_fragment_id, + )?, + ); + } + } + + Ok(assigned) + } + /// register connector worker for source. pub async fn register_source(&self, source: &Source) -> anyhow::Result<()> { let mut core = self.core.lock().await; diff --git a/src/meta/src/stream/stream_graph/actor.rs b/src/meta/src/stream/stream_graph/actor.rs index c42c2f5a51425..52a14af0629c6 100644 --- a/src/meta/src/stream/stream_graph/actor.rs +++ b/src/meta/src/stream/stream_graph/actor.rs @@ -122,7 +122,7 @@ impl ActorBuilder { /// During this process, the following things will be done: /// 1. Replace the logical `Exchange` in node's input with `Merge`, which can be executed on the /// compute nodes. - /// 2. Fill the upstream mview info of the `Merge` node under the `StreamScan` node. + /// 2. Fill the upstream mview info of the `Merge` node under the other "leaf" nodes. fn rewrite(&self) -> MetaResult { self.rewrite_inner(&self.nodes, 0) } @@ -254,6 +254,44 @@ impl ActorBuilder { }) } + // "Leaf" node `SourceBackfill`. + NodeBody::SourceBackfill(source_backfill) => { + let input = stream_node.get_input(); + assert_eq!(input.len(), 1); + + let merge_node = &input[0]; + assert_matches!(merge_node.node_body, Some(NodeBody::Merge(_))); + + let upstream_source_id = source_backfill.source_id; + + // Index the upstreams by the an external edge ID. + let upstreams = &self.upstreams[&EdgeId::UpstreamExternal { + upstream_table_id: upstream_source_id.into(), + downstream_fragment_id: self.fragment_id, + }]; + + let upstream_actor_id = upstreams.actors.as_global_ids(); + + // rewrite the input of `SourceBackfill` + let input = vec![ + // Fill the merge node body with correct upstream info. + StreamNode { + node_body: Some(NodeBody::Merge(MergeNode { + upstream_actor_id, + upstream_fragment_id: upstreams.fragment_id.as_global_id(), + upstream_dispatcher_type: DispatcherType::NoShuffle as _, + fields: merge_node.fields.clone(), + })), + ..merge_node.clone() + }, + ]; + + Ok(StreamNode { + input, + ..stream_node.clone() + }) + } + // For other nodes, visit the children recursively. _ => { let mut new_stream_node = stream_node.clone(); @@ -622,6 +660,7 @@ impl ActorGraphBuildState { /// The result of a built actor graph. Will be further embedded into the `Context` for building /// actors on the compute nodes. +#[derive(Debug)] pub struct ActorGraphBuildResult { /// The graph of sealed fragments, including all actors. pub graph: BTreeMap, diff --git a/src/meta/src/stream/stream_graph/fragment.rs b/src/meta/src/stream/stream_graph/fragment.rs index fdf43532ca629..c68d981e3ae42 100644 --- a/src/meta/src/stream/stream_graph/fragment.rs +++ b/src/meta/src/stream/stream_graph/fragment.rs @@ -24,7 +24,7 @@ use risingwave_common::bail; use risingwave_common::catalog::{ generate_internal_table_name_with_type, TableId, CDC_SOURCE_COLUMN_NUM, }; -use risingwave_common::util::iter_util::ZipEqFast; +use risingwave_common::util::iter_util::{IntoIteratorExt, ZipEqFast}; use risingwave_common::util::stream_graph_visitor; use risingwave_pb::catalog::Table; use risingwave_pb::ddl_service::TableJobType; @@ -54,7 +54,8 @@ pub(super) struct BuildingFragment { /// The ID of the job if it contains the streaming job node. table_id: Option, - /// The required columns of each upstream table. + /// The required column IDs of each upstream table. + /// Will be converted to indices when building the edge connected to the upstream. /// /// For shared CDC table on source, its `vec![]`, since the upstream source's output schema is fixed. upstream_table_columns: HashMap>, @@ -177,6 +178,15 @@ impl BuildingFragment { stream_scan.upstream_column_ids.clone(), ), NodeBody::CdcFilter(cdc_filter) => (cdc_filter.upstream_source_id.into(), vec![]), + NodeBody::SourceBackfill(backfill) => ( + backfill.source_id.into(), + // FIXME: only pass required columns instead of all columns here + backfill + .columns + .iter() + .map(|c| c.column_desc.as_ref().unwrap().column_id) + .collect(), + ), _ => return, }; table_columns @@ -187,7 +197,7 @@ impl BuildingFragment { assert_eq!( table_columns.len(), fragment.upstream_table_ids.len(), - "fragment type: {}", + "fragment type: {:b}", fragment.fragment_type_mask ); @@ -286,7 +296,7 @@ impl StreamFragmentEdge { /// This only includes nodes and edges of the current job itself. It will be converted to [`CompleteStreamFragmentGraph`] later, /// that contains the additional information of pre-existing /// fragments, which are connected to the graph's top-most or bottom-most fragments. -#[derive(Default)] +#[derive(Default, Debug)] pub struct StreamFragmentGraph { /// stores all the fragments in the graph. fragments: HashMap, @@ -513,7 +523,7 @@ pub(super) enum EitherFragment { /// An internal fragment that is being built for the current streaming job. Building(BuildingFragment), - /// An existing fragment that is external but connected to the fragments being built. + /// An existing fragment that is external but connected to the fragments being built.!!!!!!!!!!!!! Existing(Fragment), } @@ -525,6 +535,7 @@ pub(super) enum EitherFragment { /// `Materialize` node will be included in this structure. /// - if we're going to replace the plan of a table with downstream mviews, the downstream fragments /// containing the `StreamScan` nodes will be included in this structure. +#[derive(Debug)] pub struct CompleteStreamFragmentGraph { /// The fragment graph of the streaming job being built. building_graph: StreamFragmentGraph, @@ -655,50 +666,96 @@ impl CompleteStreamFragmentGraph { (source_job_id, edge) } DdlType::MaterializedView | DdlType::Sink | DdlType::Index => { - // handle MV on MV + // handle MV on MV/Source // Build the extra edges between the upstream `Materialize` and the downstream `StreamScan` // of the new materialized view. - let mview_fragment = upstream_root_fragments + let upstream_fragment = upstream_root_fragments .get(&upstream_table_id) .context("upstream materialized view fragment not found")?; - let mview_id = GlobalFragmentId::new(mview_fragment.fragment_id); - - // Resolve the required output columns from the upstream materialized view. - let (dist_key_indices, output_indices) = { - let nodes = mview_fragment.actors[0].get_nodes().unwrap(); - let mview_node = - nodes.get_node_body().unwrap().as_materialize().unwrap(); - let all_column_ids = mview_node.column_ids(); - let dist_key_indices = mview_node.dist_key_indices(); - let output_indices = output_columns - .iter() - .map(|c| { - all_column_ids - .iter() - .position(|&id| id == *c) - .map(|i| i as u32) - }) - .collect::>>() - .context( - "column not found in the upstream materialized view", - )?; - (dist_key_indices, output_indices) - }; - let dispatch_strategy = mv_on_mv_dispatch_strategy( - uses_arrangement_backfill, - dist_key_indices, - output_indices, - ); - let edge = StreamFragmentEdge { - id: EdgeId::UpstreamExternal { - upstream_table_id, - downstream_fragment_id: id, - }, - dispatch_strategy, - }; - - (mview_id, edge) + let upstream_root_fragment_id = + GlobalFragmentId::new(upstream_fragment.fragment_id); + + if upstream_fragment.fragment_type_mask & FragmentTypeFlag::Mview as u32 + != 0 + { + // Resolve the required output columns from the upstream materialized view. + let (dist_key_indices, output_indices) = { + let nodes = upstream_fragment.actors[0].get_nodes().unwrap(); + let mview_node = + nodes.get_node_body().unwrap().as_materialize().unwrap(); + let all_column_ids = mview_node.column_ids(); + let dist_key_indices = mview_node.dist_key_indices(); + let output_indices = output_columns + .map_collect::<_, _, _, Option>>(|c| { + all_column_ids + .iter() + .position(|&id| id == *c) + .map(|i| i as u32) + }) + .context( + "column not found in the upstream materialized view", + )?; + (dist_key_indices, output_indices) + }; + let dispatch_strategy = mv_on_mv_dispatch_strategy( + uses_arrangement_backfill, + dist_key_indices, + output_indices, + ); + let edge = StreamFragmentEdge { + id: EdgeId::UpstreamExternal { + upstream_table_id, + downstream_fragment_id: id, + }, + dispatch_strategy, + }; + + (upstream_root_fragment_id, edge) + } else if upstream_fragment.fragment_type_mask + & FragmentTypeFlag::Source as u32 + != 0 + { + let source_fragment = upstream_root_fragments + .get(&upstream_table_id) + .context("upstream source fragment not found")?; + let source_job_id = + GlobalFragmentId::new(source_fragment.fragment_id); + + let output_indices = { + let nodes = upstream_fragment.actors[0].get_nodes().unwrap(); + let source_node = + nodes.get_node_body().unwrap().as_source().unwrap(); + + let all_column_ids = source_node.column_ids().unwrap(); + output_columns + .map_collect::<_, _, _, Option>>(|c| { + all_column_ids + .iter() + .position(|&id| id == *c) + .map(|i| i as u32) + }) + .context("column not found in the upstream source node")? + }; + + let edge = StreamFragmentEdge { + id: EdgeId::UpstreamExternal { + upstream_table_id, + downstream_fragment_id: id, + }, + // We always use `NoShuffle` for the exchange between the upstream + // `Source` and the downstream `StreamScan` of the new MV. + dispatch_strategy: DispatchStrategy { + r#type: DispatcherType::NoShuffle as _, + dist_key_indices: vec![], // not used for `NoShuffle` + output_indices, + }, + }; + + (source_job_id, edge) + } else { + bail!("the upstream fragment should be a MView or Source, got fragment type: {:b}", upstream_fragment.fragment_type_mask) + } } DdlType::Source | DdlType::Table(_) => { bail!("the streaming job shouldn't have an upstream fragment, ddl_type: {:?}", ddl_type) diff --git a/src/meta/src/stream/stream_graph/schedule.rs b/src/meta/src/stream/stream_graph/schedule.rs index ed2dac5be0e06..1ae24ec1b7d51 100644 --- a/src/meta/src/stream/stream_graph/schedule.rs +++ b/src/meta/src/stream/stream_graph/schedule.rs @@ -326,6 +326,7 @@ impl Scheduler { /// [`Locations`] represents the parallel unit and worker locations of the actors. #[cfg_attr(test, derive(Default))] +#[derive(Debug)] pub struct Locations { /// actor location map. pub actor_locations: BTreeMap, diff --git a/src/meta/src/stream/stream_manager.rs b/src/meta/src/stream/stream_manager.rs index a949780e06ed1..06d10ccd3ae0e 100644 --- a/src/meta/src/stream/stream_manager.rs +++ b/src/meta/src/stream/stream_manager.rs @@ -325,6 +325,7 @@ impl GlobalStreamManager { } } CreatingState::Created => { + tracing::debug!(id=?table_id, "streaming job created"); self.creating_job_info.delete_job(table_id).await; return Ok(()); } @@ -453,7 +454,7 @@ impl GlobalStreamManager { } let dummy_table_id = table_fragments.table_id(); - + // TODO: does this need change? for replace_table let init_split_assignment = self.source_manager.allocate_splits(&dummy_table_id).await?; @@ -470,7 +471,16 @@ impl GlobalStreamManager { let table_id = table_fragments.table_id(); - let init_split_assignment = self.source_manager.allocate_splits(&table_id).await?; + // Here we need to consider: + // - Source with streaming job (backfill-able source) + // - Table with connector + // - MV on backfill-able source + let mut init_split_assignment = self.source_manager.allocate_splits(&table_id).await?; + init_split_assignment.extend( + self.source_manager + .allocate_splits_for_backfill(&table_id, &dispatchers) + .await?, + ); let command = Command::CreateStreamingJob { table_fragments, @@ -481,7 +491,7 @@ impl GlobalStreamManager { ddl_type, replace_table: replace_table_command, }; - + tracing::trace!(?command, "sending first barrier for creating streaming job"); if let Err(err) = self.barrier_scheduler.run_command(command).await { if create_type == CreateType::Foreground || err.is_cancelled() { let mut table_ids = HashSet::from_iter(std::iter::once(table_id)); @@ -516,7 +526,7 @@ impl GlobalStreamManager { .await?; let dummy_table_id = table_fragments.table_id(); - + // TODO: does this need change? for replace_table let init_split_assignment = self.source_manager.allocate_splits(&dummy_table_id).await?; if let Err(err) = self diff --git a/src/prost/src/lib.rs b/src/prost/src/lib.rs index 82f399d1a3de6..775394668977b 100644 --- a/src/prost/src/lib.rs +++ b/src/prost/src/lib.rs @@ -196,6 +196,61 @@ impl stream_plan::MaterializeNode { } } +impl stream_plan::SourceNode { + pub fn column_ids(&self) -> Option> { + Some( + self.source_inner + .as_ref()? + .columns + .iter() + .map(|c| c.get_column_desc().unwrap().column_id) + .collect(), + ) + } +} + +impl stream_plan::StreamNode { + /// Find the external stream source info inside the stream node, if any. + /// + /// Returns `source_id`. + pub fn find_stream_source(&self) -> Option { + if let Some(crate::stream_plan::stream_node::NodeBody::Source(source)) = + self.node_body.as_ref() + { + if let Some(inner) = &source.source_inner { + return Some(inner.source_id); + } + } + + for child in &self.input { + if let Some(source) = child.find_stream_source() { + return Some(source); + } + } + + None + } + + /// Find the external stream source info inside the stream node, if any. + /// + /// Returns `source_id`. + pub fn find_source_backfill(&self) -> Option { + if let Some(crate::stream_plan::stream_node::NodeBody::SourceBackfill(source)) = + self.node_body.as_ref() + { + return Some(source.source_id); + } + + for child in &self.input { + if let Some(source) = child.find_source_backfill() { + return Some(source); + } + } + + None + } +} + #[cfg(test)] mod tests { use crate::data::{data_type, DataType}; diff --git a/src/stream/src/executor/source/kafka_backfill_executor.rs b/src/stream/src/executor/source/kafka_backfill_executor.rs index 8611e897940b6..91701c506581e 100644 --- a/src/stream/src/executor/source/kafka_backfill_executor.rs +++ b/src/stream/src/executor/source/kafka_backfill_executor.rs @@ -562,11 +562,11 @@ impl KafkaBackfillExecutorInner { } _ => {} } - self.backfill_state_store - .state_store - .commit(barrier.epoch) - .await?; } + self.backfill_state_store + .state_store + .commit(barrier.epoch) + .await?; yield Message::Barrier(barrier); } Message::Chunk(chunk) => { @@ -694,7 +694,7 @@ impl KafkaBackfillExecutorInner { // trim dropped splits' state self.backfill_state_store.trim_state(dropped_splits).await?; } - + tracing::info!(old_state=?stage.states, new_state=?target_state, "finish split change"); stage.states = target_state; } diff --git a/src/stream/src/executor/source/kafka_backfill_state_table.rs b/src/stream/src/executor/source/kafka_backfill_state_table.rs index 2fe4c66f5e269..509a63b27d4be 100644 --- a/src/stream/src/executor/source/kafka_backfill_state_table.rs +++ b/src/stream/src/executor/source/kafka_backfill_state_table.rs @@ -67,7 +67,6 @@ impl BackfillStateTableHandler { let mut ret = vec![]; while let Some(item) = state_table_iter.next().await { let row = item?.into_owned_row(); - tracing::debug!("scanning backfill state table, row: {:?}", row); let state = match row.datum_at(1) { Some(ScalarRefImpl::Jsonb(jsonb_ref)) => { BackfillState::restore_from_json(jsonb_ref.to_owned_scalar())?