Skip to content

Commit 1b208bb

Browse files
authored
feat(streaming): separate BarrierRecv executor (risingwavelabs#8595)
Signed-off-by: Bugen Zhao <[email protected]>
1 parent 78ddbce commit 1b208bb

File tree

9 files changed

+229
-5
lines changed

9 files changed

+229
-5
lines changed

dashboard/proto/gen/stream_plan.ts

Lines changed: 47 additions & 2 deletions
Some generated files are not rendered by default. Learn more about customizing how changed files appear on GitHub.

proto/stream_plan.proto

Lines changed: 7 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -131,6 +131,10 @@ message StreamSource {
131131
string source_name = 8;
132132
}
133133

134+
// The executor only for receiving barrier from the meta service. It always resides in the leaves
135+
// of the streaming graph.
136+
message BarrierRecvNode {}
137+
134138
message SourceNode {
135139
// The source node can contain either a stream source or nothing. So here we extract all
136140
// information about stream source to a message, and here it will be an `Option` in Rust.
@@ -545,6 +549,7 @@ message StreamNode {
545549
NowNode now = 129;
546550
GroupTopNNode append_only_group_top_n = 130;
547551
TemporalJoinNode temporal_join = 131;
552+
BarrierRecvNode barrier_recv = 132;
548553
}
549554
// The id for the operator. This is local per mview.
550555
// TODO: should better be a uint32.
@@ -630,8 +635,9 @@ enum FragmentTypeFlag {
630635
SOURCE = 1;
631636
MVIEW = 2;
632637
SINK = 4;
633-
NOW = 8;
638+
NOW = 8; // TODO: Remove this and insert a `BarrierRecv` instead.
634639
CHAIN_NODE = 16;
640+
BARRIER_RECV = 32;
635641
}
636642

637643
// The environment associated with a stream plan

src/common/src/catalog/schema.rs

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -118,6 +118,11 @@ pub struct Schema {
118118
}
119119

120120
impl Schema {
121+
pub fn empty() -> &'static Self {
122+
static EMPTY: Schema = Schema { fields: Vec::new() };
123+
&EMPTY
124+
}
125+
121126
pub fn len(&self) -> usize {
122127
self.fields.len()
123128
}

src/frontend/src/stream_fragmenter/mod.rs

Lines changed: 5 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -231,6 +231,10 @@ fn build_fragment(
231231
) -> Result<StreamNode> {
232232
// Update current fragment based on the node we're visiting.
233233
match stream_node.get_node_body()? {
234+
NodeBody::BarrierRecv(_) => {
235+
current_fragment.fragment_type_mask |= FragmentTypeFlag::BarrierRecv as u32
236+
}
237+
234238
NodeBody::Source(src) => {
235239
current_fragment.fragment_type_mask |= FragmentTypeFlag::Source as u32;
236240
// Note: For creating table with connector, the source id is left with placeholder and
@@ -248,7 +252,6 @@ fn build_fragment(
248252

249253
NodeBody::TopN(_) => current_fragment.requires_singleton = true,
250254

251-
// FIXME: workaround for single-fragment mview on singleton upstream mview.
252255
NodeBody::Chain(node) => {
253256
current_fragment.fragment_type_mask |= FragmentTypeFlag::ChainNode as u32;
254257
// memorize table id for later use
@@ -259,6 +262,7 @@ fn build_fragment(
259262
}
260263

261264
NodeBody::Now(_) => {
265+
// TODO: Remove this and insert a `BarrierRecv` instead.
262266
current_fragment.fragment_type_mask |= FragmentTypeFlag::Now as u32;
263267
current_fragment.requires_singleton = true;
264268
}

src/meta/src/model/stream.rs

Lines changed: 4 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -259,7 +259,10 @@ impl TableFragments {
259259
/// Returns barrier inject actor ids.
260260
pub fn barrier_inject_actor_ids(&self) -> Vec<ActorId> {
261261
Self::filter_actor_ids(self, |fragment_type_mask| {
262-
(fragment_type_mask & (FragmentTypeFlag::Source as u32 | FragmentTypeFlag::Now as u32))
262+
(fragment_type_mask
263+
& (FragmentTypeFlag::Source as u32
264+
| FragmentTypeFlag::Now as u32
265+
| FragmentTypeFlag::BarrierRecv as u32))
263266
!= 0
264267
})
265268
}
Lines changed: 107 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,107 @@
1+
// Copyright 2023 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+
15+
use futures::StreamExt;
16+
use risingwave_common::catalog::Schema;
17+
use tokio::sync::mpsc::UnboundedReceiver;
18+
use tokio_stream::wrappers::UnboundedReceiverStream;
19+
20+
use super::{
21+
ActorContext, ActorContextRef, Barrier, BoxedMessageStream, Executor, Message, PkIndicesRef,
22+
StreamExecutorError,
23+
};
24+
25+
/// The executor only for receiving barrier from the meta service. It always resides in the leaves
26+
/// of the streaming graph.
27+
pub struct BarrierRecvExecutor {
28+
_ctx: ActorContextRef,
29+
identity: String,
30+
31+
/// The barrier receiver registered in the local barrier manager.
32+
barrier_receiver: UnboundedReceiver<Barrier>,
33+
}
34+
35+
impl BarrierRecvExecutor {
36+
pub fn new(
37+
ctx: ActorContextRef,
38+
barrier_receiver: UnboundedReceiver<Barrier>,
39+
executor_id: u64,
40+
) -> Self {
41+
Self {
42+
_ctx: ctx,
43+
identity: format!("BarrierRecvExecutor {:X}", executor_id),
44+
barrier_receiver,
45+
}
46+
}
47+
48+
pub fn for_test(barrier_receiver: UnboundedReceiver<Barrier>) -> Self {
49+
Self::new(ActorContext::create(0), barrier_receiver, 0)
50+
}
51+
}
52+
53+
impl Executor for BarrierRecvExecutor {
54+
fn execute(self: Box<Self>) -> BoxedMessageStream {
55+
UnboundedReceiverStream::new(self.barrier_receiver)
56+
.map(|barrier| Ok(Message::Barrier(barrier)))
57+
.chain(futures::stream::once(async {
58+
// We do not use the stream termination as the control message, and this line should
59+
// never be reached in normal cases. So we just return an error here.
60+
Err(StreamExecutorError::channel_closed("barrier receiver"))
61+
}))
62+
.boxed()
63+
}
64+
65+
fn schema(&self) -> &Schema {
66+
Schema::empty()
67+
}
68+
69+
fn pk_indices(&self) -> PkIndicesRef<'_> {
70+
&[]
71+
}
72+
73+
fn identity(&self) -> &str {
74+
&self.identity
75+
}
76+
}
77+
78+
#[cfg(test)]
79+
mod tests {
80+
use futures::pin_mut;
81+
use tokio::sync::mpsc;
82+
83+
use super::*;
84+
use crate::executor::test_utils::StreamExecutorTestExt;
85+
86+
#[tokio::test]
87+
async fn test_barrier_recv() {
88+
let (barrier_tx, barrier_rx) = mpsc::unbounded_channel();
89+
90+
let barrier_recv = BarrierRecvExecutor::for_test(barrier_rx).boxed();
91+
let stream = barrier_recv.execute();
92+
pin_mut!(stream);
93+
94+
barrier_tx.send(Barrier::new_test_barrier(114)).unwrap();
95+
barrier_tx.send(Barrier::new_test_barrier(514)).unwrap();
96+
97+
let barrier_1 = stream.next_unwrap_ready_barrier().unwrap();
98+
assert_eq!(barrier_1.epoch.curr, 114);
99+
let barrier_2 = stream.next_unwrap_ready_barrier().unwrap();
100+
assert_eq!(barrier_2.epoch.curr, 514);
101+
102+
stream.next_unwrap_pending();
103+
104+
drop(barrier_tx);
105+
assert!(stream.next_unwrap_ready().is_err());
106+
}
107+
}

src/stream/src/executor/mod.rs

Lines changed: 2 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -57,6 +57,7 @@ pub mod monitor;
5757

5858
pub mod agg_common;
5959
pub mod aggregation;
60+
mod barrier_recv;
6061
mod batch_query;
6162
mod chain;
6263
mod dispatch;
@@ -103,6 +104,7 @@ mod test_utils;
103104
pub use actor::{Actor, ActorContext, ActorContextRef};
104105
use anyhow::Context;
105106
pub use backfill::*;
107+
pub use barrier_recv::BarrierRecvExecutor;
106108
pub use batch_query::BatchQueryExecutor;
107109
pub use chain::ChainExecutor;
108110
pub use dispatch::{DispatchExecutor, DispatcherImpl};
Lines changed: 49 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,49 @@
1+
// Copyright 2023 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+
15+
use risingwave_pb::stream_plan::BarrierRecvNode;
16+
use tokio::sync::mpsc::unbounded_channel;
17+
18+
use super::*;
19+
use crate::executor::BarrierRecvExecutor;
20+
21+
pub struct BarrierRecvExecutorBuilder;
22+
23+
#[async_trait::async_trait]
24+
impl ExecutorBuilder for BarrierRecvExecutorBuilder {
25+
type Node = BarrierRecvNode;
26+
27+
async fn new_boxed_executor(
28+
params: ExecutorParams,
29+
_node: &Self::Node,
30+
_store: impl StateStore,
31+
stream: &mut LocalStreamManagerCore,
32+
) -> StreamResult<BoxedExecutor> {
33+
assert!(
34+
params.input.is_empty(),
35+
"barrier receiver should not have input"
36+
);
37+
38+
let (sender, barrier_receiver) = unbounded_channel();
39+
stream
40+
.context
41+
.lock_barrier_manager()
42+
.register_sender(params.actor_context.id, sender);
43+
44+
Ok(
45+
BarrierRecvExecutor::new(params.actor_context, barrier_receiver, params.executor_id)
46+
.boxed(),
47+
)
48+
}
49+
}

src/stream/src/from_proto/mod.rs

Lines changed: 3 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -15,6 +15,7 @@
1515
//! Build executor from protobuf.
1616
1717
mod agg_common;
18+
mod barrier_recv;
1819
mod batch_query;
1920
mod chain;
2021
mod dml;
@@ -51,6 +52,7 @@ use risingwave_pb::stream_plan::stream_node::NodeBody;
5152
use risingwave_pb::stream_plan::{StreamNode, TemporalJoinNode};
5253
use risingwave_storage::StateStore;
5354

55+
use self::barrier_recv::*;
5456
use self::batch_query::*;
5557
use self::chain::*;
5658
use self::dml::*;
@@ -152,5 +154,6 @@ pub async fn create_executor(
152154
NodeBody::RowIdGen => RowIdGenExecutorBuilder,
153155
NodeBody::Now => NowExecutorBuilder,
154156
NodeBody::TemporalJoin => TemporalJoinExecutorBuilder,
157+
NodeBody::BarrierRecv => BarrierRecvExecutorBuilder,
155158
}
156159
}

0 commit comments

Comments
 (0)