1mod prelude;
16
17use std::collections::{BTreeMap, HashMap, HashSet, VecDeque};
18use std::fmt::Debug;
19use std::future::pending;
20use std::hash::Hash;
21use std::pin::Pin;
22use std::sync::Arc;
23use std::task::Poll;
24use std::vec;
25
26use await_tree::InstrumentAwait;
27use enum_as_inner::EnumAsInner;
28use futures::future::try_join_all;
29use futures::stream::{BoxStream, FusedStream, FuturesUnordered, StreamFuture};
30use futures::{FutureExt, Stream, StreamExt, TryStreamExt};
31use itertools::Itertools;
32use prometheus::Histogram;
33use prometheus::core::{AtomicU64, GenericCounter};
34use risingwave_common::array::StreamChunk;
35use risingwave_common::bitmap::Bitmap;
36use risingwave_common::catalog::{Field, Schema, TableId};
37use risingwave_common::metrics::LabelGuardedMetric;
38use risingwave_common::row::OwnedRow;
39use risingwave_common::types::{DataType, Datum, DefaultOrd, ScalarImpl};
40use risingwave_common::util::epoch::{Epoch, EpochPair};
41use risingwave_common::util::tracing::TracingContext;
42use risingwave_common::util::value_encoding::{DatumFromProtoExt, DatumToProtoExt};
43use risingwave_connector::source::SplitImpl;
44use risingwave_expr::expr::{Expression, NonStrictExpression};
45use risingwave_pb::data::PbEpoch;
46use risingwave_pb::expr::PbInputRef;
47use risingwave_pb::stream_plan::add_mutation::PbNewUpstreamSink;
48use risingwave_pb::stream_plan::barrier::BarrierKind;
49use risingwave_pb::stream_plan::barrier_mutation::Mutation as PbMutation;
50use risingwave_pb::stream_plan::stream_node::PbStreamKind;
51use risingwave_pb::stream_plan::update_mutation::{DispatcherUpdate, MergeUpdate};
52use risingwave_pb::stream_plan::{
53 PbBarrier, PbBarrierMutation, PbDispatcher, PbStreamMessageBatch, PbWatermark,
54 SubscriptionUpstreamInfo,
55};
56use smallvec::SmallVec;
57use tokio::sync::mpsc;
58use tokio::time::{Duration, Instant};
59
60use crate::error::StreamResult;
61use crate::executor::exchange::input::{
62 BoxedActorInput, BoxedInput, assert_equal_dispatcher_barrier, new_input,
63};
64use crate::executor::monitor::ActorInputMetrics;
65use crate::executor::prelude::StreamingMetrics;
66use crate::executor::watermark::BufferedWatermarks;
67use crate::task::{ActorId, FragmentId, LocalBarrierManager};
68
69mod actor;
70mod barrier_align;
71pub mod exchange;
72pub mod monitor;
73
74pub mod aggregate;
75pub mod asof_join;
76mod backfill;
77mod barrier_recv;
78mod batch_query;
79mod chain;
80mod changelog;
81mod dedup;
82mod dispatch;
83pub mod dml;
84mod dynamic_filter;
85pub mod eowc;
86pub mod error;
87mod expand;
88mod filter;
89mod gap_fill;
90pub mod hash_join;
91mod hop_window;
92mod join;
93pub mod locality_provider;
94mod lookup;
95mod lookup_union;
96mod merge;
97mod mview;
98mod nested_loop_temporal_join;
99mod no_op;
100mod now;
101mod over_window;
102pub mod project;
103mod rearranged_chain;
104mod receiver;
105pub mod row_id_gen;
106mod sink;
107pub mod source;
108mod stream_reader;
109pub mod subtask;
110mod temporal_join;
111mod top_n;
112mod troublemaker;
113mod union;
114mod upstream_sink_union;
115mod values;
116mod watermark;
117mod watermark_filter;
118mod wrapper;
119
120mod approx_percentile;
121
122mod row_merge;
123
124#[cfg(test)]
125mod integration_tests;
126mod sync_kv_log_store;
127#[cfg(any(test, feature = "test"))]
128pub mod test_utils;
129mod utils;
130mod vector;
131
132pub use actor::{Actor, ActorContext, ActorContextRef};
133use anyhow::Context;
134pub use approx_percentile::global::GlobalApproxPercentileExecutor;
135pub use approx_percentile::local::LocalApproxPercentileExecutor;
136pub use backfill::arrangement_backfill::*;
137pub use backfill::cdc::{
138 CdcBackfillExecutor, ExternalStorageTable, ParallelizedCdcBackfillExecutor,
139};
140pub use backfill::no_shuffle_backfill::*;
141pub use backfill::snapshot_backfill::*;
142pub use barrier_recv::BarrierRecvExecutor;
143pub use batch_query::BatchQueryExecutor;
144pub use chain::ChainExecutor;
145pub use changelog::ChangeLogExecutor;
146pub use dedup::AppendOnlyDedupExecutor;
147pub use dispatch::{DispatchExecutor, DispatcherImpl};
148pub use dynamic_filter::DynamicFilterExecutor;
149pub use error::{StreamExecutorError, StreamExecutorResult};
150pub use expand::ExpandExecutor;
151pub use filter::{FilterExecutor, UpsertFilterExecutor};
152pub use gap_fill::{GapFillExecutor, GapFillExecutorArgs};
153pub use hash_join::*;
154pub use hop_window::HopWindowExecutor;
155pub use join::row::{CachedJoinRow, CpuEncoding, JoinEncoding, MemoryEncoding};
156pub use join::{AsOfDesc, AsOfJoinType, JoinType};
157pub use lookup::*;
158pub use lookup_union::LookupUnionExecutor;
159pub use merge::MergeExecutor;
160pub(crate) use merge::{MergeExecutorInput, MergeExecutorUpstream};
161pub use mview::{MaterializeExecutor, RefreshableMaterializeArgs};
162pub use nested_loop_temporal_join::NestedLoopTemporalJoinExecutor;
163pub use no_op::NoOpExecutor;
164pub use now::*;
165pub use over_window::*;
166pub use rearranged_chain::RearrangedChainExecutor;
167pub use receiver::ReceiverExecutor;
168use risingwave_common::id::SourceId;
169pub use row_merge::RowMergeExecutor;
170pub use sink::SinkExecutor;
171pub use sync_kv_log_store::SyncedKvLogStoreExecutor;
172pub use sync_kv_log_store::metrics::SyncedKvLogStoreMetrics;
173pub use temporal_join::TemporalJoinExecutor;
174pub use top_n::{
175 AppendOnlyGroupTopNExecutor, AppendOnlyTopNExecutor, GroupTopNExecutor, TopNExecutor,
176};
177pub use troublemaker::TroublemakerExecutor;
178pub use union::UnionExecutor;
179pub use upstream_sink_union::{UpstreamFragmentInfo, UpstreamSinkUnionExecutor};
180pub use utils::DummyExecutor;
181pub use values::ValuesExecutor;
182pub use vector::*;
183pub use watermark_filter::WatermarkFilterExecutor;
184pub use wrapper::WrapperExecutor;
185
186use self::barrier_align::AlignedMessageStream;
187
188pub type MessageStreamItemInner<M> = StreamExecutorResult<MessageInner<M>>;
189pub type MessageStreamItem = MessageStreamItemInner<BarrierMutationType>;
190pub type DispatcherMessageStreamItem = StreamExecutorResult<DispatcherMessage>;
191pub type BoxedMessageStream = BoxStream<'static, MessageStreamItem>;
192
193pub use risingwave_common::util::epoch::task_local::{curr_epoch, epoch, prev_epoch};
194use risingwave_connector::sink::catalog::SinkId;
195use risingwave_connector::source::cdc::{
196 CdcTableSnapshotSplitAssignmentWithGeneration,
197 build_actor_cdc_table_snapshot_splits_with_generation,
198};
199use risingwave_pb::id::SubscriberId;
200use risingwave_pb::stream_plan::stream_message_batch::{BarrierBatch, StreamMessageBatch};
201
202pub trait MessageStreamInner<M> = Stream<Item = MessageStreamItemInner<M>> + Send;
203pub trait MessageStream = Stream<Item = MessageStreamItem> + Send;
204pub trait DispatcherMessageStream = Stream<Item = DispatcherMessageStreamItem> + Send;
205
206#[derive(Debug, Default, Clone)]
208pub struct ExecutorInfo {
209 pub schema: Schema,
211
212 pub stream_key: StreamKey,
214
215 pub stream_kind: PbStreamKind,
217
218 pub identity: String,
220
221 pub id: u64,
223}
224
225impl ExecutorInfo {
226 pub fn for_test(schema: Schema, stream_key: StreamKey, identity: String, id: u64) -> Self {
227 Self {
228 schema,
229 stream_key,
230 stream_kind: PbStreamKind::Retract, identity,
232 id,
233 }
234 }
235}
236
237pub trait Execute: Send + 'static {
239 fn execute(self: Box<Self>) -> BoxedMessageStream;
240
241 fn execute_with_epoch(self: Box<Self>, _epoch: u64) -> BoxedMessageStream {
242 self.execute()
243 }
244
245 fn boxed(self) -> Box<dyn Execute>
246 where
247 Self: Sized + Send + 'static,
248 {
249 Box::new(self)
250 }
251}
252
253pub struct Executor {
256 info: ExecutorInfo,
257 execute: Box<dyn Execute>,
258}
259
260impl Executor {
261 pub fn new(info: ExecutorInfo, execute: Box<dyn Execute>) -> Self {
262 Self { info, execute }
263 }
264
265 pub fn info(&self) -> &ExecutorInfo {
266 &self.info
267 }
268
269 pub fn schema(&self) -> &Schema {
270 &self.info.schema
271 }
272
273 pub fn stream_key(&self) -> StreamKeyRef<'_> {
274 &self.info.stream_key
275 }
276
277 pub fn stream_kind(&self) -> PbStreamKind {
278 self.info.stream_kind
279 }
280
281 pub fn identity(&self) -> &str {
282 &self.info.identity
283 }
284
285 pub fn execute(self) -> BoxedMessageStream {
286 self.execute.execute()
287 }
288
289 pub fn execute_with_epoch(self, epoch: u64) -> BoxedMessageStream {
290 self.execute.execute_with_epoch(epoch)
291 }
292}
293
294impl std::fmt::Debug for Executor {
295 fn fmt(&self, f: &mut std::fmt::Formatter<'_>) -> std::fmt::Result {
296 f.write_str(self.identity())
297 }
298}
299
300impl From<(ExecutorInfo, Box<dyn Execute>)> for Executor {
301 fn from((info, execute): (ExecutorInfo, Box<dyn Execute>)) -> Self {
302 Self::new(info, execute)
303 }
304}
305
306impl<E> From<(ExecutorInfo, E)> for Executor
307where
308 E: Execute,
309{
310 fn from((info, execute): (ExecutorInfo, E)) -> Self {
311 Self::new(info, execute.boxed())
312 }
313}
314
315pub const INVALID_EPOCH: u64 = 0;
316
317type UpstreamFragmentId = FragmentId;
318type SplitAssignments = HashMap<ActorId, Vec<SplitImpl>>;
319
320#[derive(Debug, Clone, PartialEq)]
321#[cfg_attr(any(test, feature = "test"), derive(Default))]
322pub struct UpdateMutation {
323 pub dispatchers: HashMap<ActorId, Vec<DispatcherUpdate>>,
324 pub merges: HashMap<(ActorId, UpstreamFragmentId), MergeUpdate>,
325 pub vnode_bitmaps: HashMap<ActorId, Arc<Bitmap>>,
326 pub dropped_actors: HashSet<ActorId>,
327 pub actor_splits: SplitAssignments,
328 pub actor_new_dispatchers: HashMap<ActorId, Vec<PbDispatcher>>,
329 pub actor_cdc_table_snapshot_splits: CdcTableSnapshotSplitAssignmentWithGeneration,
330 pub sink_add_columns: HashMap<SinkId, Vec<Field>>,
331}
332
333#[derive(Debug, Clone, PartialEq)]
334#[cfg_attr(any(test, feature = "test"), derive(Default))]
335pub struct AddMutation {
336 pub adds: HashMap<ActorId, Vec<PbDispatcher>>,
337 pub added_actors: HashSet<ActorId>,
338 pub splits: SplitAssignments,
340 pub pause: bool,
341 pub subscriptions_to_add: Vec<(TableId, SubscriberId)>,
343 pub backfill_nodes_to_pause: HashSet<FragmentId>,
345 pub actor_cdc_table_snapshot_splits: CdcTableSnapshotSplitAssignmentWithGeneration,
346 pub new_upstream_sinks: HashMap<FragmentId, PbNewUpstreamSink>,
347}
348
349#[derive(Debug, Clone, PartialEq)]
350#[cfg_attr(any(test, feature = "test"), derive(Default))]
351pub struct StopMutation {
352 pub dropped_actors: HashSet<ActorId>,
353 pub dropped_sink_fragments: HashSet<FragmentId>,
354}
355
356#[derive(Debug, Clone, PartialEq)]
358pub enum Mutation {
359 Stop(StopMutation),
360 Update(UpdateMutation),
361 Add(AddMutation),
362 SourceChangeSplit(SplitAssignments),
363 Pause,
364 Resume,
365 Throttle(HashMap<ActorId, Option<u32>>),
366 ConnectorPropsChange(HashMap<u32, HashMap<String, String>>),
367 DropSubscriptions {
368 subscriptions_to_drop: Vec<(SubscriberId, TableId)>,
370 },
371 StartFragmentBackfill {
372 fragment_ids: HashSet<FragmentId>,
373 },
374 RefreshStart {
375 table_id: TableId,
376 associated_source_id: SourceId,
377 },
378 ListFinish {
379 associated_source_id: SourceId,
380 },
381 LoadFinish {
382 associated_source_id: SourceId,
383 },
384}
385
386#[derive(Debug, Clone)]
391pub struct BarrierInner<M> {
392 pub epoch: EpochPair,
393 pub mutation: M,
394 pub kind: BarrierKind,
395
396 pub tracing_context: TracingContext,
398}
399
400pub type BarrierMutationType = Option<Arc<Mutation>>;
401pub type Barrier = BarrierInner<BarrierMutationType>;
402pub type DispatcherBarrier = BarrierInner<()>;
403
404impl<M: Default> BarrierInner<M> {
405 pub fn new_test_barrier(epoch: u64) -> Self {
407 Self {
408 epoch: EpochPair::new_test_epoch(epoch),
409 kind: BarrierKind::Checkpoint,
410 tracing_context: TracingContext::none(),
411 mutation: Default::default(),
412 }
413 }
414
415 pub fn with_prev_epoch_for_test(epoch: u64, prev_epoch: u64) -> Self {
416 Self {
417 epoch: EpochPair::new(epoch, prev_epoch),
418 kind: BarrierKind::Checkpoint,
419 tracing_context: TracingContext::none(),
420 mutation: Default::default(),
421 }
422 }
423}
424
425impl Barrier {
426 pub fn into_dispatcher(self) -> DispatcherBarrier {
427 DispatcherBarrier {
428 epoch: self.epoch,
429 mutation: (),
430 kind: self.kind,
431 tracing_context: self.tracing_context,
432 }
433 }
434
435 #[must_use]
436 pub fn with_mutation(self, mutation: Mutation) -> Self {
437 Self {
438 mutation: Some(Arc::new(mutation)),
439 ..self
440 }
441 }
442
443 #[must_use]
444 pub fn with_stop(self) -> Self {
445 self.with_mutation(Mutation::Stop(StopMutation {
446 dropped_actors: Default::default(),
447 dropped_sink_fragments: Default::default(),
448 }))
449 }
450
451 pub fn is_with_stop_mutation(&self) -> bool {
453 matches!(self.mutation.as_deref(), Some(Mutation::Stop(_)))
454 }
455
456 pub fn is_stop(&self, actor_id: ActorId) -> bool {
458 self.all_stop_actors()
459 .is_some_and(|actors| actors.contains(&actor_id))
460 }
461
462 pub fn is_checkpoint(&self) -> bool {
463 self.kind == BarrierKind::Checkpoint
464 }
465
466 pub fn initial_split_assignment(&self, actor_id: ActorId) -> Option<&[SplitImpl]> {
476 match self.mutation.as_deref()? {
477 Mutation::Update(UpdateMutation { actor_splits, .. })
478 | Mutation::Add(AddMutation {
479 splits: actor_splits,
480 ..
481 }) => actor_splits.get(&actor_id),
482
483 _ => {
484 if cfg!(debug_assertions) {
485 panic!(
486 "the initial mutation of the barrier should not be {:?}",
487 self.mutation
488 );
489 }
490 None
491 }
492 }
493 .map(|s| s.as_slice())
494 }
495
496 pub fn all_stop_actors(&self) -> Option<&HashSet<ActorId>> {
498 match self.mutation.as_deref() {
499 Some(Mutation::Stop(StopMutation { dropped_actors, .. })) => Some(dropped_actors),
500 Some(Mutation::Update(UpdateMutation { dropped_actors, .. })) => Some(dropped_actors),
501 _ => None,
502 }
503 }
504
505 pub fn is_newly_added(&self, actor_id: ActorId) -> bool {
511 match self.mutation.as_deref() {
512 Some(Mutation::Add(AddMutation { added_actors, .. })) => {
513 added_actors.contains(&actor_id)
514 }
515 _ => false,
516 }
517 }
518
519 pub fn should_start_fragment_backfill(&self, fragment_id: FragmentId) -> bool {
520 if let Some(Mutation::StartFragmentBackfill { fragment_ids }) = self.mutation.as_deref() {
521 fragment_ids.contains(&fragment_id)
522 } else {
523 false
524 }
525 }
526
527 pub fn has_more_downstream_fragments(&self, upstream_actor_id: ActorId) -> bool {
545 let Some(mutation) = self.mutation.as_deref() else {
546 return false;
547 };
548 match mutation {
549 Mutation::Add(AddMutation { adds, .. }) => adds.get(&upstream_actor_id).is_some(),
551 Mutation::Update(_)
552 | Mutation::Stop(_)
553 | Mutation::Pause
554 | Mutation::Resume
555 | Mutation::SourceChangeSplit(_)
556 | Mutation::Throttle(_)
557 | Mutation::DropSubscriptions { .. }
558 | Mutation::ConnectorPropsChange(_)
559 | Mutation::StartFragmentBackfill { .. }
560 | Mutation::RefreshStart { .. }
561 | Mutation::ListFinish { .. }
562 | Mutation::LoadFinish { .. } => false,
563 }
564 }
565
566 pub fn is_pause_on_startup(&self) -> bool {
568 match self.mutation.as_deref() {
569 Some(Mutation::Add(AddMutation { pause, .. })) => *pause,
570 _ => false,
571 }
572 }
573
574 pub fn is_backfill_pause_on_startup(&self, backfill_fragment_id: FragmentId) -> bool {
575 match self.mutation.as_deref() {
576 Some(Mutation::Add(AddMutation {
577 backfill_nodes_to_pause,
578 ..
579 })) => backfill_nodes_to_pause.contains(&backfill_fragment_id),
580 Some(Mutation::Update(_)) => false,
581 _ => {
582 tracing::warn!(
583 "expected an AddMutation or UpdateMutation on Startup, instead got {:?}",
584 self
585 );
586 false
587 }
588 }
589 }
590
591 pub fn is_resume(&self) -> bool {
593 matches!(self.mutation.as_deref(), Some(Mutation::Resume))
594 }
595
596 pub fn as_update_merge(
599 &self,
600 actor_id: ActorId,
601 upstream_fragment_id: UpstreamFragmentId,
602 ) -> Option<&MergeUpdate> {
603 self.mutation
604 .as_deref()
605 .and_then(|mutation| match mutation {
606 Mutation::Update(UpdateMutation { merges, .. }) => {
607 merges.get(&(actor_id, upstream_fragment_id))
608 }
609 _ => None,
610 })
611 }
612
613 pub fn as_new_upstream_sink(&self, fragment_id: FragmentId) -> Option<&PbNewUpstreamSink> {
616 self.mutation
617 .as_deref()
618 .and_then(|mutation| match mutation {
619 Mutation::Add(AddMutation {
620 new_upstream_sinks, ..
621 }) => new_upstream_sinks.get(&fragment_id),
622 _ => None,
623 })
624 }
625
626 pub fn as_dropped_upstream_sinks(&self) -> Option<&HashSet<FragmentId>> {
628 self.mutation
629 .as_deref()
630 .and_then(|mutation| match mutation {
631 Mutation::Stop(StopMutation {
632 dropped_sink_fragments,
633 ..
634 }) => Some(dropped_sink_fragments),
635 _ => None,
636 })
637 }
638
639 pub fn as_update_vnode_bitmap(&self, actor_id: ActorId) -> Option<Arc<Bitmap>> {
645 self.mutation
646 .as_deref()
647 .and_then(|mutation| match mutation {
648 Mutation::Update(UpdateMutation { vnode_bitmaps, .. }) => {
649 vnode_bitmaps.get(&actor_id).cloned()
650 }
651 _ => None,
652 })
653 }
654
655 pub fn as_sink_add_columns(&self, sink_id: SinkId) -> Option<Vec<Field>> {
656 self.mutation
657 .as_deref()
658 .and_then(|mutation| match mutation {
659 Mutation::Update(UpdateMutation {
660 sink_add_columns, ..
661 }) => sink_add_columns.get(&sink_id).cloned(),
662 _ => None,
663 })
664 }
665
666 pub fn get_curr_epoch(&self) -> Epoch {
667 Epoch(self.epoch.curr)
668 }
669
670 pub fn tracing_context(&self) -> &TracingContext {
672 &self.tracing_context
673 }
674
675 pub fn added_subscriber_on_mv_table(
676 &self,
677 mv_table_id: TableId,
678 ) -> impl Iterator<Item = SubscriberId> + '_ {
679 if let Some(Mutation::Add(add)) = self.mutation.as_deref() {
680 Some(add)
681 } else {
682 None
683 }
684 .into_iter()
685 .flat_map(move |add| {
686 add.subscriptions_to_add.iter().filter_map(
687 move |(upstream_mv_table_id, subscriber_id)| {
688 if *upstream_mv_table_id == mv_table_id {
689 Some(*subscriber_id)
690 } else {
691 None
692 }
693 },
694 )
695 })
696 }
697}
698
699impl<M: PartialEq> PartialEq for BarrierInner<M> {
700 fn eq(&self, other: &Self) -> bool {
701 self.epoch == other.epoch && self.mutation == other.mutation
702 }
703}
704
705impl Mutation {
706 #[cfg(test)]
710 pub fn is_stop(&self) -> bool {
711 matches!(self, Mutation::Stop(_))
712 }
713
714 #[cfg(test)]
715 fn to_protobuf(&self) -> PbMutation {
716 use risingwave_connector::source::cdc::build_pb_actor_cdc_table_snapshot_splits_with_generation;
717 use risingwave_pb::source::{ConnectorSplit, ConnectorSplits};
718 use risingwave_pb::stream_plan::connector_props_change_mutation::ConnectorPropsInfo;
719 use risingwave_pb::stream_plan::throttle_mutation::RateLimit;
720 use risingwave_pb::stream_plan::{
721 PbAddMutation, PbConnectorPropsChangeMutation, PbDispatchers,
722 PbDropSubscriptionsMutation, PbPauseMutation, PbResumeMutation, PbSinkAddColumns,
723 PbSourceChangeSplitMutation, PbStartFragmentBackfillMutation, PbStopMutation,
724 PbThrottleMutation, PbUpdateMutation,
725 };
726 let actor_splits_to_protobuf = |actor_splits: &SplitAssignments| {
727 actor_splits
728 .iter()
729 .map(|(&actor_id, splits)| {
730 (
731 actor_id,
732 ConnectorSplits {
733 splits: splits.clone().iter().map(ConnectorSplit::from).collect(),
734 },
735 )
736 })
737 .collect::<HashMap<_, _>>()
738 };
739
740 match self {
741 Mutation::Stop(StopMutation {
742 dropped_actors,
743 dropped_sink_fragments,
744 }) => PbMutation::Stop(PbStopMutation {
745 actors: dropped_actors.iter().copied().collect(),
746 dropped_sink_fragments: dropped_sink_fragments.iter().copied().collect(),
747 }),
748 Mutation::Update(UpdateMutation {
749 dispatchers,
750 merges,
751 vnode_bitmaps,
752 dropped_actors,
753 actor_splits,
754 actor_new_dispatchers,
755 actor_cdc_table_snapshot_splits,
756 sink_add_columns,
757 }) => PbMutation::Update(PbUpdateMutation {
758 dispatcher_update: dispatchers.values().flatten().cloned().collect(),
759 merge_update: merges.values().cloned().collect(),
760 actor_vnode_bitmap_update: vnode_bitmaps
761 .iter()
762 .map(|(&actor_id, bitmap)| (actor_id, bitmap.to_protobuf()))
763 .collect(),
764 dropped_actors: dropped_actors.iter().copied().collect(),
765 actor_splits: actor_splits_to_protobuf(actor_splits),
766 actor_new_dispatchers: actor_new_dispatchers
767 .iter()
768 .map(|(&actor_id, dispatchers)| {
769 (
770 actor_id,
771 PbDispatchers {
772 dispatchers: dispatchers.clone(),
773 },
774 )
775 })
776 .collect(),
777 actor_cdc_table_snapshot_splits:
778 build_pb_actor_cdc_table_snapshot_splits_with_generation(
779 actor_cdc_table_snapshot_splits.clone(),
780 )
781 .into(),
782 sink_add_columns: sink_add_columns
783 .iter()
784 .map(|(sink_id, add_columns)| {
785 (
786 *sink_id,
787 PbSinkAddColumns {
788 fields: add_columns.iter().map(|field| field.to_prost()).collect(),
789 },
790 )
791 })
792 .collect(),
793 }),
794 Mutation::Add(AddMutation {
795 adds,
796 added_actors,
797 splits,
798 pause,
799 subscriptions_to_add,
800 backfill_nodes_to_pause,
801 actor_cdc_table_snapshot_splits,
802 new_upstream_sinks,
803 }) => PbMutation::Add(PbAddMutation {
804 actor_dispatchers: adds
805 .iter()
806 .map(|(&actor_id, dispatchers)| {
807 (
808 actor_id,
809 PbDispatchers {
810 dispatchers: dispatchers.clone(),
811 },
812 )
813 })
814 .collect(),
815 added_actors: added_actors.iter().copied().collect(),
816 actor_splits: actor_splits_to_protobuf(splits),
817 pause: *pause,
818 subscriptions_to_add: subscriptions_to_add
819 .iter()
820 .map(|(table_id, subscriber_id)| SubscriptionUpstreamInfo {
821 subscriber_id: *subscriber_id,
822 upstream_mv_table_id: *table_id,
823 })
824 .collect(),
825 backfill_nodes_to_pause: backfill_nodes_to_pause.iter().copied().collect(),
826 actor_cdc_table_snapshot_splits:
827 build_pb_actor_cdc_table_snapshot_splits_with_generation(
828 actor_cdc_table_snapshot_splits.clone(),
829 )
830 .into(),
831 new_upstream_sinks: new_upstream_sinks
832 .iter()
833 .map(|(k, v)| (*k, v.clone()))
834 .collect(),
835 }),
836 Mutation::SourceChangeSplit(changes) => {
837 PbMutation::Splits(PbSourceChangeSplitMutation {
838 actor_splits: changes
839 .iter()
840 .map(|(&actor_id, splits)| {
841 (
842 actor_id,
843 ConnectorSplits {
844 splits: splits
845 .clone()
846 .iter()
847 .map(ConnectorSplit::from)
848 .collect(),
849 },
850 )
851 })
852 .collect(),
853 })
854 }
855 Mutation::Pause => PbMutation::Pause(PbPauseMutation {}),
856 Mutation::Resume => PbMutation::Resume(PbResumeMutation {}),
857 Mutation::Throttle(changes) => PbMutation::Throttle(PbThrottleMutation {
858 actor_throttle: changes
859 .iter()
860 .map(|(actor_id, limit)| (*actor_id, RateLimit { rate_limit: *limit }))
861 .collect(),
862 }),
863 Mutation::DropSubscriptions {
864 subscriptions_to_drop,
865 } => PbMutation::DropSubscriptions(PbDropSubscriptionsMutation {
866 info: subscriptions_to_drop
867 .iter()
868 .map(
869 |(subscriber_id, upstream_mv_table_id)| SubscriptionUpstreamInfo {
870 subscriber_id: *subscriber_id,
871 upstream_mv_table_id: *upstream_mv_table_id,
872 },
873 )
874 .collect(),
875 }),
876 Mutation::ConnectorPropsChange(map) => {
877 PbMutation::ConnectorPropsChange(PbConnectorPropsChangeMutation {
878 connector_props_infos: map
879 .iter()
880 .map(|(actor_id, options)| {
881 (
882 *actor_id,
883 ConnectorPropsInfo {
884 connector_props_info: options
885 .iter()
886 .map(|(k, v)| (k.clone(), v.clone()))
887 .collect(),
888 },
889 )
890 })
891 .collect(),
892 })
893 }
894 Mutation::StartFragmentBackfill { fragment_ids } => {
895 PbMutation::StartFragmentBackfill(PbStartFragmentBackfillMutation {
896 fragment_ids: fragment_ids.iter().copied().collect(),
897 })
898 }
899 Mutation::RefreshStart {
900 table_id,
901 associated_source_id,
902 } => PbMutation::RefreshStart(risingwave_pb::stream_plan::RefreshStartMutation {
903 table_id: *table_id,
904 associated_source_id: *associated_source_id,
905 }),
906 Mutation::ListFinish {
907 associated_source_id,
908 } => PbMutation::ListFinish(risingwave_pb::stream_plan::ListFinishMutation {
909 associated_source_id: *associated_source_id,
910 }),
911 Mutation::LoadFinish {
912 associated_source_id,
913 } => PbMutation::LoadFinish(risingwave_pb::stream_plan::LoadFinishMutation {
914 associated_source_id: *associated_source_id,
915 }),
916 }
917 }
918
919 fn from_protobuf(prost: &PbMutation) -> StreamExecutorResult<Self> {
920 let mutation = match prost {
921 PbMutation::Stop(stop) => Mutation::Stop(StopMutation {
922 dropped_actors: stop.actors.iter().copied().collect(),
923 dropped_sink_fragments: stop.dropped_sink_fragments.iter().copied().collect(),
924 }),
925
926 PbMutation::Update(update) => Mutation::Update(UpdateMutation {
927 dispatchers: update
928 .dispatcher_update
929 .iter()
930 .map(|u| (u.actor_id, u.clone()))
931 .into_group_map(),
932 merges: update
933 .merge_update
934 .iter()
935 .map(|u| ((u.actor_id, u.upstream_fragment_id), u.clone()))
936 .collect(),
937 vnode_bitmaps: update
938 .actor_vnode_bitmap_update
939 .iter()
940 .map(|(&actor_id, bitmap)| (actor_id, Arc::new(bitmap.into())))
941 .collect(),
942 dropped_actors: update.dropped_actors.iter().copied().collect(),
943 actor_splits: update
944 .actor_splits
945 .iter()
946 .map(|(&actor_id, splits)| {
947 (
948 actor_id,
949 splits
950 .splits
951 .iter()
952 .map(|split| split.try_into().unwrap())
953 .collect(),
954 )
955 })
956 .collect(),
957 actor_new_dispatchers: update
958 .actor_new_dispatchers
959 .iter()
960 .map(|(&actor_id, dispatchers)| (actor_id, dispatchers.dispatchers.clone()))
961 .collect(),
962 actor_cdc_table_snapshot_splits:
963 build_actor_cdc_table_snapshot_splits_with_generation(
964 update
965 .actor_cdc_table_snapshot_splits
966 .clone()
967 .unwrap_or_default(),
968 ),
969 sink_add_columns: update
970 .sink_add_columns
971 .iter()
972 .map(|(sink_id, add_columns)| {
973 (
974 *sink_id,
975 add_columns.fields.iter().map(Field::from_prost).collect(),
976 )
977 })
978 .collect(),
979 }),
980
981 PbMutation::Add(add) => Mutation::Add(AddMutation {
982 adds: add
983 .actor_dispatchers
984 .iter()
985 .map(|(&actor_id, dispatchers)| (actor_id, dispatchers.dispatchers.clone()))
986 .collect(),
987 added_actors: add.added_actors.iter().copied().collect(),
988 splits: add
991 .actor_splits
992 .iter()
993 .map(|(&actor_id, splits)| {
994 (
995 actor_id,
996 splits
997 .splits
998 .iter()
999 .map(|split| split.try_into().unwrap())
1000 .collect(),
1001 )
1002 })
1003 .collect(),
1004 pause: add.pause,
1005 subscriptions_to_add: add
1006 .subscriptions_to_add
1007 .iter()
1008 .map(
1009 |SubscriptionUpstreamInfo {
1010 subscriber_id,
1011 upstream_mv_table_id,
1012 }| { (*upstream_mv_table_id, *subscriber_id) },
1013 )
1014 .collect(),
1015 backfill_nodes_to_pause: add.backfill_nodes_to_pause.iter().copied().collect(),
1016 actor_cdc_table_snapshot_splits:
1017 build_actor_cdc_table_snapshot_splits_with_generation(
1018 add.actor_cdc_table_snapshot_splits
1019 .clone()
1020 .unwrap_or_default(),
1021 ),
1022 new_upstream_sinks: add
1023 .new_upstream_sinks
1024 .iter()
1025 .map(|(k, v)| (*k, v.clone()))
1026 .collect(),
1027 }),
1028
1029 PbMutation::Splits(s) => {
1030 let mut change_splits: Vec<(ActorId, Vec<SplitImpl>)> =
1031 Vec::with_capacity(s.actor_splits.len());
1032 for (&actor_id, splits) in &s.actor_splits {
1033 if !splits.splits.is_empty() {
1034 change_splits.push((
1035 actor_id,
1036 splits
1037 .splits
1038 .iter()
1039 .map(SplitImpl::try_from)
1040 .try_collect()?,
1041 ));
1042 }
1043 }
1044 Mutation::SourceChangeSplit(change_splits.into_iter().collect())
1045 }
1046 PbMutation::Pause(_) => Mutation::Pause,
1047 PbMutation::Resume(_) => Mutation::Resume,
1048 PbMutation::Throttle(changes) => Mutation::Throttle(
1049 changes
1050 .actor_throttle
1051 .iter()
1052 .map(|(actor_id, limit)| (*actor_id, limit.rate_limit))
1053 .collect(),
1054 ),
1055 PbMutation::DropSubscriptions(drop) => Mutation::DropSubscriptions {
1056 subscriptions_to_drop: drop
1057 .info
1058 .iter()
1059 .map(|info| (info.subscriber_id, info.upstream_mv_table_id))
1060 .collect(),
1061 },
1062 PbMutation::ConnectorPropsChange(alter_connector_props) => {
1063 Mutation::ConnectorPropsChange(
1064 alter_connector_props
1065 .connector_props_infos
1066 .iter()
1067 .map(|(connector_id, options)| {
1068 (
1069 *connector_id,
1070 options
1071 .connector_props_info
1072 .iter()
1073 .map(|(k, v)| (k.clone(), v.clone()))
1074 .collect(),
1075 )
1076 })
1077 .collect(),
1078 )
1079 }
1080 PbMutation::StartFragmentBackfill(start_fragment_backfill) => {
1081 Mutation::StartFragmentBackfill {
1082 fragment_ids: start_fragment_backfill
1083 .fragment_ids
1084 .iter()
1085 .copied()
1086 .collect(),
1087 }
1088 }
1089 PbMutation::RefreshStart(refresh_start) => Mutation::RefreshStart {
1090 table_id: refresh_start.table_id,
1091 associated_source_id: refresh_start.associated_source_id,
1092 },
1093 PbMutation::ListFinish(list_finish) => Mutation::ListFinish {
1094 associated_source_id: list_finish.associated_source_id,
1095 },
1096 PbMutation::LoadFinish(load_finish) => Mutation::LoadFinish {
1097 associated_source_id: load_finish.associated_source_id,
1098 },
1099 };
1100 Ok(mutation)
1101 }
1102}
1103
1104impl<M> BarrierInner<M> {
1105 fn to_protobuf_inner(&self, barrier_fn: impl FnOnce(&M) -> Option<PbMutation>) -> PbBarrier {
1106 let Self {
1107 epoch,
1108 mutation,
1109 kind,
1110 tracing_context,
1111 ..
1112 } = self;
1113
1114 PbBarrier {
1115 epoch: Some(PbEpoch {
1116 curr: epoch.curr,
1117 prev: epoch.prev,
1118 }),
1119 mutation: barrier_fn(mutation).map(|mutation| PbBarrierMutation {
1120 mutation: Some(mutation),
1121 }),
1122 tracing_context: tracing_context.to_protobuf(),
1123 kind: *kind as _,
1124 }
1125 }
1126
1127 fn from_protobuf_inner(
1128 prost: &PbBarrier,
1129 mutation_from_pb: impl FnOnce(Option<&PbMutation>) -> StreamExecutorResult<M>,
1130 ) -> StreamExecutorResult<Self> {
1131 let epoch = prost.get_epoch()?;
1132
1133 Ok(Self {
1134 kind: prost.kind(),
1135 epoch: EpochPair::new(epoch.curr, epoch.prev),
1136 mutation: mutation_from_pb(
1137 (prost.mutation.as_ref()).and_then(|mutation| mutation.mutation.as_ref()),
1138 )?,
1139 tracing_context: TracingContext::from_protobuf(&prost.tracing_context),
1140 })
1141 }
1142
1143 pub fn map_mutation<M2>(self, f: impl FnOnce(M) -> M2) -> BarrierInner<M2> {
1144 BarrierInner {
1145 epoch: self.epoch,
1146 mutation: f(self.mutation),
1147 kind: self.kind,
1148 tracing_context: self.tracing_context,
1149 }
1150 }
1151}
1152
1153impl DispatcherBarrier {
1154 pub fn to_protobuf(&self) -> PbBarrier {
1155 self.to_protobuf_inner(|_| None)
1156 }
1157}
1158
1159impl Barrier {
1160 #[cfg(test)]
1161 pub fn to_protobuf(&self) -> PbBarrier {
1162 self.to_protobuf_inner(|mutation| mutation.as_ref().map(|mutation| mutation.to_protobuf()))
1163 }
1164
1165 pub fn from_protobuf(prost: &PbBarrier) -> StreamExecutorResult<Self> {
1166 Self::from_protobuf_inner(prost, |mutation| {
1167 mutation
1168 .map(|m| Mutation::from_protobuf(m).map(Arc::new))
1169 .transpose()
1170 })
1171 }
1172}
1173
1174#[derive(Debug, PartialEq, Eq, Clone)]
1175pub struct Watermark {
1176 pub col_idx: usize,
1177 pub data_type: DataType,
1178 pub val: ScalarImpl,
1179}
1180
1181impl PartialOrd for Watermark {
1182 fn partial_cmp(&self, other: &Self) -> Option<std::cmp::Ordering> {
1183 Some(self.cmp(other))
1184 }
1185}
1186
1187impl Ord for Watermark {
1188 fn cmp(&self, other: &Self) -> std::cmp::Ordering {
1189 self.val.default_cmp(&other.val)
1190 }
1191}
1192
1193impl Watermark {
1194 pub fn new(col_idx: usize, data_type: DataType, val: ScalarImpl) -> Self {
1195 Self {
1196 col_idx,
1197 data_type,
1198 val,
1199 }
1200 }
1201
1202 pub async fn transform_with_expr(
1203 self,
1204 expr: &NonStrictExpression<impl Expression>,
1205 new_col_idx: usize,
1206 ) -> Option<Self> {
1207 let Self { col_idx, val, .. } = self;
1208 let row = {
1209 let mut row = vec![None; col_idx + 1];
1210 row[col_idx] = Some(val);
1211 OwnedRow::new(row)
1212 };
1213 let val = expr.eval_row_infallible(&row).await?;
1214 Some(Self::new(new_col_idx, expr.inner().return_type(), val))
1215 }
1216
1217 pub fn transform_with_indices(self, output_indices: &[usize]) -> Option<Self> {
1220 output_indices
1221 .iter()
1222 .position(|p| *p == self.col_idx)
1223 .map(|new_col_idx| self.with_idx(new_col_idx))
1224 }
1225
1226 pub fn to_protobuf(&self) -> PbWatermark {
1227 PbWatermark {
1228 column: Some(PbInputRef {
1229 index: self.col_idx as _,
1230 r#type: Some(self.data_type.to_protobuf()),
1231 }),
1232 val: Some(&self.val).to_protobuf().into(),
1233 }
1234 }
1235
1236 pub fn from_protobuf(prost: &PbWatermark) -> StreamExecutorResult<Self> {
1237 let col_ref = prost.get_column()?;
1238 let data_type = DataType::from(col_ref.get_type()?);
1239 let val = Datum::from_protobuf(prost.get_val()?, &data_type)?
1240 .expect("watermark value cannot be null");
1241 Ok(Self::new(col_ref.get_index() as _, data_type, val))
1242 }
1243
1244 pub fn with_idx(self, idx: usize) -> Self {
1245 Self::new(idx, self.data_type, self.val)
1246 }
1247}
1248
1249#[derive(Debug, EnumAsInner, PartialEq, Clone)]
1250pub enum MessageInner<M> {
1251 Chunk(StreamChunk),
1252 Barrier(BarrierInner<M>),
1253 Watermark(Watermark),
1254}
1255
1256impl<M> MessageInner<M> {
1257 pub fn map_mutation<M2>(self, f: impl FnOnce(M) -> M2) -> MessageInner<M2> {
1258 match self {
1259 MessageInner::Chunk(chunk) => MessageInner::Chunk(chunk),
1260 MessageInner::Barrier(barrier) => MessageInner::Barrier(barrier.map_mutation(f)),
1261 MessageInner::Watermark(watermark) => MessageInner::Watermark(watermark),
1262 }
1263 }
1264}
1265
1266pub type Message = MessageInner<BarrierMutationType>;
1267pub type DispatcherMessage = MessageInner<()>;
1268
1269#[derive(Debug, EnumAsInner, PartialEq, Clone)]
1272pub enum MessageBatchInner<M> {
1273 Chunk(StreamChunk),
1274 BarrierBatch(Vec<BarrierInner<M>>),
1275 Watermark(Watermark),
1276}
1277pub type MessageBatch = MessageBatchInner<BarrierMutationType>;
1278pub type DispatcherBarriers = Vec<DispatcherBarrier>;
1279pub type DispatcherMessageBatch = MessageBatchInner<()>;
1280
1281impl From<DispatcherMessage> for DispatcherMessageBatch {
1282 fn from(m: DispatcherMessage) -> Self {
1283 match m {
1284 DispatcherMessage::Chunk(c) => Self::Chunk(c),
1285 DispatcherMessage::Barrier(b) => Self::BarrierBatch(vec![b]),
1286 DispatcherMessage::Watermark(w) => Self::Watermark(w),
1287 }
1288 }
1289}
1290
1291impl From<StreamChunk> for Message {
1292 fn from(chunk: StreamChunk) -> Self {
1293 Message::Chunk(chunk)
1294 }
1295}
1296
1297impl<'a> TryFrom<&'a Message> for &'a Barrier {
1298 type Error = ();
1299
1300 fn try_from(m: &'a Message) -> std::result::Result<Self, Self::Error> {
1301 match m {
1302 Message::Chunk(_) => Err(()),
1303 Message::Barrier(b) => Ok(b),
1304 Message::Watermark(_) => Err(()),
1305 }
1306 }
1307}
1308
1309impl Message {
1310 #[cfg(test)]
1315 pub fn is_stop(&self) -> bool {
1316 matches!(
1317 self,
1318 Message::Barrier(Barrier {
1319 mutation,
1320 ..
1321 }) if mutation.as_ref().unwrap().is_stop()
1322 )
1323 }
1324}
1325
1326impl DispatcherMessageBatch {
1327 pub fn to_protobuf(&self) -> PbStreamMessageBatch {
1328 let prost = match self {
1329 Self::Chunk(stream_chunk) => {
1330 let prost_stream_chunk = stream_chunk.to_protobuf();
1331 StreamMessageBatch::StreamChunk(prost_stream_chunk)
1332 }
1333 Self::BarrierBatch(barrier_batch) => StreamMessageBatch::BarrierBatch(BarrierBatch {
1334 barriers: barrier_batch.iter().map(|b| b.to_protobuf()).collect(),
1335 }),
1336 Self::Watermark(watermark) => StreamMessageBatch::Watermark(watermark.to_protobuf()),
1337 };
1338 PbStreamMessageBatch {
1339 stream_message_batch: Some(prost),
1340 }
1341 }
1342
1343 pub fn from_protobuf(prost: &PbStreamMessageBatch) -> StreamExecutorResult<Self> {
1344 let res = match prost.get_stream_message_batch()? {
1345 StreamMessageBatch::StreamChunk(chunk) => {
1346 Self::Chunk(StreamChunk::from_protobuf(chunk)?)
1347 }
1348 StreamMessageBatch::BarrierBatch(barrier_batch) => {
1349 let barriers = barrier_batch
1350 .barriers
1351 .iter()
1352 .map(|barrier| {
1353 DispatcherBarrier::from_protobuf_inner(barrier, |mutation| {
1354 if mutation.is_some() {
1355 if cfg!(debug_assertions) {
1356 panic!("should not receive message of barrier with mutation");
1357 } else {
1358 warn!(?barrier, "receive message of barrier with mutation");
1359 }
1360 }
1361 Ok(())
1362 })
1363 })
1364 .try_collect()?;
1365 Self::BarrierBatch(barriers)
1366 }
1367 StreamMessageBatch::Watermark(watermark) => {
1368 Self::Watermark(Watermark::from_protobuf(watermark)?)
1369 }
1370 };
1371 Ok(res)
1372 }
1373
1374 pub fn get_encoded_len(msg: &impl ::prost::Message) -> usize {
1375 ::prost::Message::encoded_len(msg)
1376 }
1377}
1378
1379pub type StreamKey = Vec<usize>;
1380pub type StreamKeyRef<'a> = &'a [usize];
1381pub type StreamKeyDataTypes = SmallVec<[DataType; 1]>;
1382
1383pub async fn expect_first_barrier<M: Debug>(
1385 stream: &mut (impl MessageStreamInner<M> + Unpin),
1386) -> StreamExecutorResult<BarrierInner<M>> {
1387 let message = stream
1388 .next()
1389 .instrument_await("expect_first_barrier")
1390 .await
1391 .context("failed to extract the first message: stream closed unexpectedly")??;
1392 let barrier = message
1393 .into_barrier()
1394 .expect("the first message must be a barrier");
1395 assert!(matches!(
1397 barrier.kind,
1398 BarrierKind::Checkpoint | BarrierKind::Initial
1399 ));
1400 Ok(barrier)
1401}
1402
1403pub async fn expect_first_barrier_from_aligned_stream(
1405 stream: &mut (impl AlignedMessageStream + Unpin),
1406) -> StreamExecutorResult<Barrier> {
1407 let message = stream
1408 .next()
1409 .instrument_await("expect_first_barrier")
1410 .await
1411 .context("failed to extract the first message: stream closed unexpectedly")??;
1412 let barrier = message
1413 .into_barrier()
1414 .expect("the first message must be a barrier");
1415 Ok(barrier)
1416}
1417
1418pub trait StreamConsumer: Send + 'static {
1420 type BarrierStream: Stream<Item = StreamResult<Barrier>> + Send;
1421
1422 fn execute(self: Box<Self>) -> Self::BarrierStream;
1423}
1424
1425type BoxedMessageInput<InputId, M> = BoxedInput<InputId, MessageStreamItemInner<M>>;
1426
1427pub struct DynamicReceivers<InputId, M> {
1431 barrier: Option<BarrierInner<M>>,
1433 start_ts: Option<Instant>,
1435 blocked: Vec<BoxedMessageInput<InputId, M>>,
1437 active: FuturesUnordered<StreamFuture<BoxedMessageInput<InputId, M>>>,
1439 buffered_watermarks: BTreeMap<usize, BufferedWatermarks<InputId>>,
1441 barrier_align_duration: Option<LabelGuardedMetric<GenericCounter<AtomicU64>>>,
1443 merge_barrier_align_duration: Option<LabelGuardedMetric<Histogram>>,
1445}
1446
1447impl<InputId: Clone + Ord + Hash + std::fmt::Debug + Unpin, M: Clone + Unpin> Stream
1448 for DynamicReceivers<InputId, M>
1449{
1450 type Item = MessageStreamItemInner<M>;
1451
1452 fn poll_next(
1453 mut self: Pin<&mut Self>,
1454 cx: &mut std::task::Context<'_>,
1455 ) -> Poll<Option<Self::Item>> {
1456 if self.is_empty() {
1457 return Poll::Ready(None);
1458 }
1459
1460 loop {
1461 match futures::ready!(self.active.poll_next_unpin(cx)) {
1462 Some((Some(Err(e)), _)) => {
1464 return Poll::Ready(Some(Err(e)));
1465 }
1466 Some((Some(Ok(message)), remaining)) => {
1468 let input_id = remaining.id();
1469 match message {
1470 MessageInner::Chunk(chunk) => {
1471 self.active.push(remaining.into_future());
1473 return Poll::Ready(Some(Ok(MessageInner::Chunk(chunk))));
1474 }
1475 MessageInner::Watermark(watermark) => {
1476 self.active.push(remaining.into_future());
1478 if let Some(watermark) = self.handle_watermark(input_id, watermark) {
1479 return Poll::Ready(Some(Ok(MessageInner::Watermark(watermark))));
1480 }
1481 }
1482 MessageInner::Barrier(barrier) => {
1483 if self.blocked.is_empty() {
1485 self.start_ts = Some(Instant::now());
1486 }
1487 self.blocked.push(remaining);
1488 if let Some(current_barrier) = self.barrier.as_ref() {
1489 if current_barrier.epoch != barrier.epoch {
1490 return Poll::Ready(Some(Err(
1491 StreamExecutorError::align_barrier(
1492 current_barrier.clone().map_mutation(|_| None),
1493 barrier.map_mutation(|_| None),
1494 ),
1495 )));
1496 }
1497 } else {
1498 self.barrier = Some(barrier);
1499 }
1500 }
1501 }
1502 }
1503 Some((None, remaining)) => {
1511 return Poll::Ready(Some(Err(StreamExecutorError::channel_closed(format!(
1512 "upstream input {:?} unexpectedly closed",
1513 remaining.id()
1514 )))));
1515 }
1516 None => {
1518 assert!(!self.blocked.is_empty());
1519
1520 let start_ts = self
1521 .start_ts
1522 .take()
1523 .expect("should have received at least one barrier");
1524 if let Some(barrier_align_duration) = &self.barrier_align_duration {
1525 barrier_align_duration.inc_by(start_ts.elapsed().as_nanos() as u64);
1526 }
1527 if let Some(merge_barrier_align_duration) = &self.merge_barrier_align_duration {
1528 merge_barrier_align_duration.observe(start_ts.elapsed().as_secs_f64())
1530 }
1531
1532 break;
1533 }
1534 }
1535 }
1536
1537 assert!(self.active.is_terminated());
1538
1539 let barrier = self.barrier.take().unwrap();
1540
1541 let upstreams = std::mem::take(&mut self.blocked);
1542 self.extend_active(upstreams);
1543 assert!(!self.active.is_terminated());
1544
1545 Poll::Ready(Some(Ok(MessageInner::Barrier(barrier))))
1546 }
1547}
1548
1549impl<InputId: Clone + Ord + Hash + std::fmt::Debug, M> DynamicReceivers<InputId, M> {
1550 pub fn new(
1551 upstreams: Vec<BoxedMessageInput<InputId, M>>,
1552 barrier_align_duration: Option<LabelGuardedMetric<GenericCounter<AtomicU64>>>,
1553 merge_barrier_align_duration: Option<LabelGuardedMetric<Histogram>>,
1554 ) -> Self {
1555 let mut this = Self {
1556 barrier: None,
1557 start_ts: None,
1558 blocked: Vec::with_capacity(upstreams.len()),
1559 active: Default::default(),
1560 buffered_watermarks: Default::default(),
1561 merge_barrier_align_duration,
1562 barrier_align_duration,
1563 };
1564 this.extend_active(upstreams);
1565 this
1566 }
1567
1568 pub fn extend_active(
1571 &mut self,
1572 upstreams: impl IntoIterator<Item = BoxedMessageInput<InputId, M>>,
1573 ) {
1574 assert!(self.blocked.is_empty() && self.barrier.is_none());
1575
1576 self.active
1577 .extend(upstreams.into_iter().map(|s| s.into_future()));
1578 }
1579
1580 pub fn handle_watermark(
1582 &mut self,
1583 input_id: InputId,
1584 watermark: Watermark,
1585 ) -> Option<Watermark> {
1586 let col_idx = watermark.col_idx;
1587 let upstream_ids: Vec<_> = self.upstream_input_ids().collect();
1589 let watermarks = self
1590 .buffered_watermarks
1591 .entry(col_idx)
1592 .or_insert_with(|| BufferedWatermarks::with_ids(upstream_ids));
1593 watermarks.handle_watermark(input_id, watermark)
1594 }
1595
1596 pub fn add_upstreams_from(
1599 &mut self,
1600 new_inputs: impl IntoIterator<Item = BoxedMessageInput<InputId, M>>,
1601 ) {
1602 assert!(self.blocked.is_empty() && self.barrier.is_none());
1603
1604 let new_inputs: Vec<_> = new_inputs.into_iter().collect();
1605 let input_ids = new_inputs.iter().map(|input| input.id());
1606 self.buffered_watermarks.values_mut().for_each(|buffers| {
1607 buffers.add_buffers(input_ids.clone());
1609 });
1610 self.active
1611 .extend(new_inputs.into_iter().map(|s| s.into_future()));
1612 }
1613
1614 pub fn remove_upstreams(&mut self, upstream_input_ids: &HashSet<InputId>) {
1618 assert!(self.blocked.is_empty() && self.barrier.is_none());
1619
1620 let new_upstreams = std::mem::take(&mut self.active)
1621 .into_iter()
1622 .map(|s| s.into_inner().unwrap())
1623 .filter(|u| !upstream_input_ids.contains(&u.id()));
1624 self.extend_active(new_upstreams);
1625 self.buffered_watermarks.values_mut().for_each(|buffers| {
1626 buffers.remove_buffer(upstream_input_ids.clone());
1629 });
1630 }
1631
1632 pub fn merge_barrier_align_duration(&self) -> Option<LabelGuardedMetric<Histogram>> {
1633 self.merge_barrier_align_duration.clone()
1634 }
1635
1636 pub fn flush_buffered_watermarks(&mut self) {
1637 self.buffered_watermarks
1638 .values_mut()
1639 .for_each(|buffers| buffers.clear());
1640 }
1641
1642 pub fn upstream_input_ids(&self) -> impl Iterator<Item = InputId> + '_ {
1643 self.blocked
1644 .iter()
1645 .map(|s| s.id())
1646 .chain(self.active.iter().map(|s| s.get_ref().unwrap().id()))
1647 }
1648
1649 pub fn is_empty(&self) -> bool {
1650 self.blocked.is_empty() && self.active.is_empty()
1651 }
1652}
1653
1654pub(crate) struct DispatchBarrierBuffer {
1675 buffer: VecDeque<(Barrier, Option<Vec<BoxedActorInput>>)>,
1676 barrier_rx: mpsc::UnboundedReceiver<Barrier>,
1677 recv_state: BarrierReceiverState,
1678 curr_upstream_fragment_id: FragmentId,
1679 actor_id: ActorId,
1680 build_input_ctx: Arc<BuildInputContext>,
1682}
1683
1684struct BuildInputContext {
1685 pub actor_id: ActorId,
1686 pub local_barrier_manager: LocalBarrierManager,
1687 pub metrics: Arc<StreamingMetrics>,
1688 pub fragment_id: FragmentId,
1689}
1690
1691type BoxedNewInputsFuture =
1692 Pin<Box<dyn Future<Output = StreamExecutorResult<Vec<BoxedActorInput>>> + Send>>;
1693
1694enum BarrierReceiverState {
1695 ReceivingBarrier,
1696 CreatingNewInput(Barrier, BoxedNewInputsFuture),
1697}
1698
1699impl DispatchBarrierBuffer {
1700 pub fn new(
1701 barrier_rx: mpsc::UnboundedReceiver<Barrier>,
1702 actor_id: ActorId,
1703 curr_upstream_fragment_id: FragmentId,
1704 local_barrier_manager: LocalBarrierManager,
1705 metrics: Arc<StreamingMetrics>,
1706 fragment_id: FragmentId,
1707 ) -> Self {
1708 Self {
1709 buffer: VecDeque::new(),
1710 barrier_rx,
1711 recv_state: BarrierReceiverState::ReceivingBarrier,
1712 curr_upstream_fragment_id,
1713 actor_id,
1714 build_input_ctx: Arc::new(BuildInputContext {
1715 actor_id,
1716 local_barrier_manager,
1717 metrics,
1718 fragment_id,
1719 }),
1720 }
1721 }
1722
1723 pub async fn await_next_message(
1724 &mut self,
1725 stream: &mut (impl Stream<Item = StreamExecutorResult<DispatcherMessage>> + Unpin),
1726 metrics: &ActorInputMetrics,
1727 ) -> StreamExecutorResult<DispatcherMessage> {
1728 let mut start_time = Instant::now();
1729 let interval_duration = Duration::from_secs(15);
1730 let mut interval =
1731 tokio::time::interval_at(start_time + interval_duration, interval_duration);
1732
1733 loop {
1734 tokio::select! {
1735 biased;
1736 msg = stream.try_next() => {
1737 metrics
1738 .actor_input_buffer_blocking_duration_ns
1739 .inc_by(start_time.elapsed().as_nanos() as u64);
1740 return msg?.ok_or_else(
1741 || StreamExecutorError::channel_closed("upstream executor closed unexpectedly")
1742 );
1743 }
1744
1745 e = self.continuously_fetch_barrier_rx() => {
1746 return Err(e);
1747 }
1748
1749 _ = interval.tick() => {
1750 start_time = Instant::now();
1751 metrics.actor_input_buffer_blocking_duration_ns.inc_by(interval_duration.as_nanos() as u64);
1752 continue;
1753 }
1754 }
1755 }
1756 }
1757
1758 pub async fn pop_barrier_with_inputs(
1759 &mut self,
1760 barrier: DispatcherBarrier,
1761 ) -> StreamExecutorResult<(Barrier, Option<Vec<BoxedActorInput>>)> {
1762 while self.buffer.is_empty() {
1763 self.try_fetch_barrier_rx(false).await?;
1764 }
1765 let (recv_barrier, inputs) = self.buffer.pop_front().unwrap();
1766 assert_equal_dispatcher_barrier(&recv_barrier, &barrier);
1767
1768 Ok((recv_barrier, inputs))
1769 }
1770
1771 async fn continuously_fetch_barrier_rx(&mut self) -> StreamExecutorError {
1772 loop {
1773 if let Err(e) = self.try_fetch_barrier_rx(true).await {
1774 return e;
1775 }
1776 }
1777 }
1778
1779 async fn try_fetch_barrier_rx(&mut self, pending_on_end: bool) -> StreamExecutorResult<()> {
1780 match &mut self.recv_state {
1781 BarrierReceiverState::ReceivingBarrier => {
1782 let Some(barrier) = self.barrier_rx.recv().await else {
1783 if pending_on_end {
1784 return pending().await;
1785 } else {
1786 return Err(StreamExecutorError::channel_closed(
1787 "barrier channel closed unexpectedly",
1788 ));
1789 }
1790 };
1791 if let Some(fut) = self.pre_apply_barrier(&barrier) {
1792 self.recv_state = BarrierReceiverState::CreatingNewInput(barrier, fut);
1793 } else {
1794 self.buffer.push_back((barrier, None));
1795 }
1796 }
1797 BarrierReceiverState::CreatingNewInput(barrier, fut) => {
1798 let new_inputs = fut.await?;
1799 self.buffer.push_back((barrier.clone(), Some(new_inputs)));
1800 self.recv_state = BarrierReceiverState::ReceivingBarrier;
1801 }
1802 }
1803 Ok(())
1804 }
1805
1806 fn pre_apply_barrier(&mut self, barrier: &Barrier) -> Option<BoxedNewInputsFuture> {
1807 if let Some(update) = barrier.as_update_merge(self.actor_id, self.curr_upstream_fragment_id)
1808 && !update.added_upstream_actors.is_empty()
1809 {
1810 let upstream_fragment_id =
1812 if let Some(new_upstream_fragment_id) = update.new_upstream_fragment_id {
1813 self.curr_upstream_fragment_id = new_upstream_fragment_id;
1814 new_upstream_fragment_id
1815 } else {
1816 self.curr_upstream_fragment_id
1817 };
1818 let ctx = self.build_input_ctx.clone();
1819 let added_upstream_actors = update.added_upstream_actors.clone();
1820 let barrier = barrier.clone();
1821 let fut = async move {
1822 try_join_all(added_upstream_actors.iter().map(|upstream_actor| async {
1823 let mut new_input = new_input(
1824 &ctx.local_barrier_manager,
1825 ctx.metrics.clone(),
1826 ctx.actor_id,
1827 ctx.fragment_id,
1828 upstream_actor,
1829 upstream_fragment_id,
1830 )
1831 .await?;
1832
1833 let first_barrier = expect_first_barrier(&mut new_input).await?;
1836 assert_equal_dispatcher_barrier(&barrier, &first_barrier);
1837
1838 StreamExecutorResult::Ok(new_input)
1839 }))
1840 .await
1841 }
1842 .boxed();
1843
1844 Some(fut)
1845 } else {
1846 None
1847 }
1848 }
1849}