Skip to main content

risingwave_stream/executor/backfill/cdc/
cdc_backill_v2.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;
16
17use either::Either;
18use futures::stream;
19use futures::stream::select_with_strategy;
20use itertools::Itertools;
21use risingwave_common::bitmap::BitmapBuilder;
22use risingwave_common::catalog::{ColumnDesc, Field};
23use risingwave_common::row::RowDeserializer;
24use risingwave_common::util::iter_util::ZipEqFast;
25use risingwave_common::util::sort_util::{OrderType, cmp_datum};
26use risingwave_connector::parser::{
27    BigintUnsignedHandlingMode, TimeHandling, TimestampHandling, TimestamptzHandling,
28};
29use risingwave_connector::source::cdc::CdcScanOptions;
30use risingwave_connector::source::cdc::external::{
31    CdcOffset, ExternalCdcTableType, ExternalTableReaderImpl,
32};
33use risingwave_connector::source::{CdcTableSnapshotSplit, CdcTableSnapshotSplitRaw};
34use risingwave_pb::common::ThrottleType;
35use rw_futures_util::pausable;
36use thiserror_ext::AsReport;
37use tracing::Instrument;
38
39use crate::executor::UpdateMutation;
40use crate::executor::backfill::cdc::cdc_backfill::{
41    build_reader_and_poll_upstream, transform_upstream,
42};
43use crate::executor::backfill::cdc::state_v2::ParallelizedCdcBackfillState;
44use crate::executor::backfill::cdc::upstream_table::external::ExternalStorageTable;
45use crate::executor::backfill::cdc::upstream_table::snapshot::{
46    SplitSnapshotReadArgs, UpstreamTableRead, UpstreamTableReader,
47};
48use crate::executor::backfill::utils::{get_cdc_chunk_last_offset, mapping_chunk, mapping_message};
49use crate::executor::prelude::*;
50use crate::executor::source::get_infinite_backoff_strategy;
51use crate::task::cdc_progress::CdcProgressReporter;
52pub struct ParallelizedCdcBackfillExecutor<S: StateStore> {
53    actor_ctx: ActorContextRef,
54
55    /// The external table to be backfilled
56    external_table: ExternalStorageTable,
57
58    /// Upstream changelog stream which may contain metadata columns, e.g. `_rw_offset`
59    upstream: Executor,
60
61    /// The column indices need to be forwarded to the downstream from the upstream and table scan.
62    output_indices: Vec<usize>,
63
64    /// The schema of output chunk, including additional columns if any
65    output_columns: Vec<ColumnDesc>,
66
67    /// Rate limit in rows/s.
68    rate_limit_rps: Option<u32>,
69
70    options: CdcScanOptions,
71
72    state_table: StateTable<S>,
73
74    properties: BTreeMap<String, String>,
75
76    progress: Option<CdcProgressReporter>,
77}
78
79impl<S: StateStore> ParallelizedCdcBackfillExecutor<S> {
80    #[expect(clippy::too_many_arguments)]
81    pub fn new(
82        actor_ctx: ActorContextRef,
83        external_table: ExternalStorageTable,
84        upstream: Executor,
85        output_indices: Vec<usize>,
86        output_columns: Vec<ColumnDesc>,
87        _metrics: Arc<StreamingMetrics>,
88        state_table: StateTable<S>,
89        rate_limit_rps: Option<u32>,
90        options: CdcScanOptions,
91        properties: BTreeMap<String, String>,
92        progress: Option<CdcProgressReporter>,
93    ) -> Self {
94        Self {
95            actor_ctx,
96            external_table,
97            upstream,
98            output_indices,
99            output_columns,
100            rate_limit_rps,
101            options,
102            state_table,
103            properties,
104            progress,
105        }
106    }
107
108    #[try_stream(ok = Message, error = StreamExecutorError)]
109    async fn execute_inner(mut self) {
110        assert!(!self.options.disable_backfill);
111        // The indices to primary key columns
112        let pk_indices = self.external_table.pk_indices().to_vec();
113        let table_id = self.external_table.table_id();
114        let upstream_table_name = self.external_table.qualified_table_name();
115        let schema_table_name = self.external_table.schema_table_name().clone();
116        let external_database_name = self.external_table.database_name().to_owned();
117        let additional_columns = self
118            .output_columns
119            .iter()
120            .filter(|col| col.additional_column.column_type.is_some())
121            .cloned()
122            .collect_vec();
123        assert!(
124            (self.options.backfill_split_pk_column_index as usize) < pk_indices.len(),
125            "split pk column index {} out of bound",
126            self.options.backfill_split_pk_column_index
127        );
128        let snapshot_split_column_index =
129            pk_indices[self.options.backfill_split_pk_column_index as usize];
130        let cdc_table_snapshot_split_column =
131            vec![self.external_table.schema().fields[snapshot_split_column_index].clone()];
132
133        let mut upstream = self.upstream.execute();
134        // Poll the upstream to get the first barrier.
135        let first_barrier = expect_first_barrier(&mut upstream).await?;
136        // Make sure to use mapping_message after transform_upstream.
137
138        // If user sets debezium.time.precision.mode to "connect", it means the user can guarantee
139        // that the upstream data precision is MilliSecond. In this case, we don't use GuessNumberUnit
140        // mode to guess precision, but use Milli mode directly, which can handle extreme timestamps.
141        let timestamp_handling: Option<TimestampHandling> = self
142            .properties
143            .get("debezium.time.precision.mode")
144            .map(|v| v == "connect")
145            .unwrap_or(false)
146            .then_some(TimestampHandling::Milli);
147        let timestamptz_handling: Option<TimestamptzHandling> = self
148            .properties
149            .get("debezium.time.precision.mode")
150            .map(|v| v == "connect")
151            .unwrap_or(false)
152            .then_some(TimestamptzHandling::Milli);
153        let time_handling: Option<TimeHandling> = self
154            .properties
155            .get("debezium.time.precision.mode")
156            .map(|v| v == "connect")
157            .unwrap_or(false)
158            .then_some(TimeHandling::Milli);
159        let bigint_unsigned_handling: Option<BigintUnsignedHandlingMode> = self
160            .properties
161            .get("debezium.bigint.unsigned.handling.mode")
162            .map(|v| v == "precise")
163            .unwrap_or(false)
164            .then_some(BigintUnsignedHandlingMode::Precise);
165        // Only postgres-cdc connector may trigger TOAST.
166        let handle_toast_columns: bool =
167            self.external_table.table_type() == &ExternalCdcTableType::Postgres;
168        let mut upstream = transform_upstream(
169            upstream,
170            self.output_columns.clone(),
171            timestamp_handling,
172            timestamptz_handling,
173            time_handling,
174            bigint_unsigned_handling,
175            handle_toast_columns,
176        )
177        .boxed();
178        let mut next_reset_barrier = Some(first_barrier);
179        let mut is_reset = false;
180        let mut state_impl = ParallelizedCdcBackfillState::new(self.state_table);
181        // The buffered chunks have already been mapped.
182        let mut upstream_chunk_buffer: Vec<StreamChunk> = vec![];
183
184        // Need reset on CDC table snapshot splits reschedule.
185        'with_cdc_table_snapshot_splits: loop {
186            assert!(upstream_chunk_buffer.is_empty());
187            let reset_barrier = next_reset_barrier.take().unwrap();
188            let all_snapshot_splits = match reset_barrier.mutation.as_deref() {
189                Some(Mutation::Add(add)) => &add.actor_cdc_table_snapshot_splits.splits,
190
191                Some(Mutation::Update(update)) => &update.actor_cdc_table_snapshot_splits.splits,
192                _ => {
193                    return Err(anyhow::anyhow!("ParallelizedCdcBackfillExecutor expects either Mutation::Add or Mutation::Update to initialize CDC table snapshot splits.").into());
194                }
195            };
196            let mut actor_snapshot_splits = vec![];
197            let mut generation = None;
198            // TODO(zw): optimization: remove consumed splits to reduce barrier size for downstream.
199            if let Some((splits, snapshot_generation)) = all_snapshot_splits.get(&self.actor_ctx.id)
200            {
201                actor_snapshot_splits = splits
202                    .iter()
203                    .map(|s: &CdcTableSnapshotSplitRaw| {
204                        let de = RowDeserializer::new(
205                            cdc_table_snapshot_split_column
206                                .iter()
207                                .map(Field::data_type)
208                                .collect_vec(),
209                        );
210                        let left_bound_inclusive =
211                            de.deserialize(s.left_bound_inclusive.as_ref()).unwrap();
212                        let right_bound_exclusive =
213                            de.deserialize(s.right_bound_exclusive.as_ref()).unwrap();
214                        CdcTableSnapshotSplit {
215                            split_id: s.split_id,
216                            left_bound_inclusive,
217                            right_bound_exclusive,
218                        }
219                    })
220                    .collect();
221                generation = Some(*snapshot_generation);
222            }
223            tracing::debug!(?actor_snapshot_splits, ?generation, "actor splits");
224            assert_consecutive_splits(&actor_snapshot_splits);
225
226            let mut is_snapshot_paused = reset_barrier.is_pause_on_startup();
227            let barrier_epoch = reset_barrier.epoch;
228            yield Message::Barrier(reset_barrier);
229            if !is_reset {
230                state_impl.init_epoch(barrier_epoch).await?;
231                is_reset = true;
232                tracing::info!(%table_id, "Initialize executor.");
233            } else {
234                tracing::info!(%table_id, "Reset executor.");
235            }
236
237            let mut current_actor_bounds = None;
238            let mut actor_cdc_offset_high: Option<CdcOffset> = None;
239            let mut actor_cdc_offset_low: Option<CdcOffset> = None;
240            // Find next split that need backfill.
241            let mut next_split_idx = actor_snapshot_splits.len();
242            for (idx, split) in actor_snapshot_splits.iter().enumerate() {
243                let state = state_impl.restore_state(split.split_id).await?;
244                if !state.is_finished {
245                    next_split_idx = idx;
246                    break;
247                }
248                extends_current_actor_bound(&mut current_actor_bounds, split);
249                if let Some(ref cdc_offset) = state.cdc_offset_low {
250                    if let Some(ref cur) = actor_cdc_offset_low {
251                        if *cur > *cdc_offset {
252                            actor_cdc_offset_low = state.cdc_offset_low.clone();
253                        }
254                    } else {
255                        actor_cdc_offset_low = state.cdc_offset_low.clone();
256                    }
257                }
258                if let Some(ref cdc_offset) = state.cdc_offset_high {
259                    if let Some(ref cur) = actor_cdc_offset_high {
260                        if *cur < *cdc_offset {
261                            actor_cdc_offset_high = state.cdc_offset_high.clone();
262                        }
263                    } else {
264                        actor_cdc_offset_high = state.cdc_offset_high.clone();
265                    }
266                }
267            }
268            for split in actor_snapshot_splits.iter().skip(next_split_idx) {
269                // Initialize state so that overall progress can be measured.
270                state_impl
271                    .mutate_state(split.split_id, false, 0, None, None)
272                    .await?;
273            }
274            let mut should_report_actor_backfill_progress = if next_split_idx > 0 {
275                Some((
276                    actor_snapshot_splits[0].split_id,
277                    actor_snapshot_splits[next_split_idx - 1].split_id,
278                ))
279            } else {
280                None
281            };
282
283            // After init the state table and forward the initial barrier to downstream,
284            // we now try to create the table reader with retry.
285            let mut table_reader: Option<ExternalTableReaderImpl> = None;
286            let external_table = self.external_table.clone();
287            let mut future = Box::pin(async move {
288                let backoff = get_infinite_backoff_strategy();
289                tokio_retry::Retry::spawn(backoff, || async {
290                    match external_table.create_table_reader().await {
291                        Ok(reader) => Ok(reader),
292                        Err(e) => {
293                            tracing::warn!(error = %e.as_report(), "failed to create cdc table reader, retrying...");
294                            Err(e)
295                        }
296                    }
297                })
298                    .instrument(tracing::info_span!("create_cdc_table_reader_with_retry"))
299                    .await
300                    .expect("Retry create cdc table reader until success.")
301            });
302            loop {
303                if let Some(msg) =
304                    build_reader_and_poll_upstream(&mut upstream, &mut table_reader, &mut future)
305                        .await?
306                {
307                    if let Some(msg) = mapping_message(msg, &self.output_indices) {
308                        match msg {
309                            Message::Barrier(barrier) => {
310                                state_impl.commit_state(barrier.epoch).await?;
311                                if is_reset_barrier(&barrier, self.actor_ctx.id) {
312                                    next_reset_barrier = Some(barrier);
313                                    continue 'with_cdc_table_snapshot_splits;
314                                }
315                                yield Message::Barrier(barrier);
316                            }
317                            Message::Chunk(chunk) => {
318                                if chunk.cardinality() == 0 {
319                                    continue;
320                                }
321                                if let Some(filtered_chunk) = filter_stream_chunk(
322                                    chunk,
323                                    &current_actor_bounds,
324                                    snapshot_split_column_index,
325                                ) && filtered_chunk.cardinality() > 0
326                                {
327                                    yield Message::Chunk(filtered_chunk);
328                                }
329                            }
330                            Message::Watermark(_) => {
331                                // Ignore watermark, like the `CdcBackfillExecutor`.
332                            }
333                        }
334                    }
335                } else {
336                    assert!(table_reader.is_some(), "table reader must created");
337                    tracing::info!(
338                        %table_id,
339                        upstream_table_name,
340                        "table reader created successfully"
341                    );
342                    break;
343                }
344            }
345            let upstream_table_reader = UpstreamTableReader::new(
346                self.external_table.clone(),
347                table_reader.expect("table reader must created"),
348            );
349            // let mut upstream = upstream.peekable();
350            let offset_parse_func = upstream_table_reader.reader.get_cdc_offset_parser();
351
352            // Backfill snapshot splits sequentially.
353            for split in actor_snapshot_splits.iter().skip(next_split_idx) {
354                tracing::info!(
355                    %table_id,
356                    upstream_table_name,
357                    ?split,
358                    is_snapshot_paused,
359                    "start cdc backfill split"
360                );
361                extends_current_actor_bound(&mut current_actor_bounds, split);
362
363                let split_cdc_offset_low = {
364                    // Limit concurrent CDC connections globally to 10 using a semaphore.
365                    static CDC_CONN_SEMAPHORE: tokio::sync::Semaphore =
366                        tokio::sync::Semaphore::const_new(10);
367
368                    let _permit = CDC_CONN_SEMAPHORE.acquire().await.unwrap();
369                    upstream_table_reader.current_cdc_offset().await?
370                };
371                if let Some(ref cdc_offset) = split_cdc_offset_low {
372                    if let Some(ref cur) = actor_cdc_offset_low {
373                        if *cur > *cdc_offset {
374                            actor_cdc_offset_low = split_cdc_offset_low.clone();
375                        }
376                    } else {
377                        actor_cdc_offset_low = split_cdc_offset_low.clone();
378                    }
379                }
380                let mut split_cdc_offset_high = None;
381
382                let left_upstream = upstream.by_ref().map(Either::Left);
383                let read_args = SplitSnapshotReadArgs::new(
384                    split.left_bound_inclusive.clone(),
385                    split.right_bound_exclusive.clone(),
386                    cdc_table_snapshot_split_column.clone(),
387                    self.rate_limit_rps,
388                    additional_columns.clone(),
389                    schema_table_name.clone(),
390                    external_database_name.clone(),
391                );
392                let right_snapshot = pin!(
393                    upstream_table_reader
394                        .snapshot_read_table_split(read_args)
395                        .map(Either::Right)
396                );
397                let (right_snapshot, snapshot_valve) = pausable(right_snapshot);
398                if is_snapshot_paused {
399                    snapshot_valve.pause();
400                }
401                let mut backfill_stream =
402                    select_with_strategy(left_upstream, right_snapshot, |_: &mut ()| {
403                        stream::PollNext::Left
404                    });
405                let mut row_count: u64 = 0;
406                #[for_await]
407                for either in &mut backfill_stream {
408                    match either {
409                        // Upstream
410                        Either::Left(msg) => {
411                            match msg? {
412                                Message::Barrier(barrier) => {
413                                    state_impl.commit_state(barrier.epoch).await?;
414                                    if let Some(mutation) = barrier.mutation.as_deref() {
415                                        use crate::executor::Mutation;
416                                        match mutation {
417                                            Mutation::Pause => {
418                                                is_snapshot_paused = true;
419                                                snapshot_valve.pause();
420                                            }
421                                            Mutation::Resume => {
422                                                is_snapshot_paused = false;
423                                                snapshot_valve.resume();
424                                            }
425                                            Mutation::Throttle(some) => {
426                                                // TODO(zw): optimization: improve throttle.
427                                                // 1. Handle rate limit 0. Currently, to resume the process, the actor must be rebuilt.
428                                                // 2. Apply new rate limit immediately.
429                                                if let Some(entry) =
430                                                    some.get(&self.actor_ctx.fragment_id)
431                                                    && entry.throttle_type()
432                                                        == ThrottleType::Backfill
433                                                    && entry.rate_limit != self.rate_limit_rps
434                                                {
435                                                    // The new rate limit will take effect since next split.
436                                                    self.rate_limit_rps = entry.rate_limit;
437                                                }
438                                            }
439                                            Mutation::Update(UpdateMutation {
440                                                dropped_actors,
441                                                ..
442                                            }) if dropped_actors.contains(&self.actor_ctx.id) => {
443                                                tracing::info!(
444                                                    %table_id,
445                                                    upstream_table_name,
446                                                    "CdcBackfill has been dropped due to config change"
447                                                );
448                                                for chunk in upstream_chunk_buffer.drain(..) {
449                                                    yield Message::Chunk(chunk);
450                                                }
451                                                yield Message::Barrier(barrier);
452                                                let () = futures::future::pending().await;
453                                                unreachable!();
454                                            }
455                                            _ => (),
456                                        }
457                                    }
458                                    if is_reset_barrier(&barrier, self.actor_ctx.id) {
459                                        next_reset_barrier = Some(barrier);
460                                        for chunk in upstream_chunk_buffer.drain(..) {
461                                            yield Message::Chunk(chunk);
462                                        }
463                                        continue 'with_cdc_table_snapshot_splits;
464                                    }
465                                    if let Some(split_range) =
466                                        should_report_actor_backfill_progress.take()
467                                        && let Some(ref progress) = self.progress
468                                    {
469                                        progress.update(
470                                            self.actor_ctx.fragment_id,
471                                            self.actor_ctx.id,
472                                            barrier.epoch,
473                                            generation.expect("should have set generation when having progress to report"),
474                                            split_range,
475                                        );
476                                    }
477                                    // emit barrier and continue to consume the backfill stream
478                                    yield Message::Barrier(barrier);
479                                }
480                                Message::Chunk(chunk) => {
481                                    // skip empty upstream chunk
482                                    if chunk.cardinality() == 0 {
483                                        continue;
484                                    }
485
486                                    // TODO(zw): re-enable
487                                    // let chunk_cdc_offset =
488                                    //     get_cdc_chunk_last_offset(&offset_parse_func, &chunk)?;
489                                    // if *self.external_table.table_type()
490                                    //     == ExternalCdcTableType::Postgres
491                                    //     && let Some(cur) = actor_cdc_offset_low.as_ref()
492                                    //     && let Some(chunk_offset) = chunk_cdc_offset
493                                    //     && chunk_offset < *cur
494                                    // {
495                                    //     continue;
496                                    // }
497
498                                    let chunk = mapping_chunk(chunk, &self.output_indices);
499                                    if let Some(filtered_chunk) = filter_stream_chunk(
500                                        chunk,
501                                        &current_actor_bounds,
502                                        snapshot_split_column_index,
503                                    ) && filtered_chunk.cardinality() > 0
504                                    {
505                                        // Buffer the upstream chunk.
506                                        upstream_chunk_buffer.push(filtered_chunk.compact_vis());
507                                    }
508                                }
509                                Message::Watermark(_) => {
510                                    // Ignore watermark during backfill, like the `CdcBackfillExecutor`.
511                                }
512                            }
513                        }
514                        // Snapshot read
515                        Either::Right(msg) => {
516                            match msg? {
517                                None => {
518                                    tracing::info!(
519                                        %table_id,
520                                        split_id = split.split_id,
521                                        "snapshot read stream ends"
522                                    );
523                                    for chunk in upstream_chunk_buffer.drain(..) {
524                                        yield Message::Chunk(chunk);
525                                    }
526
527                                    split_cdc_offset_high = {
528                                        // Limit concurrent CDC connections globally to 10 using a semaphore.
529                                        static CDC_CONN_SEMAPHORE: tokio::sync::Semaphore =
530                                            tokio::sync::Semaphore::const_new(10);
531
532                                        let _permit = CDC_CONN_SEMAPHORE.acquire().await.unwrap();
533                                        upstream_table_reader.current_cdc_offset().await?
534                                    };
535                                    if let Some(ref cdc_offset) = split_cdc_offset_high {
536                                        if let Some(ref cur) = actor_cdc_offset_high {
537                                            if *cur < *cdc_offset {
538                                                actor_cdc_offset_high =
539                                                    split_cdc_offset_high.clone();
540                                            }
541                                        } else {
542                                            actor_cdc_offset_high = split_cdc_offset_high.clone();
543                                        }
544                                    }
545                                    // Next split.
546                                    break;
547                                }
548                                Some(chunk) => {
549                                    let chunk_cardinality = chunk.cardinality() as u64;
550                                    row_count = row_count.saturating_add(chunk_cardinality);
551                                    yield Message::Chunk(mapping_chunk(
552                                        chunk,
553                                        &self.output_indices,
554                                    ));
555                                }
556                            }
557                        }
558                    }
559                }
560                // Mark current split backfill as finished. The state will be persisted by next barrier.
561                state_impl
562                    .mutate_state(
563                        split.split_id,
564                        true,
565                        row_count,
566                        split_cdc_offset_low,
567                        split_cdc_offset_high,
568                    )
569                    .await?;
570                if let Some((_, right_split)) = &mut should_report_actor_backfill_progress {
571                    assert!(
572                        *right_split < split.split_id,
573                        "{} {}",
574                        *right_split,
575                        split.split_id
576                    );
577                    *right_split = split.split_id;
578                } else {
579                    should_report_actor_backfill_progress = Some((split.split_id, split.split_id));
580                }
581            }
582
583            upstream_table_reader.disconnect().await?;
584            tracing::info!(
585                %table_id,
586                upstream_table_name,
587                "CdcBackfill has already finished and will forward messages directly to the downstream"
588            );
589
590            let mut should_report_actor_backfill_done = false;
591            // After backfill progress finished
592            // we can forward messages directly to the downstream,
593            // as backfill is finished.
594            #[for_await]
595            for msg in &mut upstream {
596                let msg = msg?;
597                match msg {
598                    Message::Barrier(barrier) => {
599                        state_impl.commit_state(barrier.epoch).await?;
600                        if is_reset_barrier(&barrier, self.actor_ctx.id) {
601                            next_reset_barrier = Some(barrier);
602                            continue 'with_cdc_table_snapshot_splits;
603                        }
604                        if let Some(split_range) = should_report_actor_backfill_progress.take()
605                            && let Some(ref progress) = self.progress
606                        {
607                            progress.update(
608                                self.actor_ctx.fragment_id,
609                                self.actor_ctx.id,
610                                barrier.epoch,
611                                generation.expect(
612                                    "should have set generation when having progress to report",
613                                ),
614                                split_range,
615                            );
616                        }
617                        if should_report_actor_backfill_done {
618                            should_report_actor_backfill_done = false;
619                            assert!(!actor_snapshot_splits.is_empty());
620                            if let Some(ref progress) = self.progress {
621                                progress.finish(
622                                    self.actor_ctx.fragment_id,
623                                    self.actor_ctx.id,
624                                    barrier.epoch,
625                                    generation.expect(
626                                        "should have set generation when having progress to report",
627                                    ),
628                                    (
629                                        actor_snapshot_splits[0].split_id,
630                                        actor_snapshot_splits[actor_snapshot_splits.len() - 1]
631                                            .split_id,
632                                    ),
633                                );
634                            }
635                        }
636                        yield Message::Barrier(barrier);
637                    }
638                    Message::Chunk(chunk) => {
639                        if actor_snapshot_splits.is_empty() {
640                            continue;
641                        }
642                        if chunk.cardinality() == 0 {
643                            continue;
644                        }
645
646                        let chunk_cdc_offset =
647                            get_cdc_chunk_last_offset(&offset_parse_func, &chunk)?;
648                        // // TODO(zw): re-enable
649                        // if *self.external_table.table_type() == ExternalCdcTableType::Postgres
650                        //     && let Some(cur) = actor_cdc_offset_low.as_ref()
651                        //     && let Some(ref chunk_offset) = chunk_cdc_offset
652                        //     && *chunk_offset < *cur
653                        // {
654                        //     continue;
655                        // }
656
657                        // should_report_actor_backfill_done is set to true at most once.
658                        if let Some(high) = actor_cdc_offset_high.as_ref() {
659                            if state_impl.is_legacy_state() {
660                                // Since the legacy state does not track CDC offsets, report backfill completion immediately.
661                                actor_cdc_offset_high = None;
662                                should_report_actor_backfill_done = true;
663                            } else if let Some(ref chunk_offset) = chunk_cdc_offset
664                                && *chunk_offset >= *high
665                            {
666                                // Report backfill completion once the latest CDC offset exceeds the highest offset tracked during the backfill.
667                                actor_cdc_offset_high = None;
668                                should_report_actor_backfill_done = true;
669                            }
670                        }
671                        let chunk = mapping_chunk(chunk, &self.output_indices);
672                        if let Some(filtered_chunk) = filter_stream_chunk(
673                            chunk,
674                            &current_actor_bounds,
675                            snapshot_split_column_index,
676                        ) && filtered_chunk.cardinality() > 0
677                        {
678                            yield Message::Chunk(filtered_chunk);
679                        }
680                    }
681                    msg @ Message::Watermark(_) => {
682                        if let Some(msg) = mapping_message(msg, &self.output_indices) {
683                            yield msg;
684                        }
685                    }
686                }
687            }
688        }
689    }
690}
691
692fn filter_stream_chunk(
693    chunk: StreamChunk,
694    bound: &Option<(OwnedRow, OwnedRow)>,
695    snapshot_split_column_index: usize,
696) -> Option<StreamChunk> {
697    let Some((left, right)) = bound else {
698        return None;
699    };
700    assert_eq!(left.len(), 1, "multiple split columns is not supported yet");
701    assert_eq!(
702        right.len(),
703        1,
704        "multiple split columns is not supported yet"
705    );
706    let left_split_key = left.datum_at(0);
707    let right_split_key = right.datum_at(0);
708    let is_leftmost_bound = is_leftmost_bound(left);
709    let is_rightmost_bound = is_rightmost_bound(right);
710    if is_leftmost_bound && is_rightmost_bound {
711        return Some(chunk);
712    }
713    let mut new_bitmap = BitmapBuilder::with_capacity(chunk.capacity());
714    let (ops, columns, visibility) = chunk.into_inner();
715    for (row_split_key, v) in columns[snapshot_split_column_index]
716        .iter()
717        .zip_eq_fast(visibility.iter())
718    {
719        if !v {
720            new_bitmap.append(false);
721            continue;
722        }
723        let mut is_in_range = true;
724        if !is_leftmost_bound {
725            is_in_range = cmp_datum(
726                row_split_key,
727                left_split_key,
728                OrderType::ascending_nulls_first(),
729            )
730            .is_ge();
731        }
732        if is_in_range && !is_rightmost_bound {
733            is_in_range = cmp_datum(
734                row_split_key,
735                right_split_key,
736                OrderType::ascending_nulls_first(),
737            )
738            .is_lt();
739        }
740        if !is_in_range {
741            tracing::trace!(?row_split_key, ?left_split_key, ?right_split_key, snapshot_split_column_index, data_type = ?columns[snapshot_split_column_index].data_type(), "filter out row")
742        }
743        new_bitmap.append(is_in_range);
744    }
745    Some(StreamChunk::with_visibility(
746        ops,
747        columns,
748        new_bitmap.finish(),
749    ))
750}
751
752fn is_leftmost_bound(row: &OwnedRow) -> bool {
753    row.iter().all(|d| d.is_none())
754}
755
756fn is_rightmost_bound(row: &OwnedRow) -> bool {
757    row.iter().all(|d| d.is_none())
758}
759
760impl<S: StateStore> Execute for ParallelizedCdcBackfillExecutor<S> {
761    fn execute(self: Box<Self>) -> BoxedMessageStream {
762        self.execute_inner().boxed()
763    }
764}
765
766fn extends_current_actor_bound(
767    current: &mut Option<(OwnedRow, OwnedRow)>,
768    split: &CdcTableSnapshotSplit,
769) {
770    if current.is_none() {
771        *current = Some((
772            split.left_bound_inclusive.clone(),
773            split.right_bound_exclusive.clone(),
774        ));
775    } else {
776        current.as_mut().unwrap().1 = split.right_bound_exclusive.clone();
777    }
778}
779
780fn is_reset_barrier(barrier: &Barrier, actor_id: ActorId) -> bool {
781    match barrier.mutation.as_deref() {
782        Some(Mutation::Update(update)) => update
783            .actor_cdc_table_snapshot_splits
784            .splits
785            .contains_key(&actor_id),
786        _ => false,
787    }
788}
789
790fn assert_consecutive_splits(actor_snapshot_splits: &[CdcTableSnapshotSplit]) {
791    for i in 1..actor_snapshot_splits.len() {
792        assert_eq!(
793            actor_snapshot_splits[i].split_id,
794            actor_snapshot_splits[i - 1].split_id + 1,
795            "{:?}",
796            actor_snapshot_splits
797        );
798        assert!(
799            cmp_datum(
800                actor_snapshot_splits[i - 1]
801                    .right_bound_exclusive
802                    .datum_at(0),
803                actor_snapshot_splits[i].right_bound_exclusive.datum_at(0),
804                OrderType::ascending_nulls_last(),
805            )
806            .is_lt()
807        );
808    }
809}
810
811#[cfg(test)]
812mod tests {
813    use risingwave_common::array::StreamChunk;
814    use risingwave_common::row::OwnedRow;
815    use risingwave_common::types::ScalarImpl;
816
817    use crate::executor::backfill::cdc::cdc_backill_v2::filter_stream_chunk;
818
819    #[test]
820    fn test_filter_stream_chunk() {
821        use risingwave_common::array::StreamChunkTestExt;
822        let chunk = StreamChunk::from_pretty(
823            "  I I
824             + 1 6
825             - 2 .
826            U- 3 7
827            U+ 4 .",
828        );
829        let bound = None;
830        let c = filter_stream_chunk(chunk.clone(), &bound, 0);
831        assert!(c.is_none());
832
833        let bound = Some((OwnedRow::new(vec![None]), OwnedRow::new(vec![None])));
834        let c = filter_stream_chunk(chunk.clone(), &bound, 0);
835        assert_eq!(c.unwrap().compact_vis(), chunk);
836
837        let bound = Some((
838            OwnedRow::new(vec![None]),
839            OwnedRow::new(vec![Some(ScalarImpl::Int64(3))]),
840        ));
841        let c = filter_stream_chunk(chunk.clone(), &bound, 0);
842        assert_eq!(
843            c.unwrap().compact_vis(),
844            StreamChunk::from_pretty(
845                "  I I
846             + 1 6
847             - 2 .",
848            )
849        );
850
851        let bound = Some((
852            OwnedRow::new(vec![Some(ScalarImpl::Int64(3))]),
853            OwnedRow::new(vec![None]),
854        ));
855        let c = filter_stream_chunk(chunk.clone(), &bound, 0);
856        assert_eq!(
857            c.unwrap().compact_vis(),
858            StreamChunk::from_pretty(
859                "  I I
860            U- 3 7
861            U+ 4 .",
862            )
863        );
864
865        let bound = Some((
866            OwnedRow::new(vec![Some(ScalarImpl::Int64(2))]),
867            OwnedRow::new(vec![Some(ScalarImpl::Int64(4))]),
868        ));
869        let c = filter_stream_chunk(chunk.clone(), &bound, 0);
870        assert_eq!(
871            c.unwrap().compact_vis(),
872            StreamChunk::from_pretty(
873                "  I I
874             - 2 .
875            U- 3 7",
876            )
877        );
878
879        // Test NULL value.
880        let bound = None;
881        let c = filter_stream_chunk(chunk.clone(), &bound, 1);
882        assert!(c.is_none());
883
884        let bound = Some((OwnedRow::new(vec![None]), OwnedRow::new(vec![None])));
885        let c = filter_stream_chunk(chunk.clone(), &bound, 1);
886        assert_eq!(c.unwrap().compact_vis(), chunk);
887
888        let bound = Some((
889            OwnedRow::new(vec![None]),
890            OwnedRow::new(vec![Some(ScalarImpl::Int64(7))]),
891        ));
892        let c = filter_stream_chunk(chunk.clone(), &bound, 1);
893        assert_eq!(
894            c.unwrap().compact_vis(),
895            StreamChunk::from_pretty(
896                "  I I
897             + 1 6
898             - 2 .
899            U+ 4 .",
900            )
901        );
902
903        let bound = Some((
904            OwnedRow::new(vec![Some(ScalarImpl::Int64(7))]),
905            OwnedRow::new(vec![None]),
906        ));
907        let c = filter_stream_chunk(chunk, &bound, 1);
908        assert_eq!(
909            c.unwrap().compact_vis(),
910            StreamChunk::from_pretty(
911                "  I I
912            U- 3 7",
913            )
914        );
915    }
916}