1use std::collections::BTreeMap;
15use std::iter;
16use std::sync::Arc;
17
18use differential_dataflow::{AsCollection, VecCollection};
19use mz_ore::cast::CastLossy;
20use mz_persist_client::operators::shard_source::SnapshotMode;
21use mz_repr::{Datum, Diff, GlobalId, Row, RowPacker};
22use mz_storage_operators::persist_source;
23use mz_storage_operators::persist_source::Subtime;
24use mz_storage_types::controller::CollectionMetadata;
25use mz_storage_types::dyncfgs;
26use mz_storage_types::errors::{
27 DataflowError, DecodeError, EnvelopeError, UpsertError, UpsertNullKeyError, UpsertValueError,
28};
29use mz_storage_types::parameters::StorageMaxInflightBytesConfig;
30use mz_storage_types::sources::envelope::{KeyEnvelope, NoneEnvelope, UpsertEnvelope, UpsertStyle};
31use mz_storage_types::sources::*;
32use mz_timely_util::builder_async::PressOnDropButton;
33use mz_timely_util::operator::CollectionExt;
34use mz_timely_util::order::refine_antichain;
35use serde::{Deserialize, Serialize};
36use timely::container::CapacityContainerBuilder;
37use timely::dataflow::StreamVec;
38use timely::dataflow::operators::vec::Map;
39use timely::dataflow::operators::{ConnectLoop, Feedback, Leave, OkErr};
40use timely::dataflow::scope::Scope;
41use timely::progress::{Antichain, Timestamp};
42
43use crate::decode::{render_decode_cdcv2, render_decode_delimited};
44use crate::healthcheck::{HealthStatusMessage, StatusNamespace};
45use crate::source::types::{DecodeResult, SourceOutput, SourceRender};
46use crate::source::{self, RawSourceCreationConfig, SourceExportCreationConfig};
47use crate::upsert::{UpsertKey, UpsertValue};
48
49pub fn render_source<'scope, 'root, C>(
61 scope: Scope<'scope, mz_repr::Timestamp>,
62 root_scope: Scope<'root, ()>,
63 dataflow_debug_name: &String,
64 connection: C,
65 description: IngestionDescription<CollectionMetadata>,
66 resume_stream: StreamVec<'scope, mz_repr::Timestamp, ()>,
67 storage_state: &crate::storage_state::StorageState,
68 base_source_config: RawSourceCreationConfig,
69) -> (
70 BTreeMap<
71 GlobalId,
72 (
73 VecCollection<'scope, mz_repr::Timestamp, Row, Diff>,
74 VecCollection<'scope, mz_repr::Timestamp, DataflowError, Diff>,
75 ),
76 >,
77 Vec<StreamVec<'root, (), HealthStatusMessage>>,
78 Vec<PressOnDropButton>,
79)
80where
81 C: SourceConnection + SourceRender + 'static,
82{
83 let mut needed_tokens = Vec::new();
85
86 let (starter, mut start_signal) = tokio::sync::mpsc::channel::<()>(1);
99 let start_signal = async move {
100 let _ = start_signal.recv().await;
101 };
102
103 let (exports, health, source_tokens) = source::create_raw_source(
106 scope,
107 root_scope,
108 storage_state,
109 resume_stream,
110 &base_source_config,
111 connection,
112 start_signal,
113 );
114
115 needed_tokens.extend(source_tokens);
116
117 let mut health_streams = Vec::with_capacity(exports.len() + 1);
118 health_streams.push(health);
119
120 let mut outputs = BTreeMap::new();
121 for (export_id, export) in exports {
122 type CB<C> = CapacityContainerBuilder<C>;
123 let (ok_stream, err_stream) =
124 export.map_fallible::<CB<_>, CB<_>, _, _, _>("export-demux-ok-err", |r| r);
125
126 let mut error_collections = Vec::new();
130
131 let data_config = base_source_config.source_exports[&export_id]
132 .data_config
133 .clone();
134 let (ok, extra_tokens, health_stream) = render_source_stream(
135 scope,
136 dataflow_debug_name,
137 export_id,
138 ok_stream,
139 data_config,
140 &description,
141 &mut error_collections,
142 storage_state,
143 &base_source_config,
144 starter.clone(),
145 );
146 needed_tokens.extend(extra_tokens);
147
148 let err_collection = match error_collections.len() {
150 0 => err_stream,
151 _ => err_stream.concatenate(error_collections),
152 };
153
154 outputs.insert(export_id, (ok, err_collection));
155
156 health_streams.extend(health_stream.into_iter().map(|s| s.leave(root_scope)));
157 }
158 (outputs, health_streams, needed_tokens)
159}
160
161fn render_source_stream<'scope, FromTime>(
164 scope: Scope<'scope, mz_repr::Timestamp>,
165 dataflow_debug_name: &String,
166 export_id: GlobalId,
167 ok_source: VecCollection<'scope, mz_repr::Timestamp, SourceOutput<FromTime>, Diff>,
168 data_config: SourceExportDataConfig,
169 description: &IngestionDescription<CollectionMetadata>,
170 error_collections: &mut Vec<VecCollection<'scope, mz_repr::Timestamp, DataflowError, Diff>>,
171 storage_state: &crate::storage_state::StorageState,
172 base_source_config: &RawSourceCreationConfig,
173 rehydrated_token: impl std::any::Any + 'static,
174) -> (
175 VecCollection<'scope, mz_repr::Timestamp, Row, Diff>,
176 Vec<PressOnDropButton>,
177 Vec<StreamVec<'scope, mz_repr::Timestamp, HealthStatusMessage>>,
178)
179where
180 FromTime: Timestamp + Sync,
181{
182 let mut needed_tokens = vec![];
183
184 let SourceExportDataConfig { encoding, envelope } = data_config;
186
187 let SourceDesc {
188 connection: _,
189 timestamp_interval: _,
190 } = description.desc;
191
192 let (decoded_stream, decode_health) = match encoding {
193 None => (
194 ok_source.map(|r| DecodeResult {
195 key: Some(Ok(r.key)),
203 value: Some(Ok(r.value)),
204 metadata: r.metadata,
205 from_time: r.from_time,
206 }),
207 None,
208 ),
209 Some(encoding) => {
210 let (decoded_stream, decode_health) = render_decode_delimited(
211 ok_source,
212 encoding.key,
213 encoding.value,
214 dataflow_debug_name.clone(),
215 storage_state.metrics.decode_defs.clone(),
216 storage_state.storage_configuration.clone(),
217 );
218 (decoded_stream, Some(decode_health))
219 }
220 };
221
222 let (envelope_ok, envelope_health) = match &envelope {
224 SourceEnvelope::Upsert(upsert_envelope) => {
225 let upsert_input = upsert_commands(decoded_stream, upsert_envelope.clone());
226
227 let persist_clients = Arc::clone(&storage_state.persist_clients);
228 let resume_upper = base_source_config.resume_uppers[&export_id].clone();
230
231 let upper_ts = resume_upper
232 .as_option()
233 .expect("resuming an already finished ingestion")
234 .clone();
235 let outer_mz_scope = scope.clone();
236 let (upsert, health_update) = scope.scoped(
237 &format!("upsert_rehydration_backpressure({})", export_id),
238 |scope| {
239 let (previous, previous_token, feedback_handle, backpressure_metrics) = {
240 let as_of = Antichain::from_elem(upper_ts.saturating_sub(1));
241
242 let backpressure_max_inflight_bytes = get_backpressure_max_inflight_bytes(
243 &storage_state
244 .storage_configuration
245 .parameters
246 .storage_dataflow_max_inflight_bytes_config,
247 &storage_state.instance_context.cluster_memory_limit,
248 );
249
250 let (feedback_handle, flow_control, backpressure_metrics) =
251 if let Some(storage_dataflow_max_inflight_bytes) =
252 backpressure_max_inflight_bytes
253 {
254 tracing::info!(
255 ?backpressure_max_inflight_bytes,
256 "timely-{} using backpressure in upsert for source {}",
257 base_source_config.worker_id,
258 export_id
259 );
260 if !storage_state
261 .storage_configuration
262 .parameters
263 .storage_dataflow_max_inflight_bytes_config
264 .disk_only
265 || storage_state.instance_context.scratch_directory.is_some()
266 {
267 let (feedback_handle, feedback_data) =
268 scope.feedback(Default::default());
269
270 let backpressure_metrics = Some(
272 base_source_config
273 .metrics
274 .get_backpressure_metrics(export_id, scope.index()),
275 );
276
277 (
278 Some(feedback_handle),
279 Some(persist_source::FlowControl {
280 progress_stream: feedback_data,
281 max_inflight_bytes: storage_dataflow_max_inflight_bytes,
282 summary: (Default::default(), Subtime::least_summary()),
283 metrics: backpressure_metrics.clone(),
284 }),
285 backpressure_metrics,
286 )
287 } else {
288 (None, None, None)
289 }
290 } else {
291 (None, None, None)
292 };
293
294 let storage_metadata = description.source_exports[&export_id]
295 .storage_metadata
296 .clone();
297
298 let error_handler =
299 storage_state.error_handler("upsert_rehydration", export_id);
300
301 let (stream, tok) = persist_source::persist_source_core(
302 outer_mz_scope,
303 scope,
304 export_id,
305 persist_clients,
306 storage_metadata,
307 None,
308 Some(as_of),
309 SnapshotMode::Include,
310 Antichain::new(),
311 None,
312 flow_control,
313 false.then_some(|| unreachable!()),
314 async {},
315 error_handler,
316 );
317 (
318 stream.as_collection(),
319 Some(tok),
320 feedback_handle,
321 backpressure_metrics,
322 )
323 };
324
325 let export_statistics = storage_state
326 .aggregated_statistics
327 .get_source(&export_id)
328 .expect("statistics initialized")
329 .clone();
330 let export_config = SourceExportCreationConfig {
331 id: export_id,
332 worker_id: base_source_config.worker_id,
333 metrics: base_source_config.metrics.clone(),
334 source_statistics: export_statistics,
335 };
336 let (upsert, health_update, snapshot_progress, upsert_token) =
337 crate::upsert::upsert(
338 upsert_input.enter(scope),
339 upsert_envelope.clone(),
340 refine_antichain(&resume_upper),
341 previous,
342 previous_token,
343 export_config,
344 &storage_state.instance_context,
345 &storage_state.storage_configuration,
346 &storage_state.dataflow_parameters,
347 backpressure_metrics,
348 );
349
350 needed_tokens.push(upsert_token);
355
356 if dyncfgs::DELAY_SOURCES_PAST_REHYDRATION
360 .get(storage_state.storage_configuration.config_set())
361 {
362 crate::upsert::rehydration_finished(
363 scope.clone(),
364 base_source_config,
365 rehydrated_token,
366 refine_antichain(&resume_upper),
367 snapshot_progress.clone(),
368 );
369 } else {
370 drop(rehydrated_token)
371 };
372
373 if let Some(feedback_handle) = feedback_handle {
376 snapshot_progress.connect_loop(feedback_handle);
377 }
378
379 (
380 upsert.leave(outer_mz_scope),
381 health_update
382 .map(|(id, update)| HealthStatusMessage {
383 id,
384 namespace: StatusNamespace::Upsert,
385 update,
386 })
387 .leave(outer_mz_scope),
388 )
389 },
390 );
391
392 let (upsert_ok, upsert_err) = upsert.inner.ok_err(split_ok_err);
393 error_collections.push(upsert_err.as_collection());
394
395 (upsert_ok.as_collection(), Some(health_update))
396 }
397 SourceEnvelope::None(none_envelope) => {
398 let results = append_metadata_to_value(decoded_stream);
399
400 let flattened_stream = flatten_results_prepend_keys(none_envelope, results);
401
402 let (stream, errors) = flattened_stream.inner.ok_err(split_ok_err);
403
404 error_collections.push(errors.as_collection());
405 (stream.as_collection(), None)
406 }
407 SourceEnvelope::CdcV2 => {
408 let (oks, token) = render_decode_cdcv2(&decoded_stream);
409 needed_tokens.push(token);
410 (oks, None)
411 }
412 };
413
414 let health = decode_health.into_iter().chain(envelope_health).collect();
416 (envelope_ok, needed_tokens, health)
417}
418
419fn get_backpressure_max_inflight_bytes(
422 inflight_bytes_config: &StorageMaxInflightBytesConfig,
423 cluster_memory_limit: &Option<usize>,
424) -> Option<usize> {
425 let StorageMaxInflightBytesConfig {
426 max_inflight_bytes_default,
427 max_inflight_bytes_cluster_size_fraction,
428 disk_only: _,
429 } = inflight_bytes_config;
430
431 if max_inflight_bytes_default.is_some() {
433 let current_cluster_max_bytes_limit =
434 cluster_memory_limit.as_ref().and_then(|cluster_memory| {
435 max_inflight_bytes_cluster_size_fraction.map(|fraction| {
436 usize::cast_lossy(f64::cast_lossy(*cluster_memory) * fraction)
438 })
439 });
440 current_cluster_max_bytes_limit.or(*max_inflight_bytes_default)
441 } else {
442 None
443 }
444}
445
446fn split_ok_err<O, E, T, D>(x: (Result<O, E>, T, D)) -> Result<(O, T, D), (E, T, D)> {
449 match x {
450 (Ok(ok), ts, diff) => Ok((ok, ts, diff)),
451 (Err(err), ts, diff) => Err((err, ts, diff)),
452 }
453}
454
455#[derive(
457 Debug,
458 Clone,
459 Hash,
460 PartialEq,
461 Eq,
462 Ord,
463 PartialOrd,
464 Serialize,
465 Deserialize
466)]
467struct KV {
468 key: Option<Result<Row, DecodeError>>,
469 val: Option<Result<Row, DecodeError>>,
470}
471
472fn append_metadata_to_value<'scope, T: Timestamp, FromTime: Timestamp>(
473 results: VecCollection<'scope, T, DecodeResult<FromTime>, Diff>,
474) -> VecCollection<'scope, T, KV, Diff> {
475 results.map(move |res| {
476 let val = res.value.map(|val_result| {
477 val_result.map(|mut val| {
478 if !res.metadata.is_empty() {
479 RowPacker::for_existing_row(&mut val).extend_by_row(&res.metadata);
480 }
481 val
482 })
483 });
484
485 KV { val, key: res.key }
486 })
487}
488
489fn upsert_commands<'scope, T: Timestamp, FromTime: Timestamp>(
491 input: VecCollection<'scope, T, DecodeResult<FromTime>, Diff>,
492 upsert_envelope: UpsertEnvelope,
493) -> VecCollection<'scope, T, (UpsertKey, Option<UpsertValue>, FromTime), Diff> {
494 let mut row_buf = Row::default();
495 input.map(move |result| {
496 let from_time = result.from_time;
497
498 let key = match result.key {
499 Some(Ok(key)) => Ok(key),
500 None => Err(UpsertError::NullKey(UpsertNullKeyError)),
501 Some(Err(err)) => Err(UpsertError::KeyDecode(err)),
502 };
503
504 let key = match key {
506 Ok(key) => key,
507 Err(err) => match result.value {
508 Some(_) => {
509 return (
510 UpsertKey::from_key(Err(&err)),
511 Some(Err(Box::new(err))),
512 from_time,
513 );
514 }
515 None => return (UpsertKey::from_key(Err(&err)), None, from_time),
516 },
517 };
518
519 let key_row = match upsert_envelope.style {
521 UpsertStyle::Debezium { .. }
523 | UpsertStyle::Default(KeyEnvelope::Flattened)
524 | UpsertStyle::ValueErrInline {
525 key_envelope: KeyEnvelope::Flattened,
526 error_column: _,
527 } => key,
528 UpsertStyle::Default(KeyEnvelope::Named(_))
530 | UpsertStyle::ValueErrInline {
531 key_envelope: KeyEnvelope::Named(_),
532 error_column: _,
533 } => {
534 if key.iter().nth(1).is_none() {
535 key
536 } else {
537 row_buf.packer().push_list(key.iter());
538 row_buf.clone()
539 }
540 }
541 UpsertStyle::Default(KeyEnvelope::None)
542 | UpsertStyle::ValueErrInline {
543 key_envelope: KeyEnvelope::None,
544 error_column: _,
545 } => unreachable!(),
546 };
547
548 let key = UpsertKey::from_key(Ok(&key_row));
549
550 let metadata = result.metadata;
551
552 let value = match result.value {
553 Some(Ok(ref row)) => match upsert_envelope.style {
554 UpsertStyle::Debezium { after_idx } => match row.iter().nth(after_idx).unwrap() {
555 Datum::List(after) => {
556 let mut packer = row_buf.packer();
557 packer.extend(after.iter());
558 packer.extend_by_row(&metadata);
559 Some(Ok(row_buf.clone()))
560 }
561 Datum::Null => None,
562 d => panic!("type error: expected record, found {:?}", d),
563 },
564 UpsertStyle::Default(_) => {
565 let mut packer = row_buf.packer();
566 packer.extend_by_row(&key_row);
567 packer.extend_by_row(row);
568 packer.extend_by_row(&metadata);
569 Some(Ok(row_buf.clone()))
570 }
571 UpsertStyle::ValueErrInline { .. } => {
572 let mut packer = row_buf.packer();
573 packer.extend_by_row(&key_row);
574 packer.push(Datum::Null);
576 packer.extend_by_row(row);
577 packer.extend_by_row(&metadata);
578 Some(Ok(row_buf.clone()))
579 }
580 },
581 Some(Err(inner)) => {
582 match upsert_envelope.style {
583 UpsertStyle::ValueErrInline { .. } => {
584 let mut count = 0;
585 let err_string = inner.to_string();
587 let mut packer = row_buf.packer();
588 for datum in key_row.iter() {
589 packer.push(datum);
590 count += 1;
591 }
592 packer.push_list(iter::once(Datum::String(&err_string)));
594 count += 1;
595 let metadata_len = metadata.as_row_ref().iter().count();
596 packer.extend(
598 iter::repeat(Datum::Null)
599 .take(upsert_envelope.source_arity - count - metadata_len),
600 );
601 packer.extend_by_row(&metadata);
602 Some(Ok(row_buf.clone()))
603 }
604 _ => Some(Err(Box::new(UpsertError::Value(UpsertValueError {
605 for_key: key_row,
606 inner,
607 })))),
608 }
609 }
610 None => None,
611 };
612
613 (key, value, from_time)
614 })
615}
616
617fn flatten_results_prepend_keys<'scope, T: Timestamp>(
619 none_envelope: &NoneEnvelope,
620 results: VecCollection<'scope, T, KV, Diff>,
621) -> VecCollection<'scope, T, Result<Row, DataflowError>, Diff> {
622 let NoneEnvelope {
623 key_envelope,
624 key_arity,
625 } = none_envelope;
626
627 let null_key_columns = Row::pack_slice(&vec![Datum::Null; *key_arity]);
628
629 match key_envelope {
630 KeyEnvelope::None => {
631 results.flat_map(|KV { val, .. }| val.map(|result| result.map_err(Into::into)))
632 }
633 KeyEnvelope::Flattened => results
634 .flat_map(raise_key_value_errors)
635 .map(move |maybe_kv| {
636 maybe_kv.map(|(key, value)| {
637 let mut key = key.unwrap_or_else(|| null_key_columns.clone());
638 RowPacker::for_existing_row(&mut key).extend_by_row(&value);
639 key
640 })
641 }),
642 KeyEnvelope::Named(_) => {
643 results
644 .flat_map(raise_key_value_errors)
645 .map(move |maybe_kv| {
646 maybe_kv.map(|(key, value)| {
647 let mut key = key.unwrap_or_else(|| null_key_columns.clone());
648 let row = if key.iter().nth(1).is_none() {
651 RowPacker::for_existing_row(&mut key).extend_by_row(&value);
652 key
653 } else {
654 let mut new_row = Row::default();
655 let mut packer = new_row.packer();
656 packer.push_list(key.iter());
657 packer.extend_by_row(&value);
658 new_row
659 };
660 row
661 })
662 })
663 }
664 }
665}
666
667fn raise_key_value_errors(
669 KV { key, val }: KV,
670) -> Option<Result<(Option<Row>, Row), DataflowError>> {
671 match (key, val) {
672 (Some(Ok(key)), Some(Ok(value))) => Some(Ok((Some(key), value))),
673 (None, Some(Ok(value))) => Some(Ok((None, value))),
674 (_, Some(Err(e))) => Some(Err(e.into())),
676 (Some(Err(e)), _) => Some(Err(e.into())),
677 (None, None) => None,
678 _ => Some(Err(DataflowError::from(EnvelopeError::Flat(
680 "Value not present for message".into(),
681 )))),
682 }
683}
684
685#[cfg(test)]
686mod test {
687 use super::*;
688
689 #[mz_ore::test]
690 fn test_no_default() {
691 let config = StorageMaxInflightBytesConfig {
692 max_inflight_bytes_default: None,
693 max_inflight_bytes_cluster_size_fraction: Some(0.5),
694 disk_only: false,
695 };
696 let memory_limit = Some(1000);
697
698 let backpressure_inflight_bytes_limit =
699 get_backpressure_max_inflight_bytes(&config, &memory_limit);
700
701 assert_eq!(backpressure_inflight_bytes_limit, None)
702 }
703
704 #[mz_ore::test]
705 fn test_no_matching_size() {
706 let config = StorageMaxInflightBytesConfig {
707 max_inflight_bytes_default: Some(10000),
708 max_inflight_bytes_cluster_size_fraction: Some(0.5),
709 disk_only: false,
710 };
711
712 let backpressure_inflight_bytes_limit = get_backpressure_max_inflight_bytes(&config, &None);
713
714 assert_eq!(
715 backpressure_inflight_bytes_limit,
716 config.max_inflight_bytes_default
717 )
718 }
719
720 #[mz_ore::test]
721 fn test_calculated_cluster_limit() {
722 let config = StorageMaxInflightBytesConfig {
723 max_inflight_bytes_default: Some(10000),
724 max_inflight_bytes_cluster_size_fraction: Some(0.5),
725 disk_only: false,
726 };
727 let memory_limit = Some(2000);
728
729 let backpressure_inflight_bytes_limit =
730 get_backpressure_max_inflight_bytes(&config, &memory_limit);
731
732 assert_eq!(backpressure_inflight_bytes_limit, Some(1000));
734 }
735}