risingwave_stream/executor/exchange/
input.rs1use std::pin::Pin;
16use std::task::{Context, Poll};
17
18use either::Either;
19use local_input::LocalInputStreamInner;
20use pin_project::pin_project;
21use risingwave_common::config::StreamingConfig;
22use risingwave_common::util::addr::{HostAddr, is_local_address};
23
24use super::permit::Receiver;
25use crate::executor::prelude::*;
26use crate::executor::{
27 BarrierInner, DispatcherMessage, DispatcherMessageBatch, DispatcherMessageStreamItem,
28};
29use crate::task::{FragmentId, LocalBarrierManager, UpDownActorIds, UpDownFragmentIds};
30
31pub trait Input: Stream + Send {
34 type InputId;
35 fn id(&self) -> Self::InputId;
37
38 fn boxed_input(self) -> BoxedInput<Self::InputId, Self::Item>
39 where
40 Self: Sized + 'static,
41 {
42 Box::pin(self)
43 }
44}
45
46pub type BoxedInput<InputId, Item> = Pin<Box<dyn Input<InputId = InputId, Item = Item>>>;
47
48pub trait ActorInput = Input<Item = DispatcherMessageStreamItem, InputId = ActorId>;
52
53pub type BoxedActorInput = Pin<Box<dyn ActorInput>>;
54
55impl std::fmt::Debug for dyn ActorInput {
56 fn fmt(&self, f: &mut std::fmt::Formatter<'_>) -> std::fmt::Result {
57 f.debug_struct("Input")
58 .field("actor_id", &self.id())
59 .finish_non_exhaustive()
60 }
61}
62
63#[pin_project]
65pub struct LocalInput {
66 #[pin]
67 inner: LocalInputStreamInner,
68
69 actor_id: ActorId,
70}
71
72pub(crate) fn assert_equal_dispatcher_barrier<M1, M2>(
73 first: &BarrierInner<M1>,
74 second: &BarrierInner<M2>,
75) {
76 assert_eq!(first.epoch, second.epoch);
77 assert_eq!(first.kind, second.kind);
78}
79
80impl LocalInput {
81 pub fn new(channel: Receiver, upstream_actor_id: ActorId) -> Self {
82 Self {
83 inner: local_input::run(channel, upstream_actor_id),
84 actor_id: upstream_actor_id,
85 }
86 }
87}
88
89mod local_input {
90 use await_tree::InstrumentAwait;
91 use either::Either;
92
93 use crate::executor::exchange::error::ExchangeChannelClosed;
94 use crate::executor::exchange::permit::Receiver;
95 use crate::executor::prelude::try_stream;
96 use crate::executor::{DispatcherMessage, StreamExecutorError};
97 use crate::task::ActorId;
98
99 pub(super) type LocalInputStreamInner = impl crate::executor::DispatcherMessageStream;
100
101 #[define_opaque(LocalInputStreamInner)]
102 pub(super) fn run(channel: Receiver, upstream_actor_id: ActorId) -> LocalInputStreamInner {
103 run_inner(channel, upstream_actor_id)
104 }
105
106 #[try_stream(ok = DispatcherMessage, error = StreamExecutorError)]
107 async fn run_inner(mut channel: Receiver, upstream_actor_id: ActorId) {
108 let span = await_tree::span!("LocalInput (actor {upstream_actor_id})").verbose();
109 while let Some(msg) = channel.recv().instrument_await(span.clone()).await {
110 match msg.into_messages() {
111 Either::Left(barriers) => {
112 for b in barriers {
113 yield b;
114 }
115 }
116 Either::Right(m) => {
117 yield m;
118 }
119 }
120 }
121 Err(ExchangeChannelClosed::local_input(upstream_actor_id))?
124 }
125}
126
127impl Stream for LocalInput {
128 type Item = DispatcherMessageStreamItem;
129
130 fn poll_next(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Option<Self::Item>> {
131 self.project().inner.poll_next(cx)
133 }
134}
135
136impl Input for LocalInput {
137 type InputId = ActorId;
138
139 fn id(&self) -> Self::InputId {
140 self.actor_id
141 }
142}
143
144#[pin_project]
146pub struct RemoteInput {
147 #[pin]
148 inner: RemoteInputStreamInner,
149
150 actor_id: ActorId,
151}
152
153use remote_input::RemoteInputStreamInner;
154use risingwave_pb::common::ActorInfo;
155
156impl RemoteInput {
157 pub async fn new(
160 local_barrier_manager: &LocalBarrierManager,
161 upstream_addr: HostAddr,
162 up_down_ids: UpDownActorIds,
163 up_down_frag: UpDownFragmentIds,
164 metrics: Arc<StreamingMetrics>,
165 actor_config: Arc<StreamingConfig>,
166 ) -> StreamExecutorResult<Self> {
167 let actor_id = up_down_ids.0;
168
169 let client = local_barrier_manager
170 .env
171 .client_pool()
172 .get_by_addr(upstream_addr)
173 .await?;
174 let (stream, permits_tx) = client
175 .get_stream(
176 up_down_ids.0,
177 up_down_ids.1,
178 up_down_frag.0,
179 up_down_frag.1,
180 local_barrier_manager.database_id,
181 local_barrier_manager.term_id.clone(),
182 )
183 .await?;
184
185 Ok(Self {
186 actor_id,
187 inner: remote_input::run(
188 stream,
189 permits_tx,
190 up_down_ids,
191 up_down_frag,
192 metrics,
193 actor_config.developer.exchange_batched_permits,
194 ),
195 })
196 }
197}
198
199mod remote_input {
200 use std::sync::Arc;
201
202 use anyhow::Context;
203 use await_tree::InstrumentAwait;
204 use either::Either;
205 use risingwave_pb::task_service::{GetStreamResponse, permits};
206 use tokio::sync::mpsc;
207 use tonic::Streaming;
208
209 use crate::executor::exchange::error::ExchangeChannelClosed;
210 use crate::executor::monitor::StreamingMetrics;
211 use crate::executor::prelude::{StreamExt, pin_mut, try_stream};
212 use crate::executor::{DispatcherMessage, StreamExecutorError};
213 use crate::task::{UpDownActorIds, UpDownFragmentIds};
214
215 pub(super) type RemoteInputStreamInner = impl crate::executor::DispatcherMessageStream;
216
217 #[define_opaque(RemoteInputStreamInner)]
218 pub(super) fn run(
219 stream: Streaming<GetStreamResponse>,
220 permits_tx: mpsc::UnboundedSender<permits::Value>,
221 up_down_ids: UpDownActorIds,
222 up_down_frag: UpDownFragmentIds,
223 metrics: Arc<StreamingMetrics>,
224 batched_permits_limit: usize,
225 ) -> RemoteInputStreamInner {
226 run_inner(
227 stream,
228 permits_tx,
229 up_down_ids,
230 up_down_frag,
231 metrics,
232 batched_permits_limit,
233 )
234 }
235
236 #[try_stream(ok = DispatcherMessage, error = StreamExecutorError)]
237 async fn run_inner(
238 stream: Streaming<GetStreamResponse>,
239 permits_tx: mpsc::UnboundedSender<permits::Value>,
240 up_down_ids: UpDownActorIds,
241 up_down_frag: UpDownFragmentIds,
242 metrics: Arc<StreamingMetrics>,
243 batched_permits_limit: usize,
244 ) {
245 let up_actor_id = up_down_ids.0.to_string();
246 let up_fragment_id = up_down_frag.0.to_string();
247 let down_fragment_id = up_down_frag.1.to_string();
248 let exchange_frag_recv_size_metrics = metrics
249 .exchange_frag_recv_size
250 .with_guarded_label_values(&[&up_fragment_id, &down_fragment_id]);
251
252 let span = await_tree::span!("RemoteInput (actor {up_actor_id})").verbose();
253
254 let mut batched_permits_accumulated = 0;
255
256 pin_mut!(stream);
257 while let Some(data_res) = stream.next().instrument_await(span.clone()).await {
258 match data_res {
259 Ok(GetStreamResponse { message, permits }) => {
260 use crate::executor::DispatcherMessageBatch;
261 let msg = message.unwrap();
262 let bytes = DispatcherMessageBatch::get_encoded_len(&msg);
263
264 exchange_frag_recv_size_metrics.inc_by(bytes as u64);
265
266 let msg_res = DispatcherMessageBatch::from_protobuf(&msg);
267 if let Some(add_back_permits) = match permits.unwrap().value {
268 Some(permits::Value::Record(p)) => {
271 batched_permits_accumulated += p;
272 if batched_permits_accumulated >= batched_permits_limit as u32 {
273 let permits = std::mem::take(&mut batched_permits_accumulated);
274 Some(permits::Value::Record(permits))
275 } else {
276 None
277 }
278 }
279 Some(permits::Value::Barrier(p)) => Some(permits::Value::Barrier(p)),
281 None => None,
282 } {
283 permits_tx
284 .send(add_back_permits)
285 .context("RemoteInput backward permits channel closed.")?;
286 }
287
288 let msg = msg_res.context("RemoteInput decode message error")?;
289 match msg.into_messages() {
290 Either::Left(barriers) => {
291 for b in barriers {
292 yield b;
293 }
294 }
295 Either::Right(m) => {
296 yield m;
297 }
298 }
299 }
300
301 Err(e) => Err(ExchangeChannelClosed::remote_input(up_down_ids.0, Some(e)))?,
302 }
303 }
304
305 Err(ExchangeChannelClosed::remote_input(up_down_ids.0, None))?
308 }
309}
310
311impl Stream for RemoteInput {
312 type Item = DispatcherMessageStreamItem;
313
314 fn poll_next(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Option<Self::Item>> {
315 self.project().inner.poll_next(cx)
316 }
317}
318
319impl Input for RemoteInput {
320 type InputId = ActorId;
321
322 fn id(&self) -> Self::InputId {
323 self.actor_id
324 }
325}
326
327pub(crate) async fn new_input(
330 local_barrier_manager: &LocalBarrierManager,
331 metrics: Arc<StreamingMetrics>,
332 actor_id: ActorId,
333 fragment_id: FragmentId,
334 upstream_actor_info: &ActorInfo,
335 upstream_fragment_id: FragmentId,
336 actor_config: Arc<StreamingConfig>,
337) -> StreamExecutorResult<BoxedActorInput> {
338 let upstream_actor_id = upstream_actor_info.actor_id;
339 let upstream_addr = upstream_actor_info.get_host()?.into();
340
341 let input = if is_local_address(local_barrier_manager.env.server_address(), &upstream_addr) {
342 LocalInput::new(
343 local_barrier_manager.register_local_upstream_output(actor_id, upstream_actor_id),
344 upstream_actor_id,
345 )
346 .boxed_input()
347 } else {
348 RemoteInput::new(
349 local_barrier_manager,
350 upstream_addr,
351 (upstream_actor_id, actor_id),
352 (upstream_fragment_id, fragment_id),
353 metrics,
354 actor_config,
355 )
356 .await?
357 .boxed_input()
358 };
359
360 Ok(input)
361}
362
363impl DispatcherMessageBatch {
364 fn into_messages(self) -> Either<impl Iterator<Item = DispatcherMessage>, DispatcherMessage> {
365 match self {
366 DispatcherMessageBatch::BarrierBatch(barriers) => {
367 Either::Left(barriers.into_iter().map(DispatcherMessage::Barrier))
368 }
369 DispatcherMessageBatch::Chunk(c) => Either::Right(DispatcherMessage::Chunk(c)),
370 DispatcherMessageBatch::Watermark(w) => Either::Right(DispatcherMessage::Watermark(w)),
371 }
372 }
373}