risingwave_stream/executor/
actor.rs1use std::collections::{HashMap, HashSet};
16use std::sync::atomic::{AtomicUsize, Ordering};
17use std::sync::{Arc, LazyLock};
18
19use anyhow::anyhow;
20use await_tree::InstrumentAwait;
21use futures::FutureExt;
22use futures::future::join_all;
23use hytra::TrAdder;
24use risingwave_common::bitmap::Bitmap;
25use risingwave_common::config::StreamingConfig;
26use risingwave_common::hash::VirtualNode;
27use risingwave_common::log::LogSuppressor;
28use risingwave_common::metrics::{GLOBAL_ERROR_METRICS, IntGaugeExt};
29use risingwave_common::util::epoch::EpochPair;
30use risingwave_expr::ExprError;
31use risingwave_expr::expr_context::{FRAGMENT_ID, VNODE_COUNT, expr_context_scope};
32use risingwave_pb::id::SubscriberId;
33use risingwave_pb::plan_common::ExprContext;
34use risingwave_pb::stream_service::inject_barrier_request::BuildActorInfo;
35use risingwave_pb::stream_service::inject_barrier_request::build_actor_info::UpstreamActors;
36use risingwave_rpc_client::MetaClient;
37use thiserror_ext::AsReport;
38use tokio_stream::StreamExt;
39use tracing::Instrument;
40
41use super::StreamConsumer;
42use super::monitor::StreamingMetrics;
43use super::subtask::SubtaskHandle;
44use crate::CONFIG;
45use crate::error::StreamResult;
46use crate::task::{ActorId, FragmentId, LocalBarrierManager, StreamEnvironment};
47
48pub struct ActorContext {
50 pub id: ActorId,
51 pub fragment_id: FragmentId,
52 pub vnode_count: usize,
53 pub mview_definition: String,
54
55 last_mem_val: Arc<AtomicUsize>,
57 cur_mem_val: Arc<AtomicUsize>,
58 total_mem_val: Arc<TrAdder<i64>>,
59
60 pub streaming_metrics: Arc<StreamingMetrics>,
61
62 pub initial_dispatch_num: usize,
64 pub initial_subscriber_ids: HashSet<SubscriberId>,
66 pub initial_upstream_actors: HashMap<FragmentId, UpstreamActors>,
67
68 pub meta_client: Option<MetaClient>,
70
71 pub config: Arc<StreamingConfig>,
75
76 pub stream_env: StreamEnvironment,
77}
78
79pub type ActorContextRef = Arc<ActorContext>;
80
81impl ActorContext {
82 pub fn for_test(id: impl Into<ActorId>) -> ActorContextRef {
83 Self::for_test_with_config(id, StreamingConfig::default())
84 }
85
86 pub fn for_test_with_config(
87 id: impl Into<ActorId>,
88 config: StreamingConfig,
89 ) -> ActorContextRef {
90 Arc::new(Self {
91 id: id.into(),
92 fragment_id: 0.into(),
93 vnode_count: VirtualNode::COUNT_FOR_TEST,
94 mview_definition: "".to_owned(),
95 cur_mem_val: Arc::new(0.into()),
96 last_mem_val: Arc::new(0.into()),
97 total_mem_val: Arc::new(TrAdder::new()),
98 streaming_metrics: Arc::new(StreamingMetrics::unused()),
99 initial_dispatch_num: 1,
101 initial_subscriber_ids: Default::default(),
102 initial_upstream_actors: Default::default(),
103 meta_client: None,
104 config: Arc::new(config),
105 stream_env: StreamEnvironment::for_test(),
106 })
107 }
108
109 pub fn create(
110 stream_actor: &BuildActorInfo,
111 fragment_id: FragmentId,
112 total_mem_val: Arc<TrAdder<i64>>,
113 streaming_metrics: Arc<StreamingMetrics>,
114 meta_client: Option<MetaClient>,
115 config: Arc<StreamingConfig>,
116 stream_env: StreamEnvironment,
117 ) -> ActorContextRef {
118 Arc::new(Self {
119 id: stream_actor.actor_id,
120 fragment_id,
121 mview_definition: stream_actor.mview_definition.clone(),
122 vnode_count: (stream_actor.vnode_bitmap.as_ref())
123 .map_or(1, |b| Bitmap::from(b).len()),
126 cur_mem_val: Arc::new(0.into()),
127 last_mem_val: Arc::new(0.into()),
128 total_mem_val,
129 streaming_metrics,
130 initial_dispatch_num: stream_actor.dispatchers.len(),
131 initial_subscriber_ids: stream_actor
132 .initial_subscriber_ids
133 .iter()
134 .copied()
135 .collect(),
136 initial_upstream_actors: stream_actor.fragment_upstreams.clone(),
137 meta_client,
138 config,
139 stream_env,
140 })
141 }
142
143 pub fn on_compute_error(&self, err: ExprError, identity: &str) {
144 static LOG_SUPPRESSOR: LazyLock<LogSuppressor> = LazyLock::new(LogSuppressor::default);
145 if let Ok(suppressed_count) = LOG_SUPPRESSOR.check() {
146 tracing::error!(target: "stream_expr_error", identity, error = %err.as_report(), suppressed_count, "failed to evaluate expression");
147 }
148
149 let executor_name = identity.split(' ').next().unwrap_or("name_not_found");
150 GLOBAL_ERROR_METRICS.user_compute_error.report([
151 "ExprError".to_owned(),
152 executor_name.to_owned(),
153 self.fragment_id.to_string(),
154 ]);
155 }
156
157 pub fn store_mem_usage(&self, val: usize) {
158 let old_value = self.cur_mem_val.load(Ordering::Relaxed);
162 self.last_mem_val.store(old_value, Ordering::Relaxed);
163 let diff = val as i64 - old_value as i64;
164
165 self.total_mem_val.inc(diff);
166
167 self.cur_mem_val.store(val, Ordering::Relaxed);
168 }
169
170 pub fn mem_usage(&self) -> usize {
171 self.cur_mem_val.load(Ordering::Relaxed)
172 }
173}
174
175pub struct Actor<C> {
177 consumer: C,
179 subtasks: Vec<SubtaskHandle>,
181
182 pub actor_context: ActorContextRef,
183 expr_context: ExprContext,
184 barrier_manager: LocalBarrierManager,
185}
186
187impl<C> Actor<C>
188where
189 C: StreamConsumer,
190{
191 pub fn new(
192 consumer: C,
193 subtasks: Vec<SubtaskHandle>,
194 _metrics: Arc<StreamingMetrics>,
195 actor_context: ActorContextRef,
196 expr_context: ExprContext,
197 barrier_manager: LocalBarrierManager,
198 ) -> Self {
199 Self {
200 consumer,
201 subtasks,
202 actor_context,
203 expr_context,
204 barrier_manager,
205 }
206 }
207
208 #[inline(always)]
209 pub async fn run(mut self) -> StreamResult<()> {
210 let expr_context = self.expr_context.clone();
211 let fragment_id = self.actor_context.fragment_id;
212 let vnode_count = self.actor_context.vnode_count;
213 let config = self.actor_context.config.clone();
214
215 let run = async move {
216 tokio::join!(
217 join_all(std::mem::take(&mut self.subtasks)),
219 self.run_consumer(),
220 )
221 .1
222 }
223 .boxed();
224
225 let run = expr_context_scope(expr_context, run);
227 let run = FRAGMENT_ID::scope(fragment_id, run);
228 let run = VNODE_COUNT::scope(vnode_count, run);
229 let run = CONFIG.scope(config, run);
230
231 run.await
232 }
233
234 async fn run_consumer(self) -> StreamResult<()> {
235 fail::fail_point!("start_actors_err", |_| Err(anyhow::anyhow!(
236 "intentional start_actors_err"
237 )
238 .into()));
239
240 let id = self.actor_context.id;
241 let span_name = format!("Actor {id}");
242
243 let new_span = |epoch: Option<EpochPair>| {
244 tracing::info_span!(
245 parent: None,
246 "actor",
247 "otel.name" = span_name,
248 actor_id = %id,
249 prev_epoch = epoch.map(|e| e.prev),
250 curr_epoch = epoch.map(|e| e.curr),
251 )
252 };
253 let mut span = new_span(None);
254
255 let actor_count = self
256 .actor_context
257 .streaming_metrics
258 .actor_count
259 .with_guarded_label_values(&[&self.actor_context.fragment_id.to_string()]);
260 let _actor_count_guard = actor_count.inc_guard();
261
262 let current_epoch = self
263 .actor_context
264 .streaming_metrics
265 .actor_current_epoch
266 .with_guarded_label_values(&[
267 &self.actor_context.id.to_string(),
268 &self.actor_context.fragment_id.to_string(),
269 ]);
270
271 let mut last_epoch: Option<EpochPair> = None;
272 let mut stream = Box::pin(Box::new(self.consumer).execute());
273
274 let result = loop {
276 let barrier = match stream
277 .try_next()
278 .instrument(span.clone())
279 .instrument_await(
280 last_epoch.map_or(await_tree::span!("Epoch <initial>"), |e| {
281 await_tree::span!("Epoch {}", e.curr)
282 }),
283 )
284 .await
285 {
286 Ok(Some(barrier)) => barrier,
287 Ok(None) => break Err(anyhow!("actor exited unexpectedly").into()),
288 Err(err) => break Err(err),
289 };
290
291 fail::fail_point!("collect_actors_err", id == 10, |_| Err(anyhow::anyhow!(
292 "intentional collect_actors_err"
293 )
294 .into()));
295
296 if barrier.is_stop(id) {
298 debug!(actor_id = %id, epoch = ?barrier.epoch, "stop at barrier");
299 break Ok(barrier);
300 }
301
302 current_epoch.set(barrier.epoch.curr as i64);
303
304 self.barrier_manager.collect(id, &barrier);
306
307 last_epoch = Some(barrier.epoch);
309 span = barrier.tracing_context().attach(new_span(last_epoch));
310 };
311
312 spawn_blocking_drop_stream(stream).await;
313
314 let result = result.map(|stop_barrier| {
315 self.barrier_manager.collect(id, &stop_barrier);
317 });
318
319 tracing::debug!(actor_id = %id, ok = result.is_ok(), "actor exit");
320 result
321 }
322}
323
324pub async fn spawn_blocking_drop_stream<T: Send + 'static>(stream: T) {
332 let _ = tokio::task::spawn_blocking(move || drop(stream))
333 .instrument_await("drop_stream")
334 .await;
335}