1use std::fmt::Debug;
13use std::ops::ControlFlow::{self, Break, Continue};
14use std::sync::Arc;
15use std::time::{Duration, Instant, SystemTime};
16
17use differential_dataflow::difference::Monoid;
18use differential_dataflow::lattice::Lattice;
19use futures::FutureExt;
20use futures::future::{self, BoxFuture};
21use mz_dyncfg::{Config, ConfigSet};
22use mz_ore::cast::CastFrom;
23use mz_ore::error::ErrorExt;
24#[allow(unused_imports)] use mz_ore::fmt::FormatBuffer;
26use mz_ore::task::JoinHandle;
27use mz_ore::{assert_none, soft_assert_no_log};
28use mz_persist::location::{ExternalError, Indeterminate, SeqNo};
29use mz_persist::retry::Retry;
30use mz_persist_types::schema::SchemaId;
31use mz_persist_types::{Codec, Codec64, Opaque};
32use semver::Version;
33use timely::PartialOrder;
34use timely::progress::{Antichain, Timestamp};
35use tracing::{Instrument, debug, info, trace_span, warn};
36
37use crate::async_runtime::IsolatedRuntime;
38use crate::batch::INLINE_WRITES_TOTAL_MAX_BYTES;
39use crate::cache::StateCache;
40use crate::cfg::RetryParameters;
41use crate::critical::CriticalReaderId;
42use crate::error::{CodecMismatch, InvalidUsage};
43use crate::internal::apply::Applier;
44use crate::internal::compact::CompactReq;
45use crate::internal::gc::GarbageCollector;
46use crate::internal::maintenance::{RoutineMaintenance, WriterMaintenance};
47use crate::internal::metrics::{CmdMetrics, Metrics, MetricsRetryStream, RetryMetrics};
48use crate::internal::paths::PartialRollupKey;
49use crate::internal::state::{
50 CompareAndAppendBreak, CriticalReaderState, HandleDebugState, HollowBatch, HollowRollup,
51 IdempotencyToken, LeasedReaderState, NoOpStateTransition, Since, SnapshotErr, StateCollections,
52 Upper,
53};
54use crate::internal::state_versions::StateVersions;
55use crate::internal::trace::{ApplyMergeResult, FueledMergeRes};
56use crate::internal::watch::StateWatch;
57use crate::read::{LeasedReaderId, READER_LEASE_DURATION};
58use crate::rpc::PubSubSender;
59use crate::schema::CaESchema;
60use crate::write::WriterId;
61use crate::{Diagnostics, PersistConfig, ShardId};
62
63#[derive(Debug)]
64pub struct Machine<K, V, T, D> {
65 pub(crate) applier: Applier<K, V, T, D>,
66 pub(crate) isolated_runtime: Arc<IsolatedRuntime>,
67}
68
69impl<K, V, T: Clone, D> Clone for Machine<K, V, T, D> {
71 fn clone(&self) -> Self {
72 Self {
73 applier: self.applier.clone(),
74 isolated_runtime: Arc::clone(&self.isolated_runtime),
75 }
76 }
77}
78
79pub(crate) const CLAIM_UNCLAIMED_COMPACTIONS: Config<bool> = Config::new(
80 "persist_claim_unclaimed_compactions",
81 false,
82 "If an append doesn't result in a compaction request, but there is some uncompacted batch \
83 in state, compact that instead.",
84);
85
86pub(crate) const CLAIM_COMPACTION_PERCENT: Config<usize> = Config::new(
87 "persist_claim_compaction_percent",
88 100,
89 "Claim a compaction with the given percent chance, if claiming compactions is enabled. \
90 (If over 100, we'll always claim at least one; for example, if set to 365, we'll claim at least \
91 three and have a 65% chance of claiming a fourth.)",
92);
93
94pub(crate) const CLAIM_COMPACTION_MIN_VERSION: Config<String> = Config::new(
95 "persist_claim_compaction_min_version",
96 String::new(),
97 "If set to a valid version string, compact away any earlier versions if possible.",
98);
99
100impl<K, V, T, D> Machine<K, V, T, D>
101where
102 K: Debug + Codec,
103 V: Debug + Codec,
104 T: Timestamp + Lattice + Codec64 + Sync,
105 D: Monoid + Codec64,
106{
107 pub async fn new(
108 cfg: PersistConfig,
109 shard_id: ShardId,
110 metrics: Arc<Metrics>,
111 state_versions: Arc<StateVersions>,
112 shared_states: Arc<StateCache>,
113 pubsub_sender: Arc<dyn PubSubSender>,
114 isolated_runtime: Arc<IsolatedRuntime>,
115 diagnostics: Diagnostics,
116 ) -> Result<Self, Box<CodecMismatch>> {
117 let applier = Applier::new(
118 cfg,
119 shard_id,
120 metrics,
121 state_versions,
122 shared_states,
123 pubsub_sender,
124 diagnostics,
125 )
126 .await?;
127 Ok(Machine {
128 applier,
129 isolated_runtime,
130 })
131 }
132
133 pub fn shard_id(&self) -> ShardId {
134 self.applier.shard_id
135 }
136
137 pub fn seqno(&self) -> SeqNo {
138 self.applier.seqno()
139 }
140
141 pub async fn add_rollup_for_current_seqno(&self) -> RoutineMaintenance {
142 let rollup = self.applier.write_rollup_for_state().await;
143 let Some(rollup) = rollup else {
144 return RoutineMaintenance::default();
145 };
146
147 let (applied, maintenance) = self.add_rollup((rollup.seqno, &rollup.to_hollow())).await;
148 if !applied {
149 self.applier
152 .state_versions
153 .delete_rollup(&rollup.shard_id, &rollup.key)
154 .await;
155 }
156 maintenance
157 }
158
159 pub async fn add_rollup(
160 &self,
161 add_rollup: (SeqNo, &HollowRollup),
162 ) -> (bool, RoutineMaintenance) {
163 let mut applied_ever_true = false;
166 let metrics = Arc::clone(&self.applier.metrics);
167 let (_seqno, _applied, maintenance) = self
168 .apply_unbatched_idempotent_cmd(&metrics.cmds.add_rollup, |_, _, state| {
169 let ret = state.add_rollup(add_rollup);
170 if let Continue(applied) = ret {
171 applied_ever_true = applied_ever_true || applied;
172 }
173 ret
174 })
175 .await;
176 (applied_ever_true, maintenance)
177 }
178
179 pub async fn remove_rollups(
180 &self,
181 remove_rollups: &[(SeqNo, PartialRollupKey)],
182 ) -> (Vec<SeqNo>, RoutineMaintenance) {
183 let metrics = Arc::clone(&self.applier.metrics);
184 let (_seqno, removed_rollup_seqnos, maintenance) = self
185 .apply_unbatched_idempotent_cmd(&metrics.cmds.remove_rollups, |_, _, state| {
186 state.remove_rollups(remove_rollups)
187 })
188 .await;
189 (removed_rollup_seqnos, maintenance)
190 }
191
192 pub async fn upgrade_version(&self) -> Result<RoutineMaintenance, Version> {
195 let metrics = Arc::clone(&self.applier.metrics);
196 let (_seqno, upgrade_result, maintenance) = self
197 .apply_unbatched_idempotent_cmd(&metrics.cmds.remove_rollups, |_, cfg, state| {
198 if state.version <= cfg.build_version {
199 state.version = cfg.build_version.clone();
202 Continue(Ok(()))
203 } else {
204 Break(NoOpStateTransition(Err(state.version.clone())))
205 }
206 })
207 .await;
208
209 match upgrade_result {
210 Ok(()) => Ok(maintenance),
211 Err(version) => {
212 soft_assert_no_log!(
213 maintenance.is_empty(),
214 "should not generate maintenance on failed upgrade"
215 );
216 Err(version)
217 }
218 }
219 }
220
221 pub async fn register_leased_reader(
222 &self,
223 reader_id: &LeasedReaderId,
224 purpose: &str,
225 lease_duration: Duration,
226 heartbeat_timestamp_ms: u64,
227 use_critical_since: bool,
228 ) -> (LeasedReaderState<T>, RoutineMaintenance) {
229 let metrics = Arc::clone(&self.applier.metrics);
230 let (_seqno, (reader_state, seqno_since), maintenance) = self
231 .apply_unbatched_idempotent_cmd(&metrics.cmds.register, |seqno, cfg, state| {
232 state.register_leased_reader(
233 &cfg.hostname,
234 reader_id,
235 purpose,
236 seqno,
237 lease_duration,
238 heartbeat_timestamp_ms,
239 use_critical_since,
240 )
241 })
242 .await;
243 debug_assert!(
252 reader_state.seqno >= seqno_since,
253 "{} vs {}",
254 reader_state.seqno,
255 seqno_since,
256 );
257 (reader_state, maintenance)
258 }
259
260 pub async fn register_critical_reader<O: Opaque + Codec64>(
261 &self,
262 reader_id: &CriticalReaderId,
263 purpose: &str,
264 ) -> (CriticalReaderState<T>, RoutineMaintenance) {
265 let metrics = Arc::clone(&self.applier.metrics);
266 let (_seqno, state, maintenance) = self
267 .apply_unbatched_idempotent_cmd(&metrics.cmds.register, |_seqno, cfg, state| {
268 state.register_critical_reader::<O>(&cfg.hostname, reader_id, purpose)
269 })
270 .await;
271 (state, maintenance)
272 }
273
274 pub async fn register_schema(
275 &self,
276 key_schema: &K::Schema,
277 val_schema: &V::Schema,
278 ) -> (Option<SchemaId>, RoutineMaintenance) {
279 let metrics = Arc::clone(&self.applier.metrics);
280 let (_seqno, state, maintenance) = self
281 .apply_unbatched_idempotent_cmd(&metrics.cmds.register, |_seqno, _cfg, state| {
282 state.register_schema::<K, V>(key_schema, val_schema)
283 })
284 .await;
285 (state, maintenance)
286 }
287
288 pub async fn spine_exert(&self, fuel: usize) -> (Vec<CompactReq<T>>, RoutineMaintenance) {
289 if fuel == 0 || self.applier.all_batches().len() < 2 {
291 return (Vec::new(), RoutineMaintenance::default());
292 }
293
294 let metrics = Arc::clone(&self.applier.metrics);
295 let (_seqno, reqs, maintenance) = self
296 .apply_unbatched_idempotent_cmd(&metrics.cmds.spine_exert, |_seqno, _cfg, state| {
297 state.spine_exert(fuel)
298 })
299 .await;
300 let reqs = reqs
301 .into_iter()
302 .map(|req| CompactReq {
303 shard_id: self.shard_id(),
304 desc: req.desc,
305 inputs: req.inputs,
306 })
307 .collect();
308 (reqs, maintenance)
309 }
310
311 pub async fn compare_and_append(
312 &self,
313 batch: &HollowBatch<T>,
314 writer_id: &WriterId,
315 debug_info: &HandleDebugState,
316 heartbeat_timestamp_ms: u64,
317 ) -> CompareAndAppendRes<T> {
318 let idempotency_token = IdempotencyToken::new();
319 loop {
320 let res = self
321 .compare_and_append_idempotent(
322 batch,
323 writer_id,
324 heartbeat_timestamp_ms,
325 &idempotency_token,
326 debug_info,
327 None,
328 )
329 .await;
330 match res {
331 CompareAndAppendRes::Success(seqno, maintenance) => {
332 return CompareAndAppendRes::Success(seqno, maintenance);
333 }
334 CompareAndAppendRes::InvalidUsage(x) => {
335 return CompareAndAppendRes::InvalidUsage(x);
336 }
337 CompareAndAppendRes::InlineBackpressure => {
338 return CompareAndAppendRes::InlineBackpressure;
339 }
340 CompareAndAppendRes::UpperMismatch(seqno, _current_upper) => {
341 self.applier.fetch_and_update_state(Some(seqno)).await;
348 let (current_seqno, current_upper) =
349 self.applier.upper(|seqno, upper| (seqno, upper.clone()));
350
351 if ¤t_upper != batch.desc.lower() {
354 return CompareAndAppendRes::UpperMismatch(current_seqno, current_upper);
355 } else {
356 }
359 }
360 }
361 }
362 }
363
364 async fn compare_and_append_idempotent(
365 &self,
366 batch: &HollowBatch<T>,
367 writer_id: &WriterId,
368 heartbeat_timestamp_ms: u64,
369 idempotency_token: &IdempotencyToken,
370 debug_info: &HandleDebugState,
371 mut indeterminate: Option<Indeterminate>,
375 ) -> CompareAndAppendRes<T> {
376 let metrics = Arc::clone(&self.applier.metrics);
377 let lease_duration_ms = self
378 .applier
379 .cfg
380 .writer_lease_duration
381 .as_millis()
382 .try_into()
383 .expect("reasonable duration");
384 let mut retry = self
467 .applier
468 .metrics
469 .retries
470 .compare_and_append_idempotent
471 .stream(Retry::persist_defaults(SystemTime::now()).into_retry_stream());
472 let mut writer_was_present = false;
473 loop {
474 let cmd_res = self
475 .applier
476 .apply_unbatched_cmd(&metrics.cmds.compare_and_append, |_, cfg, state| {
477 writer_was_present = state.writers.contains_key(writer_id);
478 state.compare_and_append(
479 batch,
480 writer_id,
481 heartbeat_timestamp_ms,
482 lease_duration_ms,
483 idempotency_token,
484 debug_info,
485 INLINE_WRITES_TOTAL_MAX_BYTES.get(cfg),
486 if CLAIM_UNCLAIMED_COMPACTIONS.get(cfg) {
487 CLAIM_COMPACTION_PERCENT.get(cfg)
488 } else {
489 0
490 },
491 Version::parse(&CLAIM_COMPACTION_MIN_VERSION.get(cfg))
492 .ok()
493 .as_ref(),
494 )
495 })
496 .await;
497 let (seqno, res, routine) = match cmd_res {
498 Ok(x) => x,
499 Err(err) => {
500 info!(
503 "compare_and_append received an indeterminate error, retrying in {:?}: {}",
504 retry.next_sleep(),
505 err
506 );
507 if indeterminate.is_none() {
508 indeterminate = Some(err);
509 }
510 retry = retry.sleep().await;
511 continue;
512 }
513 };
514 match res {
515 Ok(merge_reqs) => {
516 let mut compact_reqs = Vec::with_capacity(merge_reqs.len());
519 for req in merge_reqs {
520 let req = CompactReq {
521 shard_id: self.shard_id(),
522 desc: req.desc,
523 inputs: req.inputs,
524 };
525 compact_reqs.push(req);
526 }
527 let writer_maintenance = WriterMaintenance {
528 routine,
529 compaction: compact_reqs,
530 };
531
532 if !writer_was_present {
533 metrics.state.writer_added.inc();
534 }
535 for part in &batch.parts {
536 if part.is_inline() {
537 let bytes = u64::cast_from(part.inline_bytes());
538 metrics.inline.part_commit_bytes.inc_by(bytes);
539 metrics.inline.part_commit_count.inc();
540 }
541 }
542 return CompareAndAppendRes::Success(seqno, writer_maintenance);
543 }
544 Err(CompareAndAppendBreak::AlreadyCommitted) => {
545 assert!(indeterminate.is_some());
549 self.applier.metrics.cmds.compare_and_append_noop.inc();
550 if !writer_was_present {
551 metrics.state.writer_added.inc();
552 }
553 return CompareAndAppendRes::Success(seqno, WriterMaintenance::default());
554 }
555 Err(CompareAndAppendBreak::InvalidUsage(err)) => {
556 assert_none!(indeterminate);
561 return CompareAndAppendRes::InvalidUsage(err);
562 }
563 Err(CompareAndAppendBreak::InlineBackpressure) => {
564 return CompareAndAppendRes::InlineBackpressure;
567 }
568 Err(CompareAndAppendBreak::Upper {
569 shard_upper,
570 writer_upper,
571 }) => {
572 assert!(
577 PartialOrder::less_equal(&writer_upper, &shard_upper),
578 "{:?} vs {:?}",
579 &writer_upper,
580 &shard_upper
581 );
582 if PartialOrder::less_than(&writer_upper, batch.desc.upper()) {
583 return CompareAndAppendRes::UpperMismatch(seqno, shard_upper);
587 }
588 if indeterminate.is_none() {
589 return CompareAndAppendRes::UpperMismatch(seqno, shard_upper);
594 }
595 panic!(
605 concat!(
606 "cannot distinguish compare_and_append success or failure ",
607 "caa_lower={:?} caa_upper={:?} writer_upper={:?} shard_upper={:?} err={:?}"
608 ),
609 batch.desc.lower().elements(),
610 batch.desc.upper().elements(),
611 writer_upper.elements(),
612 shard_upper.elements(),
613 indeterminate,
614 );
615 }
616 };
617 }
618 }
619
620 pub async fn downgrade_since(
621 &self,
622 reader_id: &LeasedReaderId,
623 outstanding_seqno: Option<SeqNo>,
624 new_since: &Antichain<T>,
625 heartbeat_timestamp_ms: u64,
626 ) -> (SeqNo, Since<T>, RoutineMaintenance) {
627 let metrics = Arc::clone(&self.applier.metrics);
628 self.apply_unbatched_idempotent_cmd(&metrics.cmds.downgrade_since, |seqno, _cfg, state| {
629 state.downgrade_since(
630 reader_id,
631 seqno,
632 outstanding_seqno,
633 new_since,
634 heartbeat_timestamp_ms,
635 )
636 })
637 .await
638 }
639
640 pub async fn compare_and_downgrade_since<O: Opaque + Codec64>(
641 &self,
642 reader_id: &CriticalReaderId,
643 expected_opaque: &O,
644 (new_opaque, new_since): (&O, &Antichain<T>),
645 ) -> (Result<Since<T>, (O, Since<T>)>, RoutineMaintenance) {
646 let metrics = Arc::clone(&self.applier.metrics);
647 let (_seqno, res, maintenance) = self
648 .apply_unbatched_idempotent_cmd(
649 &metrics.cmds.compare_and_downgrade_since,
650 |_seqno, _cfg, state| {
651 state.compare_and_downgrade_since::<O>(
652 reader_id,
653 expected_opaque,
654 (new_opaque, new_since),
655 )
656 },
657 )
658 .await;
659
660 match res {
661 Ok(since) => (Ok(since), maintenance),
662 Err((opaque, since)) => (Err((opaque, since)), maintenance),
663 }
664 }
665
666 pub async fn heartbeat_leased_reader(
667 &self,
668 reader_id: &LeasedReaderId,
669 heartbeat_timestamp_ms: u64,
670 ) -> (SeqNo, bool, RoutineMaintenance) {
671 let metrics = Arc::clone(&self.applier.metrics);
672 let (seqno, existed, maintenance) = self
673 .apply_unbatched_idempotent_cmd(&metrics.cmds.heartbeat_reader, |_, _, state| {
674 state.heartbeat_leased_reader(reader_id, heartbeat_timestamp_ms)
675 })
676 .await;
677 (seqno, existed, maintenance)
678 }
679
680 pub async fn expire_leased_reader(
681 &self,
682 reader_id: &LeasedReaderId,
683 ) -> (SeqNo, RoutineMaintenance) {
684 let metrics = Arc::clone(&self.applier.metrics);
685 let (seqno, _existed, maintenance) = self
686 .apply_unbatched_idempotent_cmd(&metrics.cmds.expire_reader, |_, _, state| {
687 state.expire_leased_reader(reader_id)
688 })
689 .await;
690 (seqno, maintenance)
691 }
692
693 #[allow(dead_code)] pub async fn expire_critical_reader(
695 &self,
696 reader_id: &CriticalReaderId,
697 ) -> (SeqNo, RoutineMaintenance) {
698 let metrics = Arc::clone(&self.applier.metrics);
699 let (seqno, _existed, maintenance) = self
700 .apply_unbatched_idempotent_cmd(&metrics.cmds.expire_reader, |_, _, state| {
701 state.expire_critical_reader(reader_id)
702 })
703 .await;
704 (seqno, maintenance)
705 }
706
707 pub async fn expire_writer(&self, writer_id: &WriterId) -> (SeqNo, RoutineMaintenance) {
708 let metrics = Arc::clone(&self.applier.metrics);
709 let (seqno, _existed, maintenance) = self
710 .apply_unbatched_idempotent_cmd(&metrics.cmds.expire_writer, |_, _, state| {
711 state.expire_writer(writer_id)
712 })
713 .await;
714 metrics.state.writer_removed.inc();
715 (seqno, maintenance)
716 }
717
718 pub fn is_finalized(&self) -> bool {
719 self.applier.is_finalized()
720 }
721
722 pub fn get_schema(&self, schema_id: SchemaId) -> Option<(K::Schema, V::Schema)> {
724 self.applier.get_schema(schema_id)
725 }
726
727 pub fn latest_schema(&self) -> Option<(SchemaId, K::Schema, V::Schema)> {
729 self.applier.latest_schema()
730 }
731
732 pub fn find_schema(&self, key_schema: &K::Schema, val_schema: &V::Schema) -> Option<SchemaId> {
734 self.applier.find_schema(key_schema, val_schema)
735 }
736
737 pub async fn compare_and_evolve_schema(
741 &self,
742 expected: SchemaId,
743 key_schema: &K::Schema,
744 val_schema: &V::Schema,
745 ) -> (CaESchema<K, V>, RoutineMaintenance) {
746 let metrics = Arc::clone(&self.applier.metrics);
747 let (_seqno, state, maintenance) = self
748 .apply_unbatched_idempotent_cmd(
749 &metrics.cmds.compare_and_evolve_schema,
750 |_seqno, _cfg, state| {
751 state.compare_and_evolve_schema::<K, V>(expected, key_schema, val_schema)
752 },
753 )
754 .await;
755 (state, maintenance)
756 }
757
758 async fn tombstone_step(&self) -> Result<(bool, RoutineMaintenance), InvalidUsage<T>> {
759 let metrics = Arc::clone(&self.applier.metrics);
760 let mut retry = self
761 .applier
762 .metrics
763 .retries
764 .idempotent_cmd
765 .stream(Retry::persist_defaults(SystemTime::now()).into_retry_stream());
766 loop {
767 let res = self
768 .applier
769 .apply_unbatched_cmd(&metrics.cmds.become_tombstone, |_, _, state| {
770 state.become_tombstone_and_shrink()
771 })
772 .await;
773 let err = match res {
774 Ok((_seqno, Ok(()), maintenance)) => return Ok((true, maintenance)),
775 Ok((_seqno, Err(NoOpStateTransition(())), maintenance)) => {
776 return Ok((false, maintenance));
777 }
778 Err(err) => err,
779 };
780 if retry.attempt() >= INFO_MIN_ATTEMPTS {
781 info!(
782 "become_tombstone received an indeterminate error, retrying in {:?}: {}",
783 retry.next_sleep(),
784 err
785 );
786 } else {
787 debug!(
788 "become_tombstone received an indeterminate error, retrying in {:?}: {}",
789 retry.next_sleep(),
790 err
791 );
792 }
793 retry = retry.sleep().await;
794 }
795 }
796
797 pub async fn become_tombstone(&self) -> Result<RoutineMaintenance, InvalidUsage<T>> {
798 self.applier.check_since_upper_both_empty()?;
799
800 let mut maintenance = RoutineMaintenance::default();
801
802 loop {
803 let (made_progress, more_maintenance) = self.tombstone_step().await?;
804 maintenance.merge(more_maintenance);
805 if !made_progress {
806 break;
807 }
808 }
809
810 Ok(maintenance)
811 }
812
813 pub async fn snapshot(&self, as_of: &Antichain<T>) -> Result<Vec<HollowBatch<T>>, Since<T>> {
814 let start = Instant::now();
815 let (mut seqno, mut upper) = match self.applier.snapshot(as_of) {
816 Ok(x) => return Ok(x),
817 Err(SnapshotErr::AsOfNotYetAvailable(seqno, Upper(upper))) => (seqno, upper),
818 Err(SnapshotErr::AsOfHistoricalDistinctionsLost(Since(since))) => {
819 return Err(Since(since));
820 }
821 };
822
823 let mut watch = self.applier.watch();
826 let watch = &mut watch;
827 let sleeps = self
828 .applier
829 .metrics
830 .retries
831 .snapshot
832 .stream(Retry::persist_defaults(SystemTime::now()).into_retry_stream());
833
834 enum Wake<'a, K, V, T, D> {
835 Watch(&'a mut StateWatch<K, V, T, D>),
836 Sleep(MetricsRetryStream),
837 }
838 let mut watch_fut = std::pin::pin!(
839 watch
840 .wait_for_seqno_ge(seqno.next())
841 .map(Wake::Watch)
842 .instrument(trace_span!("snapshot::watch")),
843 );
844 let mut sleep_fut = std::pin::pin!(
845 sleeps
846 .sleep()
847 .map(Wake::Sleep)
848 .instrument(trace_span!("snapshot::sleep")),
849 );
850
851 let mut logged_at_info = false;
855 loop {
856 if !logged_at_info && start.elapsed() >= Duration::from_millis(1024) {
860 logged_at_info = true;
861 info!(
862 "snapshot {} {} as of {:?} not yet available for {} upper {:?}",
863 self.applier.shard_metrics.name,
864 self.shard_id(),
865 as_of.elements(),
866 seqno,
867 upper.elements(),
868 );
869 } else {
870 debug!(
871 "snapshot {} {} as of {:?} not yet available for {} upper {:?}",
872 self.applier.shard_metrics.name,
873 self.shard_id(),
874 as_of.elements(),
875 seqno,
876 upper.elements(),
877 );
878 }
879
880 let wake = match future::select(watch_fut.as_mut(), sleep_fut.as_mut()).await {
881 future::Either::Left((wake, _)) => wake,
882 future::Either::Right((wake, _)) => wake,
883 };
884 match &wake {
888 Wake::Watch(_) => self.applier.metrics.watch.snapshot_woken_via_watch.inc(),
889 Wake::Sleep(_) => {
890 self.applier.metrics.watch.snapshot_woken_via_sleep.inc();
891 self.applier.fetch_and_update_state(Some(seqno)).await;
892 }
893 }
894
895 (seqno, upper) = match self.applier.snapshot(as_of) {
896 Ok(x) => {
897 if logged_at_info {
898 info!(
899 "snapshot {} {} as of {:?} now available",
900 self.applier.shard_metrics.name,
901 self.shard_id(),
902 as_of.elements(),
903 );
904 }
905 return Ok(x);
906 }
907 Err(SnapshotErr::AsOfNotYetAvailable(seqno, Upper(upper))) => {
908 (seqno, upper)
910 }
911 Err(SnapshotErr::AsOfHistoricalDistinctionsLost(Since(since))) => {
912 return Err(Since(since));
913 }
914 };
915
916 match wake {
917 Wake::Watch(watch) => {
918 watch_fut.set(
919 watch
920 .wait_for_seqno_ge(seqno.next())
921 .map(Wake::Watch)
922 .instrument(trace_span!("snapshot::watch")),
923 );
924 }
925 Wake::Sleep(sleeps) => {
926 debug!(
927 "snapshot {} {} sleeping for {:?}",
928 self.applier.shard_metrics.name,
929 self.shard_id(),
930 sleeps.next_sleep()
931 );
932 sleep_fut.set(
933 sleeps
934 .sleep()
935 .map(Wake::Sleep)
936 .instrument(trace_span!("snapshot::sleep")),
937 );
938 }
939 }
940 }
941 }
942
943 pub fn verify_listen(&self, as_of: &Antichain<T>) -> Result<(), Since<T>> {
945 self.applier.verify_listen(as_of)
946 }
947
948 pub async fn next_listen_batch(
949 &self,
950 frontier: &Antichain<T>,
951 watch: &mut StateWatch<K, V, T, D>,
952 reader_id: Option<&LeasedReaderId>,
953 retry: Option<RetryParameters>,
955 ) -> HollowBatch<T> {
956 let mut seqno = match self.applier.next_listen_batch(frontier) {
957 Ok(b) => return b,
958 Err(seqno) => seqno,
959 };
960
961 let retry = retry.unwrap_or_else(|| next_listen_batch_retry_params(&self.applier.cfg));
964 let sleeps = self
965 .applier
966 .metrics
967 .retries
968 .next_listen_batch
969 .stream(retry.into_retry(SystemTime::now()).into_retry_stream());
970
971 enum Wake<'a, K, V, T, D> {
972 Watch(&'a mut StateWatch<K, V, T, D>),
973 Sleep(MetricsRetryStream),
974 }
975 let mut watch_fut = std::pin::pin!(
976 watch
977 .wait_for_seqno_ge(seqno.next())
978 .map(Wake::Watch)
979 .instrument(trace_span!("snapshot::watch"))
980 );
981 let mut sleep_fut = std::pin::pin!(
982 sleeps
983 .sleep()
984 .map(Wake::Sleep)
985 .instrument(trace_span!("snapshot::sleep"))
986 );
987
988 loop {
989 let wake = match future::select(watch_fut.as_mut(), sleep_fut.as_mut()).await {
990 future::Either::Left((wake, _)) => wake,
991 future::Either::Right((wake, _)) => wake,
992 };
993 match &wake {
997 Wake::Watch(_) => self.applier.metrics.watch.listen_woken_via_watch.inc(),
998 Wake::Sleep(_) => {
999 self.applier.metrics.watch.listen_woken_via_sleep.inc();
1000 self.applier.fetch_and_update_state(Some(seqno)).await;
1001 }
1002 }
1003
1004 seqno = match self.applier.next_listen_batch(frontier) {
1005 Ok(b) => {
1006 match &wake {
1007 Wake::Watch(_) => {
1008 self.applier.metrics.watch.listen_resolved_via_watch.inc()
1009 }
1010 Wake::Sleep(_) => {
1011 self.applier.metrics.watch.listen_resolved_via_sleep.inc()
1012 }
1013 }
1014 return b;
1015 }
1016 Err(seqno) => seqno,
1017 };
1018
1019 match wake {
1022 Wake::Watch(watch) => {
1023 watch_fut.set(
1024 watch
1025 .wait_for_seqno_ge(seqno.next())
1026 .map(Wake::Watch)
1027 .instrument(trace_span!("snapshot::watch")),
1028 );
1029 }
1030 Wake::Sleep(sleeps) => {
1031 debug!(
1032 "{:?}: {} {} next_listen_batch didn't find new data, retrying in {:?}",
1033 reader_id,
1034 self.applier.shard_metrics.name,
1035 self.shard_id(),
1036 sleeps.next_sleep()
1037 );
1038 sleep_fut.set(
1039 sleeps
1040 .sleep()
1041 .map(Wake::Sleep)
1042 .instrument(trace_span!("snapshot::sleep")),
1043 );
1044 }
1045 }
1046 }
1047 }
1048
1049 async fn apply_unbatched_idempotent_cmd<
1050 R,
1051 WorkFn: FnMut(
1052 SeqNo,
1053 &PersistConfig,
1054 &mut StateCollections<T>,
1055 ) -> ControlFlow<NoOpStateTransition<R>, R>,
1056 >(
1057 &self,
1058 cmd: &CmdMetrics,
1059 mut work_fn: WorkFn,
1060 ) -> (SeqNo, R, RoutineMaintenance) {
1061 let mut retry = self
1062 .applier
1063 .metrics
1064 .retries
1065 .idempotent_cmd
1066 .stream(Retry::persist_defaults(SystemTime::now()).into_retry_stream());
1067 loop {
1068 match self.applier.apply_unbatched_cmd(cmd, &mut work_fn).await {
1069 Ok((seqno, x, maintenance)) => match x {
1070 Ok(x) => {
1071 return (seqno, x, maintenance);
1072 }
1073 Err(NoOpStateTransition(x)) => {
1074 return (seqno, x, maintenance);
1075 }
1076 },
1077 Err(err) => {
1078 if retry.attempt() >= INFO_MIN_ATTEMPTS {
1079 info!(
1080 "apply_unbatched_idempotent_cmd {} received an indeterminate error, retrying in {:?}: {}",
1081 cmd.name,
1082 retry.next_sleep(),
1083 err
1084 );
1085 } else {
1086 debug!(
1087 "apply_unbatched_idempotent_cmd {} received an indeterminate error, retrying in {:?}: {}",
1088 cmd.name,
1089 retry.next_sleep(),
1090 err
1091 );
1092 }
1093 retry = retry.sleep().await;
1094 continue;
1095 }
1096 }
1097 }
1098 }
1099}
1100
1101impl<K, V, T, D> Machine<K, V, T, D>
1102where
1103 K: Debug + Codec,
1104 V: Debug + Codec,
1105 T: Timestamp + Lattice + Codec64 + Sync,
1106 D: Monoid + Codec64 + PartialEq,
1107{
1108 pub async fn merge_res(
1109 &self,
1110 res: &FueledMergeRes<T>,
1111 ) -> (ApplyMergeResult, RoutineMaintenance) {
1112 let metrics = Arc::clone(&self.applier.metrics);
1113
1114 let mut merge_result_ever_applied = ApplyMergeResult::NotAppliedNoMatch;
1143 let (_seqno, _apply_merge_result, maintenance) = self
1144 .apply_unbatched_idempotent_cmd(&metrics.cmds.merge_res, |_, _, state| {
1145 let ret = state.apply_merge_res::<D>(res, &Arc::clone(&metrics).columnar);
1146 if let Continue(result) = ret {
1147 if result.applied() {
1149 merge_result_ever_applied = result;
1150 }
1151 if result.matched() && !result.applied() && !merge_result_ever_applied.applied()
1154 {
1155 merge_result_ever_applied = result;
1156 }
1157 }
1158 ret
1159 })
1160 .await;
1161 (merge_result_ever_applied, maintenance)
1162 }
1163}
1164
1165pub(crate) struct ExpireFn(
1166 pub(crate) Box<dyn FnOnce() -> BoxFuture<'static, ()> + Send + Sync + 'static>,
1171);
1172
1173impl Debug for ExpireFn {
1174 fn fmt(&self, f: &mut std::fmt::Formatter<'_>) -> std::fmt::Result {
1175 f.debug_struct("ExpireFn").finish_non_exhaustive()
1176 }
1177}
1178
1179#[derive(Debug)]
1180pub(crate) enum CompareAndAppendRes<T> {
1181 Success(SeqNo, WriterMaintenance<T>),
1182 InvalidUsage(InvalidUsage<T>),
1183 UpperMismatch(SeqNo, Antichain<T>),
1184 InlineBackpressure,
1185}
1186
1187#[cfg(test)]
1188impl<T: Debug> CompareAndAppendRes<T> {
1189 #[track_caller]
1190 fn unwrap(self) -> (SeqNo, WriterMaintenance<T>) {
1191 match self {
1192 CompareAndAppendRes::Success(seqno, maintenance) => (seqno, maintenance),
1193 x => panic!("{:?}", x),
1194 }
1195 }
1196}
1197
1198impl<K, V, T, D> Machine<K, V, T, D>
1199where
1200 K: Debug + Codec,
1201 V: Debug + Codec,
1202 T: Timestamp + Lattice + Codec64 + Sync,
1203 D: Monoid + Codec64 + Send + Sync,
1204{
1205 #[allow(clippy::unused_async)]
1206 pub async fn start_reader_heartbeat_tasks(
1207 self,
1208 reader_id: LeasedReaderId,
1209 gc: GarbageCollector<K, V, T, D>,
1210 ) -> Vec<JoinHandle<()>> {
1211 let mut ret = Vec::new();
1212 let metrics = Arc::clone(&self.applier.metrics);
1213
1214 let name = format!("persist::heartbeat_read({},{})", self.shard_id(), reader_id);
1226 ret.push(mz_ore::task::spawn(|| name, {
1227 let machine = self.clone();
1228 let reader_id = reader_id.clone();
1229 let gc = gc.clone();
1230 metrics
1231 .tasks
1232 .heartbeat_read
1233 .instrument_task(Self::reader_heartbeat_task(machine, reader_id, gc))
1234 }));
1235
1236 let isolated_runtime = Arc::clone(&self.isolated_runtime);
1237 let name = format!(
1238 "persist::heartbeat_read_isolated({},{})",
1239 self.shard_id(),
1240 reader_id
1241 );
1242 ret.push(
1243 isolated_runtime.spawn_named(
1244 || name,
1245 metrics
1246 .tasks
1247 .heartbeat_read
1248 .instrument_task(Self::reader_heartbeat_task(self, reader_id, gc)),
1249 ),
1250 );
1251
1252 ret
1253 }
1254
1255 async fn reader_heartbeat_task(
1256 machine: Self,
1257 reader_id: LeasedReaderId,
1258 gc: GarbageCollector<K, V, T, D>,
1259 ) {
1260 let sleep_duration = READER_LEASE_DURATION.get(&machine.applier.cfg) / 2;
1261 loop {
1262 let before_sleep = Instant::now();
1263 tokio::time::sleep(sleep_duration).await;
1264
1265 let elapsed_since_before_sleeping = before_sleep.elapsed();
1266 if elapsed_since_before_sleeping > sleep_duration + Duration::from_secs(60) {
1267 warn!(
1268 "reader ({}) of shard ({}) went {}s between heartbeats",
1269 reader_id,
1270 machine.shard_id(),
1271 elapsed_since_before_sleeping.as_secs_f64()
1272 );
1273 }
1274
1275 let before_heartbeat = Instant::now();
1276 let (_seqno, existed, maintenance) = machine
1277 .heartbeat_leased_reader(&reader_id, (machine.applier.cfg.now)())
1278 .await;
1279 maintenance.start_performing(&machine, &gc);
1280
1281 let elapsed_since_heartbeat = before_heartbeat.elapsed();
1282 if elapsed_since_heartbeat > Duration::from_secs(60) {
1283 warn!(
1284 "reader ({}) of shard ({}) heartbeat call took {}s",
1285 reader_id,
1286 machine.shard_id(),
1287 elapsed_since_heartbeat.as_secs_f64(),
1288 );
1289 }
1290
1291 if !existed {
1292 warn!(
1300 "heartbeat task for reader ({}) of shard ({}) exiting due to expired lease \
1301 while read handle is live",
1302 reader_id,
1303 machine.shard_id(),
1304 );
1305 return;
1306 }
1307 }
1308 }
1309}
1310
1311pub(crate) const NEXT_LISTEN_BATCH_RETRYER_FIXED_SLEEP: Config<Duration> = Config::new(
1312 "persist_next_listen_batch_retryer_fixed_sleep",
1313 Duration::from_millis(1200), "\
1315 The fixed sleep when polling for new batches from a Listen or Subscribe. Skipped if zero.",
1316);
1317
1318pub(crate) const NEXT_LISTEN_BATCH_RETRYER_INITIAL_BACKOFF: Config<Duration> = Config::new(
1319 "persist_next_listen_batch_retryer_initial_backoff",
1320 Duration::from_millis(100), "The initial backoff when polling for new batches from a Listen or Subscribe.",
1322);
1323
1324pub(crate) const NEXT_LISTEN_BATCH_RETRYER_MULTIPLIER: Config<u32> = Config::new(
1325 "persist_next_listen_batch_retryer_multiplier",
1326 2,
1327 "The backoff multiplier when polling for new batches from a Listen or Subscribe.",
1328);
1329
1330pub(crate) const NEXT_LISTEN_BATCH_RETRYER_CLAMP: Config<Duration> = Config::new(
1331 "persist_next_listen_batch_retryer_clamp",
1332 Duration::from_secs(16), "The backoff clamp duration when polling for new batches from a Listen or Subscribe.",
1334);
1335
1336fn next_listen_batch_retry_params(cfg: &ConfigSet) -> RetryParameters {
1337 RetryParameters {
1338 fixed_sleep: NEXT_LISTEN_BATCH_RETRYER_FIXED_SLEEP.get(cfg),
1339 initial_backoff: NEXT_LISTEN_BATCH_RETRYER_INITIAL_BACKOFF.get(cfg),
1340 multiplier: NEXT_LISTEN_BATCH_RETRYER_MULTIPLIER.get(cfg),
1341 clamp: NEXT_LISTEN_BATCH_RETRYER_CLAMP.get(cfg),
1342 }
1343}
1344
1345pub const INFO_MIN_ATTEMPTS: usize = 3;
1346
1347pub async fn retry_external<R, F, WorkFn>(metrics: &RetryMetrics, mut work_fn: WorkFn) -> R
1348where
1349 F: std::future::Future<Output = Result<R, ExternalError>>,
1350 WorkFn: FnMut() -> F,
1351{
1352 let mut retry = metrics.stream(Retry::persist_defaults(SystemTime::now()).into_retry_stream());
1353 loop {
1354 match work_fn().await {
1355 Ok(x) => {
1356 if retry.attempt() > 0 {
1357 debug!(
1358 "external operation {} succeeded after failing at least once",
1359 metrics.name,
1360 );
1361 }
1362 return x;
1363 }
1364 Err(err) => {
1365 if retry.attempt() >= INFO_MIN_ATTEMPTS {
1366 info!(
1367 "external operation {} failed, retrying in {:?}: {}",
1368 metrics.name,
1369 retry.next_sleep(),
1370 err.display_with_causes()
1371 );
1372 } else {
1373 debug!(
1374 "external operation {} failed, retrying in {:?}: {}",
1375 metrics.name,
1376 retry.next_sleep(),
1377 err.display_with_causes()
1378 );
1379 }
1380 retry = retry.sleep().await;
1381 }
1382 }
1383 }
1384}
1385
1386pub async fn retry_determinate<R, F, WorkFn>(
1387 metrics: &RetryMetrics,
1388 mut work_fn: WorkFn,
1389) -> Result<R, Indeterminate>
1390where
1391 F: std::future::Future<Output = Result<R, ExternalError>>,
1392 WorkFn: FnMut() -> F,
1393{
1394 let mut retry = metrics.stream(Retry::persist_defaults(SystemTime::now()).into_retry_stream());
1395 loop {
1396 match work_fn().await {
1397 Ok(x) => {
1398 if retry.attempt() > 0 {
1399 debug!(
1400 "external operation {} succeeded after failing at least once",
1401 metrics.name,
1402 );
1403 }
1404 return Ok(x);
1405 }
1406 Err(ExternalError::Determinate(err)) => {
1407 debug!(
1416 "external operation {} failed, retrying in {:?}: {}",
1417 metrics.name,
1418 retry.next_sleep(),
1419 err.display_with_causes()
1420 );
1421 retry = retry.sleep().await;
1422 continue;
1423 }
1424 Err(ExternalError::Indeterminate(x)) => return Err(x),
1425 }
1426 }
1427}
1428
1429#[cfg(test)]
1430pub mod datadriven {
1431 use std::collections::{BTreeMap, BTreeSet};
1432 use std::pin::pin;
1433 use std::sync::{Arc, LazyLock};
1434
1435 use anyhow::anyhow;
1436 use differential_dataflow::consolidation::consolidate_updates;
1437 use differential_dataflow::trace::Description;
1438 use futures::StreamExt;
1439 use mz_dyncfg::{ConfigUpdates, ConfigVal};
1440 use mz_persist::indexed::encoding::BlobTraceBatchPart;
1441 use mz_persist_types::codec_impls::{StringSchema, UnitSchema};
1442
1443 use crate::batch::{
1444 BLOB_TARGET_SIZE, Batch, BatchBuilder, BatchBuilderConfig, BatchBuilderInternal,
1445 BatchParts, validate_truncate_batch,
1446 };
1447 use crate::cfg::COMPACTION_MEMORY_BOUND_BYTES;
1448 use crate::fetch::{EncodedPart, FetchConfig};
1449 use crate::internal::compact::{CompactConfig, CompactReq, Compactor};
1450 use crate::internal::datadriven::DirectiveArgs;
1451 use crate::internal::encoding::Schemas;
1452 use crate::internal::gc::GcReq;
1453 use crate::internal::paths::{BlobKey, BlobKeyPrefix, PartialBlobKey};
1454 use crate::internal::state::{BatchPart, RunOrder, RunPart};
1455 use crate::internal::state_versions::EncodedRollup;
1456 use crate::internal::trace::{CompactionInput, IdHollowBatch, SpineId};
1457 use crate::read::{Listen, ListenEvent, READER_LEASE_DURATION};
1458 use crate::rpc::NoopPubSubSender;
1459 use crate::tests::new_test_client;
1460 use crate::write::COMBINE_INLINE_WRITES;
1461 use crate::{GarbageCollector, PersistClient};
1462
1463 use super::*;
1464
1465 static SCHEMAS: LazyLock<Schemas<String, ()>> = LazyLock::new(|| Schemas {
1466 id: Some(SchemaId(0)),
1467 key: Arc::new(StringSchema),
1468 val: Arc::new(UnitSchema),
1469 });
1470
1471 #[derive(Debug)]
1473 pub struct MachineState {
1474 pub client: PersistClient,
1475 pub shard_id: ShardId,
1476 pub state_versions: Arc<StateVersions>,
1477 pub machine: Machine<String, (), u64, i64>,
1478 pub gc: GarbageCollector<String, (), u64, i64>,
1479 pub batches: BTreeMap<String, IdHollowBatch<u64>>,
1480 pub next_id: usize,
1481 pub rollups: BTreeMap<String, EncodedRollup>,
1482 pub listens: BTreeMap<String, Listen<String, (), u64, i64>>,
1483 pub routine: Vec<RoutineMaintenance>,
1484 pub compactions: BTreeMap<String, CompactReq<u64>>,
1485 }
1486
1487 impl MachineState {
1488 pub async fn new(dyncfgs: &ConfigUpdates) -> Self {
1489 let shard_id = ShardId::new();
1490 let client = new_test_client(dyncfgs).await;
1491 client
1494 .cfg
1495 .set_config(&BLOB_TARGET_SIZE, *BLOB_TARGET_SIZE.default());
1496 client.cfg.set_config(&COMBINE_INLINE_WRITES, false);
1500 let state_versions = Arc::new(StateVersions::new(
1501 client.cfg.clone(),
1502 Arc::clone(&client.consensus),
1503 Arc::clone(&client.blob),
1504 Arc::clone(&client.metrics),
1505 ));
1506 let machine = Machine::new(
1507 client.cfg.clone(),
1508 shard_id,
1509 Arc::clone(&client.metrics),
1510 Arc::clone(&state_versions),
1511 Arc::clone(&client.shared_states),
1512 Arc::new(NoopPubSubSender),
1513 Arc::clone(&client.isolated_runtime),
1514 Diagnostics::for_tests(),
1515 )
1516 .await
1517 .expect("codecs should match");
1518 let gc = GarbageCollector::new(machine.clone(), Arc::clone(&client.isolated_runtime));
1519 MachineState {
1520 shard_id,
1521 client,
1522 state_versions,
1523 machine,
1524 gc,
1525 batches: BTreeMap::default(),
1526 rollups: BTreeMap::default(),
1527 listens: BTreeMap::default(),
1528 routine: Vec::new(),
1529 compactions: BTreeMap::default(),
1530 next_id: 0,
1531 }
1532 }
1533
1534 fn to_batch(&self, hollow: HollowBatch<u64>) -> Batch<String, (), u64, i64> {
1535 Batch::new(
1536 true,
1537 Arc::clone(&self.client.metrics),
1538 Arc::clone(&self.client.blob),
1539 self.client.metrics.shards.shard(&self.shard_id, "test"),
1540 self.client.cfg.build_version.clone(),
1541 hollow,
1542 )
1543 }
1544 }
1545
1546 pub async fn consensus_scan(
1549 datadriven: &MachineState,
1550 args: DirectiveArgs<'_>,
1551 ) -> Result<String, anyhow::Error> {
1552 let from = args.expect("from_seqno");
1553
1554 let mut states = datadriven
1555 .state_versions
1556 .fetch_all_live_states::<u64>(datadriven.shard_id)
1557 .await
1558 .expect("should only be called on an initialized shard")
1559 .check_ts_codec()
1560 .expect("shard codecs should not change");
1561 let mut s = String::new();
1562 while let Some(x) = states.next(|_| {}) {
1563 if x.seqno < from {
1564 continue;
1565 }
1566 let rollups: Vec<_> = x
1567 .collections
1568 .rollups
1569 .keys()
1570 .map(|seqno| seqno.to_string())
1571 .collect();
1572 let batches: Vec<_> = x
1573 .collections
1574 .trace
1575 .batches()
1576 .filter(|b| !b.is_empty())
1577 .filter_map(|b| {
1578 datadriven
1579 .batches
1580 .iter()
1581 .find(|(_, original_batch)| original_batch.batch.parts == b.parts)
1582 .map(|(batch_name, _)| batch_name.to_owned())
1583 })
1584 .collect();
1585 write!(
1586 s,
1587 "seqno={} batches={} rollups={}\n",
1588 x.seqno,
1589 batches.join(","),
1590 rollups.join(","),
1591 );
1592 }
1593 Ok(s)
1594 }
1595
1596 pub async fn consensus_truncate(
1597 datadriven: &MachineState,
1598 args: DirectiveArgs<'_>,
1599 ) -> Result<String, anyhow::Error> {
1600 let to = args.expect("to_seqno");
1601 let removed = datadriven
1602 .client
1603 .consensus
1604 .truncate(&datadriven.shard_id.to_string(), to)
1605 .await
1606 .expect("valid truncation");
1607 Ok(format!("{}\n", removed))
1608 }
1609
1610 pub async fn blob_scan_batches(
1611 datadriven: &MachineState,
1612 _args: DirectiveArgs<'_>,
1613 ) -> Result<String, anyhow::Error> {
1614 let key_prefix = BlobKeyPrefix::Shard(&datadriven.shard_id).to_string();
1615
1616 let mut s = String::new();
1617 let () = datadriven
1618 .state_versions
1619 .blob
1620 .list_keys_and_metadata(&key_prefix, &mut |x| {
1621 let (_, key) = BlobKey::parse_ids(x.key).expect("key should be valid");
1622 if let PartialBlobKey::Batch(_, _) = key {
1623 write!(s, "{}: {}b\n", x.key, x.size_in_bytes);
1624 }
1625 })
1626 .await?;
1627 Ok(s)
1628 }
1629
1630 #[allow(clippy::unused_async)]
1631 pub async fn shard_desc(
1632 datadriven: &MachineState,
1633 _args: DirectiveArgs<'_>,
1634 ) -> Result<String, anyhow::Error> {
1635 Ok(format!(
1636 "since={:?} upper={:?}\n",
1637 datadriven.machine.applier.since().elements(),
1638 datadriven.machine.applier.clone_upper().elements()
1639 ))
1640 }
1641
1642 pub async fn downgrade_since(
1643 datadriven: &mut MachineState,
1644 args: DirectiveArgs<'_>,
1645 ) -> Result<String, anyhow::Error> {
1646 let since = args.expect_antichain("since");
1647 let seqno = args.optional("seqno");
1648 let reader_id = args.expect("reader_id");
1649 let (_, since, routine) = datadriven
1650 .machine
1651 .downgrade_since(
1652 &reader_id,
1653 seqno,
1654 &since,
1655 (datadriven.machine.applier.cfg.now)(),
1656 )
1657 .await;
1658 datadriven.routine.push(routine);
1659 Ok(format!(
1660 "{} {:?}\n",
1661 datadriven.machine.seqno(),
1662 since.0.elements()
1663 ))
1664 }
1665
1666 #[allow(clippy::unused_async)]
1667 pub async fn dyncfg(
1668 datadriven: &MachineState,
1669 args: DirectiveArgs<'_>,
1670 ) -> Result<String, anyhow::Error> {
1671 let mut updates = ConfigUpdates::default();
1672 for x in args.input.trim().split('\n') {
1673 match x.split(' ').collect::<Vec<_>>().as_slice() {
1674 &[name, val] => {
1675 let config = datadriven
1676 .client
1677 .cfg
1678 .entries()
1679 .find(|x| x.name() == name)
1680 .ok_or_else(|| anyhow!("unknown dyncfg: {}", name))?;
1681 match config.val() {
1682 ConfigVal::Usize(_) => {
1683 let val = val.parse().map_err(anyhow::Error::new)?;
1684 updates.add_dynamic(name, ConfigVal::Usize(val));
1685 }
1686 ConfigVal::Bool(_) => {
1687 let val = val.parse().map_err(anyhow::Error::new)?;
1688 updates.add_dynamic(name, ConfigVal::Bool(val));
1689 }
1690 x => unimplemented!("dyncfg type: {:?}", x),
1691 }
1692 }
1693 x => return Err(anyhow!("expected `name val` got: {:?}", x)),
1694 }
1695 }
1696 updates.apply(&datadriven.client.cfg);
1697
1698 Ok("ok\n".to_string())
1699 }
1700
1701 pub async fn compare_and_downgrade_since(
1702 datadriven: &mut MachineState,
1703 args: DirectiveArgs<'_>,
1704 ) -> Result<String, anyhow::Error> {
1705 let expected_opaque: u64 = args.expect("expect_opaque");
1706 let new_opaque: u64 = args.expect("opaque");
1707 let new_since = args.expect_antichain("since");
1708 let reader_id = args.expect("reader_id");
1709 let (res, routine) = datadriven
1710 .machine
1711 .compare_and_downgrade_since(&reader_id, &expected_opaque, (&new_opaque, &new_since))
1712 .await;
1713 datadriven.routine.push(routine);
1714 let since = res.map_err(|(opaque, since)| {
1715 anyhow!("mismatch: opaque={} since={:?}", opaque, since.0.elements())
1716 })?;
1717 Ok(format!(
1718 "{} {} {:?}\n",
1719 datadriven.machine.seqno(),
1720 new_opaque,
1721 since.0.elements()
1722 ))
1723 }
1724
1725 pub async fn write_rollup(
1726 datadriven: &mut MachineState,
1727 args: DirectiveArgs<'_>,
1728 ) -> Result<String, anyhow::Error> {
1729 let output = args.expect_str("output");
1730
1731 let rollup = datadriven
1732 .machine
1733 .applier
1734 .write_rollup_for_state()
1735 .await
1736 .expect("rollup");
1737
1738 datadriven
1739 .rollups
1740 .insert(output.to_string(), rollup.clone());
1741
1742 Ok(format!(
1743 "state={} diffs=[{}, {})\n",
1744 rollup.seqno,
1745 rollup._desc.lower().first().expect("seqno"),
1746 rollup._desc.upper().first().expect("seqno"),
1747 ))
1748 }
1749
1750 pub async fn add_rollup(
1751 datadriven: &mut MachineState,
1752 args: DirectiveArgs<'_>,
1753 ) -> Result<String, anyhow::Error> {
1754 let input = args.expect_str("input");
1755 let rollup = datadriven
1756 .rollups
1757 .get(input)
1758 .expect("unknown batch")
1759 .clone();
1760
1761 let (applied, maintenance) = datadriven
1762 .machine
1763 .add_rollup((rollup.seqno, &rollup.to_hollow()))
1764 .await;
1765
1766 if !applied {
1767 return Err(anyhow!("failed to apply rollup for: {}", rollup.seqno));
1768 }
1769
1770 datadriven.routine.push(maintenance);
1771 Ok(format!("{}\n", datadriven.machine.seqno()))
1772 }
1773
1774 pub async fn write_batch(
1775 datadriven: &mut MachineState,
1776 args: DirectiveArgs<'_>,
1777 ) -> Result<String, anyhow::Error> {
1778 let output = args.expect_str("output");
1779 let lower = args.expect_antichain("lower");
1780 let upper = args.expect_antichain("upper");
1781 assert!(PartialOrder::less_than(&lower, &upper));
1782 let since = args
1783 .optional_antichain("since")
1784 .unwrap_or_else(|| Antichain::from_elem(0));
1785 let target_size = args.optional("target_size");
1786 let parts_size_override = args.optional("parts_size_override");
1787 let consolidate = args.optional("consolidate").unwrap_or(true);
1788 let mut updates: Vec<_> = args
1789 .input
1790 .split('\n')
1791 .flat_map(DirectiveArgs::parse_update)
1792 .collect();
1793
1794 let mut cfg = BatchBuilderConfig::new(&datadriven.client.cfg, datadriven.shard_id);
1795 if let Some(target_size) = target_size {
1796 cfg.blob_target_size = target_size;
1797 };
1798 if consolidate {
1799 consolidate_updates(&mut updates);
1800 }
1801 let run_order = if consolidate {
1802 cfg.preferred_order
1803 } else {
1804 RunOrder::Unordered
1805 };
1806 let parts = BatchParts::new_ordered::<i64>(
1807 cfg.clone(),
1808 run_order,
1809 Arc::clone(&datadriven.client.metrics),
1810 Arc::clone(&datadriven.machine.applier.shard_metrics),
1811 datadriven.shard_id,
1812 Arc::clone(&datadriven.client.blob),
1813 Arc::clone(&datadriven.client.isolated_runtime),
1814 &datadriven.client.metrics.user,
1815 );
1816 let builder = BatchBuilderInternal::new(
1817 cfg.clone(),
1818 parts,
1819 Arc::clone(&datadriven.client.metrics),
1820 SCHEMAS.clone(),
1821 Arc::clone(&datadriven.client.blob),
1822 datadriven.shard_id.clone(),
1823 datadriven.client.cfg.build_version.clone(),
1824 );
1825 let mut builder = BatchBuilder::new(builder, Description::new(lower, upper.clone(), since));
1826 for ((k, ()), t, d) in updates {
1827 builder.add(&k, &(), &t, &d).await.expect("invalid batch");
1828 }
1829 let mut batch = builder.finish(upper).await?;
1830 if parts_size_override.is_some() {
1833 batch
1834 .flush_to_blob(
1835 &cfg,
1836 &datadriven.client.metrics.user,
1837 &datadriven.client.isolated_runtime,
1838 &SCHEMAS,
1839 )
1840 .await;
1841 }
1842 let batch = batch.into_hollow_batch();
1843 let batch = IdHollowBatch {
1844 batch: Arc::new(batch),
1845 id: SpineId(datadriven.next_id, datadriven.next_id + 1),
1846 };
1847 datadriven.next_id += 1;
1848
1849 if let Some(size) = parts_size_override {
1850 let mut batch = batch.clone();
1851 let mut hollow_batch = (*batch.batch).clone();
1852 for part in hollow_batch.parts.iter_mut() {
1853 match part {
1854 RunPart::Many(run) => run.max_part_bytes = size,
1855 RunPart::Single(BatchPart::Hollow(part)) => part.encoded_size_bytes = size,
1856 RunPart::Single(BatchPart::Inline { .. }) => unreachable!("flushed out above"),
1857 }
1858 }
1859 batch.batch = Arc::new(hollow_batch);
1860 datadriven.batches.insert(output.to_owned(), batch);
1861 } else {
1862 datadriven.batches.insert(output.to_owned(), batch.clone());
1863 }
1864 Ok(format!(
1865 "parts={} len={}\n",
1866 batch.batch.part_count(),
1867 batch.batch.len
1868 ))
1869 }
1870
1871 pub async fn fetch_batch(
1872 datadriven: &MachineState,
1873 args: DirectiveArgs<'_>,
1874 ) -> Result<String, anyhow::Error> {
1875 let input = args.expect_str("input");
1876 let stats = args.optional_str("stats");
1877 let batch = datadriven.batches.get(input).expect("unknown batch");
1878
1879 let mut s = String::new();
1880 let mut stream = pin!(
1881 batch
1882 .batch
1883 .part_stream(
1884 datadriven.shard_id,
1885 &*datadriven.state_versions.blob,
1886 &*datadriven.state_versions.metrics
1887 )
1888 .enumerate()
1889 );
1890 while let Some((idx, part)) = stream.next().await {
1891 let part = &*part?;
1892 write!(s, "<part {idx}>\n");
1893
1894 let lower = match part {
1895 BatchPart::Inline { updates, .. } => {
1896 let updates: BlobTraceBatchPart<u64> =
1897 updates.decode(&datadriven.client.metrics.columnar)?;
1898 updates.structured_key_lower()
1899 }
1900 other => other.structured_key_lower(),
1901 };
1902
1903 if let Some(lower) = lower {
1904 if stats == Some("lower") {
1905 writeln!(s, "<key lower={}>", lower.get())
1906 }
1907 }
1908
1909 match part {
1910 BatchPart::Hollow(part) => {
1911 let blob_batch = datadriven
1912 .client
1913 .blob
1914 .get(&part.key.complete(&datadriven.shard_id))
1915 .await;
1916 match blob_batch {
1917 Ok(Some(_)) | Err(_) => {}
1918 Ok(None) => {
1921 s.push_str("<empty>\n");
1922 continue;
1923 }
1924 };
1925 }
1926 BatchPart::Inline { .. } => {}
1927 };
1928 let part = EncodedPart::fetch(
1929 &FetchConfig::from_persist_config(&datadriven.client.cfg),
1930 &datadriven.shard_id,
1931 datadriven.client.blob.as_ref(),
1932 datadriven.client.metrics.as_ref(),
1933 datadriven.machine.applier.shard_metrics.as_ref(),
1934 &datadriven.client.metrics.read.batch_fetcher,
1935 &batch.batch.desc,
1936 part,
1937 )
1938 .await
1939 .expect("invalid batch part");
1940 let part = part
1941 .normalize(&datadriven.client.metrics.columnar)
1942 .into_part::<String, ()>(&*SCHEMAS.key, &*SCHEMAS.val);
1943
1944 for ((k, _v), t, d) in part
1945 .decode_iter::<_, _, u64, i64>(&*SCHEMAS.key, &*SCHEMAS.val)
1946 .expect("valid schemas")
1947 {
1948 writeln!(s, "{k} {t} {d}");
1949 }
1950 }
1951 if !s.is_empty() {
1952 for (idx, (_meta, run)) in batch.batch.runs().enumerate() {
1953 write!(s, "<run {idx}>\n");
1954 for part in run {
1955 let part_idx = batch
1956 .batch
1957 .parts
1958 .iter()
1959 .position(|p| p == part)
1960 .expect("part should exist");
1961 write!(s, "part {part_idx}\n");
1962 }
1963 }
1964 }
1965 Ok(s)
1966 }
1967
1968 #[allow(clippy::unused_async)]
1969 pub async fn truncate_batch_desc(
1970 datadriven: &mut MachineState,
1971 args: DirectiveArgs<'_>,
1972 ) -> Result<String, anyhow::Error> {
1973 let input = args.expect_str("input");
1974 let output = args.expect_str("output");
1975 let lower = args.expect_antichain("lower");
1976 let upper = args.expect_antichain("upper");
1977
1978 let batch = datadriven
1979 .batches
1980 .get(input)
1981 .expect("unknown batch")
1982 .clone();
1983 let truncated_desc = Description::new(lower, upper, batch.batch.desc.since().clone());
1984 let () = validate_truncate_batch(&batch.batch, &truncated_desc, false, true)?;
1985 let mut new_hollow_batch = (*batch.batch).clone();
1986 new_hollow_batch.desc = truncated_desc;
1987 let new_batch = IdHollowBatch {
1988 batch: Arc::new(new_hollow_batch),
1989 id: batch.id,
1990 };
1991 datadriven
1992 .batches
1993 .insert(output.to_owned(), new_batch.clone());
1994 Ok(format!(
1995 "parts={} len={}\n",
1996 batch.batch.part_count(),
1997 batch.batch.len
1998 ))
1999 }
2000
2001 #[allow(clippy::unused_async)]
2002 pub async fn set_batch_parts_size(
2003 datadriven: &mut MachineState,
2004 args: DirectiveArgs<'_>,
2005 ) -> Result<String, anyhow::Error> {
2006 let input = args.expect_str("input");
2007 let size = args.expect("size");
2008 let batch = datadriven.batches.get_mut(input).expect("unknown batch");
2009 let mut hollow_batch = (*batch.batch).clone();
2010 for part in hollow_batch.parts.iter_mut() {
2011 match part {
2012 RunPart::Single(BatchPart::Hollow(x)) => x.encoded_size_bytes = size,
2013 _ => {
2014 panic!("set_batch_parts_size only supports hollow parts")
2015 }
2016 }
2017 }
2018 batch.batch = Arc::new(hollow_batch);
2019 Ok("ok\n".to_string())
2020 }
2021
2022 pub async fn compact(
2023 datadriven: &mut MachineState,
2024 args: DirectiveArgs<'_>,
2025 ) -> Result<String, anyhow::Error> {
2026 let output = args.expect_str("output");
2027 let lower = args.expect_antichain("lower");
2028 let upper = args.expect_antichain("upper");
2029 let since = args.expect_antichain("since");
2030 let target_size = args.optional("target_size");
2031 let memory_bound = args.optional("memory_bound");
2032
2033 let mut inputs = Vec::new();
2034 for input in args.args.get("inputs").expect("missing inputs") {
2035 inputs.push(
2036 datadriven
2037 .batches
2038 .get(input)
2039 .expect("unknown batch")
2040 .clone(),
2041 );
2042 }
2043
2044 let cfg = datadriven.client.cfg.clone();
2045 if let Some(target_size) = target_size {
2046 cfg.set_config(&BLOB_TARGET_SIZE, target_size);
2047 };
2048 if let Some(memory_bound) = memory_bound {
2049 cfg.set_config(&COMPACTION_MEMORY_BOUND_BYTES, memory_bound);
2050 }
2051 let req = CompactReq {
2052 shard_id: datadriven.shard_id,
2053 desc: Description::new(lower, upper, since),
2054 inputs: inputs.clone(),
2055 };
2056 datadriven
2057 .compactions
2058 .insert(output.to_owned(), req.clone());
2059 let spine_lower = inputs
2060 .first()
2061 .map_or_else(|| datadriven.next_id, |x| x.id.0);
2062 let spine_upper = inputs.last().map_or_else(
2063 || {
2064 datadriven.next_id += 1;
2065 datadriven.next_id
2066 },
2067 |x| x.id.1,
2068 );
2069 let new_spine_id = SpineId(spine_lower, spine_upper);
2070 let res = Compactor::<String, (), u64, i64>::compact(
2071 CompactConfig::new(&cfg, datadriven.shard_id),
2072 Arc::clone(&datadriven.client.blob),
2073 Arc::clone(&datadriven.client.metrics),
2074 Arc::clone(&datadriven.machine.applier.shard_metrics),
2075 Arc::clone(&datadriven.client.isolated_runtime),
2076 req,
2077 SCHEMAS.clone(),
2078 )
2079 .await?;
2080
2081 let batch = IdHollowBatch {
2082 batch: Arc::new(res.output.clone()),
2083 id: new_spine_id,
2084 };
2085
2086 datadriven.batches.insert(output.to_owned(), batch.clone());
2087 Ok(format!(
2088 "parts={} len={}\n",
2089 res.output.part_count(),
2090 res.output.len
2091 ))
2092 }
2093
2094 pub async fn clear_blob(
2095 datadriven: &MachineState,
2096 _args: DirectiveArgs<'_>,
2097 ) -> Result<String, anyhow::Error> {
2098 let mut to_delete = vec![];
2099 datadriven
2100 .client
2101 .blob
2102 .list_keys_and_metadata("", &mut |meta| {
2103 to_delete.push(meta.key.to_owned());
2104 })
2105 .await?;
2106 for blob in &to_delete {
2107 datadriven.client.blob.delete(blob).await?;
2108 }
2109 Ok(format!("deleted={}\n", to_delete.len()))
2110 }
2111
2112 pub async fn restore_blob(
2113 datadriven: &MachineState,
2114 _args: DirectiveArgs<'_>,
2115 ) -> Result<String, anyhow::Error> {
2116 let not_restored = crate::internal::restore::restore_blob(
2117 &datadriven.state_versions,
2118 datadriven.client.blob.as_ref(),
2119 &datadriven.client.cfg.build_version,
2120 datadriven.shard_id,
2121 &*datadriven.state_versions.metrics,
2122 )
2123 .await?;
2124 let mut out = String::new();
2125 for key in not_restored {
2126 writeln!(&mut out, "{key}");
2127 }
2128 Ok(out)
2129 }
2130
2131 #[allow(clippy::unused_async)]
2132 pub async fn rewrite_ts(
2133 datadriven: &mut MachineState,
2134 args: DirectiveArgs<'_>,
2135 ) -> Result<String, anyhow::Error> {
2136 let input = args.expect_str("input");
2137 let ts_rewrite = args.expect_antichain("frontier");
2138 let upper = args.expect_antichain("upper");
2139
2140 let batch = datadriven.batches.get_mut(input).expect("unknown batch");
2141 let mut hollow_batch = (*batch.batch).clone();
2142 let () = hollow_batch
2143 .rewrite_ts(&ts_rewrite, upper)
2144 .map_err(|err| anyhow!("invalid rewrite: {}", err))?;
2145 batch.batch = Arc::new(hollow_batch);
2146 Ok("ok\n".into())
2147 }
2148
2149 pub async fn gc(
2150 datadriven: &mut MachineState,
2151 args: DirectiveArgs<'_>,
2152 ) -> Result<String, anyhow::Error> {
2153 let new_seqno_since = args.expect("to_seqno");
2154
2155 let req = GcReq {
2156 shard_id: datadriven.shard_id,
2157 new_seqno_since,
2158 };
2159 let (maintenance, stats) =
2160 GarbageCollector::gc_and_truncate(&datadriven.machine, req).await;
2161 datadriven.routine.push(maintenance);
2162
2163 Ok(format!(
2164 "{} batch_parts={} rollups={} truncated={} state_rollups={}\n",
2165 datadriven.machine.seqno(),
2166 stats.batch_parts_deleted_from_blob,
2167 stats.rollups_deleted_from_blob,
2168 stats
2169 .truncated_consensus_to
2170 .iter()
2171 .map(|x| x.to_string())
2172 .collect::<Vec<_>>()
2173 .join(","),
2174 stats
2175 .rollups_removed_from_state
2176 .iter()
2177 .map(|x| x.to_string())
2178 .collect::<Vec<_>>()
2179 .join(","),
2180 ))
2181 }
2182
2183 pub async fn snapshot(
2184 datadriven: &MachineState,
2185 args: DirectiveArgs<'_>,
2186 ) -> Result<String, anyhow::Error> {
2187 let as_of = args.expect_antichain("as_of");
2188 let snapshot = datadriven
2189 .machine
2190 .snapshot(&as_of)
2191 .await
2192 .map_err(|err| anyhow!("{:?}", err))?;
2193
2194 let mut result = String::new();
2195
2196 for batch in snapshot {
2197 writeln!(
2198 result,
2199 "<batch {:?}-{:?}>",
2200 batch.desc.lower().elements(),
2201 batch.desc.upper().elements()
2202 );
2203 for (run, (_meta, parts)) in batch.runs().enumerate() {
2204 writeln!(result, "<run {run}>");
2205 let mut stream = pin!(
2206 futures::stream::iter(parts)
2207 .flat_map(|part| part.part_stream(
2208 datadriven.shard_id,
2209 &*datadriven.state_versions.blob,
2210 &*datadriven.state_versions.metrics
2211 ))
2212 .enumerate()
2213 );
2214
2215 while let Some((idx, part)) = stream.next().await {
2216 let part = &*part?;
2217 writeln!(result, "<part {idx}>");
2218
2219 let part = EncodedPart::fetch(
2220 &FetchConfig::from_persist_config(&datadriven.client.cfg),
2221 &datadriven.shard_id,
2222 datadriven.client.blob.as_ref(),
2223 datadriven.client.metrics.as_ref(),
2224 datadriven.machine.applier.shard_metrics.as_ref(),
2225 &datadriven.client.metrics.read.batch_fetcher,
2226 &batch.desc,
2227 part,
2228 )
2229 .await
2230 .expect("invalid batch part");
2231 let part = part
2232 .normalize(&datadriven.client.metrics.columnar)
2233 .into_part::<String, ()>(&*SCHEMAS.key, &*SCHEMAS.val);
2234
2235 let mut updates = Vec::new();
2236
2237 for ((k, _v), mut t, d) in part
2238 .decode_iter::<_, _, u64, i64>(&*SCHEMAS.key, &*SCHEMAS.val)
2239 .expect("valid schemas")
2240 {
2241 t.advance_by(as_of.borrow());
2242 updates.push((k, t, d));
2243 }
2244
2245 consolidate_updates(&mut updates);
2246
2247 for (k, t, d) in updates {
2248 writeln!(result, "{k} {t} {d}");
2249 }
2250 }
2251 }
2252 }
2253
2254 Ok(result)
2255 }
2256
2257 pub async fn register_listen(
2258 datadriven: &mut MachineState,
2259 args: DirectiveArgs<'_>,
2260 ) -> Result<String, anyhow::Error> {
2261 let output = args.expect_str("output");
2262 let as_of = args.expect_antichain("as_of");
2263 let read = datadriven
2264 .client
2265 .open_leased_reader::<String, (), u64, i64>(
2266 datadriven.shard_id,
2267 Arc::new(StringSchema),
2268 Arc::new(UnitSchema),
2269 Diagnostics::for_tests(),
2270 true,
2271 )
2272 .await
2273 .expect("invalid shard types");
2274 let listen = read
2275 .listen(as_of)
2276 .await
2277 .map_err(|err| anyhow!("{:?}", err))?;
2278 datadriven.listens.insert(output.to_owned(), listen);
2279 Ok("ok\n".into())
2280 }
2281
2282 pub async fn listen_through(
2283 datadriven: &mut MachineState,
2284 args: DirectiveArgs<'_>,
2285 ) -> Result<String, anyhow::Error> {
2286 let input = args.expect_str("input");
2287 let frontier = args.expect("frontier");
2290 let listen = datadriven.listens.get_mut(input).expect("unknown listener");
2291 let mut s = String::new();
2292 loop {
2293 for event in listen.fetch_next().await {
2294 match event {
2295 ListenEvent::Updates(x) => {
2296 for ((k, _v), t, d) in x.iter() {
2297 write!(s, "{} {} {}\n", k.as_ref().unwrap(), t, d);
2298 }
2299 }
2300 ListenEvent::Progress(x) => {
2301 if !x.less_than(&frontier) {
2302 return Ok(s);
2303 }
2304 }
2305 }
2306 }
2307 }
2308 }
2309
2310 pub async fn register_critical_reader(
2311 datadriven: &mut MachineState,
2312 args: DirectiveArgs<'_>,
2313 ) -> Result<String, anyhow::Error> {
2314 let reader_id = args.expect("reader_id");
2315 let (state, maintenance) = datadriven
2316 .machine
2317 .register_critical_reader::<u64>(&reader_id, "tests")
2318 .await;
2319 datadriven.routine.push(maintenance);
2320 Ok(format!(
2321 "{} {:?}\n",
2322 datadriven.machine.seqno(),
2323 state.since.elements(),
2324 ))
2325 }
2326
2327 pub async fn register_leased_reader(
2328 datadriven: &mut MachineState,
2329 args: DirectiveArgs<'_>,
2330 ) -> Result<String, anyhow::Error> {
2331 let reader_id = args.expect("reader_id");
2332 let (reader_state, maintenance) = datadriven
2333 .machine
2334 .register_leased_reader(
2335 &reader_id,
2336 "tests",
2337 READER_LEASE_DURATION.get(&datadriven.client.cfg),
2338 (datadriven.client.cfg.now)(),
2339 false,
2340 )
2341 .await;
2342 datadriven.routine.push(maintenance);
2343 Ok(format!(
2344 "{} {:?}\n",
2345 datadriven.machine.seqno(),
2346 reader_state.since.elements(),
2347 ))
2348 }
2349
2350 pub async fn heartbeat_leased_reader(
2351 datadriven: &MachineState,
2352 args: DirectiveArgs<'_>,
2353 ) -> Result<String, anyhow::Error> {
2354 let reader_id = args.expect("reader_id");
2355 let _ = datadriven
2356 .machine
2357 .heartbeat_leased_reader(&reader_id, (datadriven.client.cfg.now)())
2358 .await;
2359 Ok(format!("{} ok\n", datadriven.machine.seqno()))
2360 }
2361
2362 pub async fn expire_critical_reader(
2363 datadriven: &mut MachineState,
2364 args: DirectiveArgs<'_>,
2365 ) -> Result<String, anyhow::Error> {
2366 let reader_id = args.expect("reader_id");
2367 let (_, maintenance) = datadriven.machine.expire_critical_reader(&reader_id).await;
2368 datadriven.routine.push(maintenance);
2369 Ok(format!("{} ok\n", datadriven.machine.seqno()))
2370 }
2371
2372 pub async fn expire_leased_reader(
2373 datadriven: &mut MachineState,
2374 args: DirectiveArgs<'_>,
2375 ) -> Result<String, anyhow::Error> {
2376 let reader_id = args.expect("reader_id");
2377 let (_, maintenance) = datadriven.machine.expire_leased_reader(&reader_id).await;
2378 datadriven.routine.push(maintenance);
2379 Ok(format!("{} ok\n", datadriven.machine.seqno()))
2380 }
2381
2382 pub async fn compare_and_append_batches(
2383 datadriven: &MachineState,
2384 args: DirectiveArgs<'_>,
2385 ) -> Result<String, anyhow::Error> {
2386 let expected_upper = args.expect_antichain("expected_upper");
2387 let new_upper = args.expect_antichain("new_upper");
2388
2389 let mut batches: Vec<Batch<String, (), u64, i64>> = args
2390 .args
2391 .get("batches")
2392 .expect("missing batches")
2393 .into_iter()
2394 .map(|batch| {
2395 let hollow = (*datadriven
2396 .batches
2397 .get(batch)
2398 .expect("unknown batch")
2399 .clone()
2400 .batch)
2401 .clone();
2402 datadriven.to_batch(hollow)
2403 })
2404 .collect();
2405
2406 let mut writer = datadriven
2407 .client
2408 .open_writer(
2409 datadriven.shard_id,
2410 Arc::new(StringSchema),
2411 Arc::new(UnitSchema),
2412 Diagnostics::for_tests(),
2413 )
2414 .await?;
2415
2416 let mut batch_refs: Vec<_> = batches.iter_mut().collect();
2417
2418 let () = writer
2419 .compare_and_append_batch(batch_refs.as_mut_slice(), expected_upper, new_upper, true)
2420 .await?
2421 .map_err(|err| anyhow!("upper mismatch: {:?}", err))?;
2422
2423 writer.expire().await;
2424
2425 Ok("ok\n".into())
2426 }
2427
2428 pub async fn expire_writer(
2429 datadriven: &mut MachineState,
2430 args: DirectiveArgs<'_>,
2431 ) -> Result<String, anyhow::Error> {
2432 let writer_id = args.expect("writer_id");
2433 let (_, maintenance) = datadriven.machine.expire_writer(&writer_id).await;
2434 datadriven.routine.push(maintenance);
2435 Ok(format!("{} ok\n", datadriven.machine.seqno()))
2436 }
2437
2438 pub(crate) async fn finalize(
2439 datadriven: &mut MachineState,
2440 _args: DirectiveArgs<'_>,
2441 ) -> anyhow::Result<String> {
2442 let maintenance = datadriven.machine.become_tombstone().await?;
2443 datadriven.routine.push(maintenance);
2444 Ok(format!("{} ok\n", datadriven.machine.seqno()))
2445 }
2446
2447 pub(crate) fn is_finalized(
2448 datadriven: &MachineState,
2449 _args: DirectiveArgs<'_>,
2450 ) -> anyhow::Result<String> {
2451 let seqno = datadriven.machine.seqno();
2452 let tombstone = datadriven.machine.is_finalized();
2453 Ok(format!("{seqno} {tombstone}\n"))
2454 }
2455
2456 pub async fn compare_and_append(
2457 datadriven: &mut MachineState,
2458 args: DirectiveArgs<'_>,
2459 ) -> Result<String, anyhow::Error> {
2460 let input = args.expect_str("input");
2461 let writer_id = args.expect("writer_id");
2462 let mut batch = datadriven
2463 .batches
2464 .get(input)
2465 .expect("unknown batch")
2466 .clone();
2467 let token = args.optional("token").unwrap_or_else(IdempotencyToken::new);
2468 let now = (datadriven.client.cfg.now)();
2469
2470 let (id, maintenance) = datadriven
2471 .machine
2472 .register_schema(&*SCHEMAS.key, &*SCHEMAS.val)
2473 .await;
2474 assert_eq!(id, SCHEMAS.id);
2475 datadriven.routine.push(maintenance);
2476 let maintenance = loop {
2477 let indeterminate = args
2478 .optional::<String>("prev_indeterminate")
2479 .map(|x| Indeterminate::new(anyhow::Error::msg(x)));
2480 let res = datadriven
2481 .machine
2482 .compare_and_append_idempotent(
2483 &batch.batch,
2484 &writer_id,
2485 now,
2486 &token,
2487 &HandleDebugState::default(),
2488 indeterminate,
2489 )
2490 .await;
2491 match res {
2492 CompareAndAppendRes::Success(_, x) => break x,
2493 CompareAndAppendRes::UpperMismatch(_seqno, upper) => {
2494 return Err(anyhow!("{:?}", Upper(upper)));
2495 }
2496 CompareAndAppendRes::InlineBackpressure => {
2497 let hollow_batch = (*batch.batch).clone();
2498 let mut b = datadriven.to_batch(hollow_batch);
2499 let cfg = BatchBuilderConfig::new(&datadriven.client.cfg, datadriven.shard_id);
2500 b.flush_to_blob(
2501 &cfg,
2502 &datadriven.client.metrics.user,
2503 &datadriven.client.isolated_runtime,
2504 &*SCHEMAS,
2505 )
2506 .await;
2507 batch.batch = Arc::new(b.into_hollow_batch());
2508 continue;
2509 }
2510 _ => panic!("{:?}", res),
2511 };
2512 };
2513 datadriven.routine.push(maintenance.routine);
2516 Ok(format!(
2517 "{} {:?}\n",
2518 datadriven.machine.seqno(),
2519 datadriven.machine.applier.clone_upper().elements(),
2520 ))
2521 }
2522
2523 pub async fn apply_merge_res(
2524 datadriven: &mut MachineState,
2525 args: DirectiveArgs<'_>,
2526 ) -> Result<String, anyhow::Error> {
2527 let input = args.expect_str("input");
2528 let batch = datadriven
2529 .batches
2530 .get(input)
2531 .expect("unknown batch")
2532 .clone();
2533 let compact_req = datadriven
2534 .compactions
2535 .get(input)
2536 .expect("unknown compact req")
2537 .clone();
2538 let input_batches = compact_req
2539 .inputs
2540 .iter()
2541 .map(|x| x.id)
2542 .collect::<BTreeSet<_>>();
2543 let lower_spine_bound = input_batches
2544 .first()
2545 .map(|id| id.0)
2546 .expect("at least one batch must be present");
2547 let upper_spine_bound = input_batches
2548 .last()
2549 .map(|id| id.1)
2550 .expect("at least one batch must be present");
2551 let id = SpineId(lower_spine_bound, upper_spine_bound);
2552 let hollow_batch = (*batch.batch).clone();
2553
2554 let (merge_res, maintenance) = datadriven
2555 .machine
2556 .merge_res(&FueledMergeRes {
2557 output: hollow_batch,
2558 input: CompactionInput::IdRange(id),
2559 new_active_compaction: None,
2560 })
2561 .await;
2562 datadriven.routine.push(maintenance);
2563 Ok(format!(
2564 "{} {}\n",
2565 datadriven.machine.seqno(),
2566 merge_res.applied()
2567 ))
2568 }
2569
2570 pub async fn perform_maintenance(
2571 datadriven: &mut MachineState,
2572 _args: DirectiveArgs<'_>,
2573 ) -> Result<String, anyhow::Error> {
2574 let mut s = String::new();
2575 for maintenance in datadriven.routine.drain(..) {
2576 let () = maintenance
2577 .perform(&datadriven.machine, &datadriven.gc)
2578 .await;
2579 let () = datadriven
2580 .machine
2581 .applier
2582 .fetch_and_update_state(None)
2583 .await;
2584 write!(s, "{} ok\n", datadriven.machine.seqno());
2585 }
2586 Ok(s)
2587 }
2588}
2589
2590#[cfg(test)]
2591pub mod tests {
2592 use std::sync::Arc;
2593
2594 use mz_dyncfg::ConfigUpdates;
2595 use mz_ore::cast::CastFrom;
2596 use mz_ore::task::spawn;
2597 use mz_persist::intercept::{InterceptBlob, InterceptHandle};
2598 use mz_persist::location::SeqNo;
2599 use mz_persist_types::PersistLocation;
2600 use semver::Version;
2601 use timely::progress::Antichain;
2602
2603 use crate::batch::BatchBuilderConfig;
2604 use crate::cache::StateCache;
2605 use crate::internal::gc::{GarbageCollector, GcReq};
2606 use crate::internal::state::{HandleDebugState, ROLLUP_THRESHOLD};
2607 use crate::tests::{new_test_client, new_test_client_cache};
2608 use crate::{Diagnostics, PersistClient, ShardId};
2609
2610 #[mz_persist_proc::test(tokio::test(flavor = "multi_thread"))]
2611 #[cfg_attr(miri, ignore)] async fn apply_unbatched_cmd_truncate(dyncfgs: ConfigUpdates) {
2613 mz_ore::test::init_logging();
2614
2615 let client = new_test_client(&dyncfgs).await;
2616 client.cfg.set_config(&ROLLUP_THRESHOLD, 5);
2618 let (mut write, _) = client
2619 .expect_open::<String, (), u64, i64>(ShardId::new())
2620 .await;
2621
2622 const NUM_BATCHES: u64 = 100;
2625 for idx in 0..NUM_BATCHES {
2626 let mut batch = write
2627 .expect_batch(&[((idx.to_string(), ()), idx, 1)], idx, idx + 1)
2628 .await;
2629 let cfg = BatchBuilderConfig::new(&client.cfg, write.shard_id());
2632 batch
2633 .flush_to_blob(
2634 &cfg,
2635 &client.metrics.user,
2636 &client.isolated_runtime,
2637 &write.write_schemas,
2638 )
2639 .await;
2640 let (_, writer_maintenance) = write
2641 .machine
2642 .compare_and_append(
2643 &batch.into_hollow_batch(),
2644 &write.writer_id,
2645 &HandleDebugState::default(),
2646 (write.cfg.now)(),
2647 )
2648 .await
2649 .unwrap();
2650 writer_maintenance
2651 .perform(&write.machine, &write.gc, write.compact.as_ref())
2652 .await;
2653 }
2654 let live_diffs = write
2655 .machine
2656 .applier
2657 .state_versions
2658 .fetch_all_live_diffs(&write.machine.shard_id())
2659 .await;
2660 assert!(live_diffs.0.len() > 0);
2662 let max_live_diffs = 2 * usize::cast_from(NUM_BATCHES.next_power_of_two().trailing_zeros());
2666 assert!(
2667 live_diffs.0.len() <= max_live_diffs,
2668 "{} vs {}",
2669 live_diffs.0.len(),
2670 max_live_diffs
2671 );
2672 }
2673
2674 #[mz_persist_proc::test(tokio::test(flavor = "multi_thread"))]
2678 #[cfg_attr(miri, ignore)] async fn regression_gc_skipped_req_and_interrupted(dyncfgs: ConfigUpdates) {
2680 let mut client = new_test_client(&dyncfgs).await;
2681 let intercept = InterceptHandle::default();
2682 client.blob = Arc::new(InterceptBlob::new(
2683 Arc::clone(&client.blob),
2684 intercept.clone(),
2685 ));
2686 let (_, mut read) = client
2687 .expect_open::<String, String, u64, i64>(ShardId::new())
2688 .await;
2689
2690 read.downgrade_since(&Antichain::from_elem(1)).await;
2692 let new_seqno_since = read.machine.applier.seqno_since();
2693
2694 assert!(new_seqno_since > SeqNo::minimum());
2699 intercept.set_post_delete(Some(Arc::new(|_, _| panic!("boom"))));
2700 let machine = read.machine.clone();
2701 let gc = spawn(|| "", async move {
2703 let req = GcReq {
2704 shard_id: machine.shard_id(),
2705 new_seqno_since,
2706 };
2707 GarbageCollector::gc_and_truncate(&machine, req).await
2708 });
2709 let _ = gc.await;
2712
2713 intercept.set_post_delete(None);
2716 let req = GcReq {
2717 shard_id: read.machine.shard_id(),
2718 new_seqno_since,
2719 };
2720 let _ = GarbageCollector::gc_and_truncate(&read.machine, req.clone()).await;
2721 }
2722
2723 #[mz_persist_proc::test(tokio::test(flavor = "multi_thread"))]
2728 #[cfg_attr(miri, ignore)] async fn regression_update_state_after_upper_mismatch(dyncfgs: ConfigUpdates) {
2730 let client = new_test_client(&dyncfgs).await;
2731 let mut client2 = client.clone();
2732
2733 let new_state_cache = Arc::new(StateCache::new_no_metrics());
2736 client2.shared_states = new_state_cache;
2737
2738 let shard_id = ShardId::new();
2739 let (mut write1, _) = client.expect_open::<String, (), u64, i64>(shard_id).await;
2740 let (mut write2, _) = client2.expect_open::<String, (), u64, i64>(shard_id).await;
2741
2742 let data = [
2743 (("1".to_owned(), ()), 1, 1),
2744 (("2".to_owned(), ()), 2, 1),
2745 (("3".to_owned(), ()), 3, 1),
2746 (("4".to_owned(), ()), 4, 1),
2747 (("5".to_owned(), ()), 5, 1),
2748 ];
2749
2750 write1.expect_compare_and_append(&data[..1], 0, 2).await;
2751
2752 write2.expect_compare_and_append(&data[1..2], 2, 3).await;
2755 }
2756
2757 #[mz_persist_proc::test(tokio::test(flavor = "multi_thread"))]
2758 #[cfg_attr(miri, ignore)]
2759 async fn version_upgrade(dyncfgs: ConfigUpdates) {
2760 let mut cache = new_test_client_cache(&dyncfgs);
2761 cache.cfg.build_version = Version::new(26, 1, 0);
2762 let shard_id = ShardId::new();
2763
2764 async fn fetch_catalog_upgrade_shard_version(
2765 persist_client: &PersistClient,
2766 upgrade_shard_id: ShardId,
2767 ) -> Option<semver::Version> {
2768 let shard_state = persist_client
2769 .inspect_shard::<u64>(&upgrade_shard_id)
2770 .await
2771 .ok()?;
2772 let json_state = serde_json::to_value(shard_state).expect("state serialization error");
2773 let upgrade_version = json_state
2774 .get("applier_version")
2775 .cloned()
2776 .expect("missing applier_version");
2777 let upgrade_version =
2778 serde_json::from_value(upgrade_version).expect("version deserialization error");
2779 Some(upgrade_version)
2780 }
2781
2782 cache.cfg.build_version = Version::new(26, 1, 0);
2783 let client = cache.open(PersistLocation::new_in_mem()).await.unwrap();
2784 let (_, mut reader) = client.expect_open::<String, (), u64, i64>(shard_id).await;
2785 reader.downgrade_since(&Antichain::from_elem(1)).await;
2786 assert_eq!(
2787 fetch_catalog_upgrade_shard_version(&client, shard_id).await,
2788 Some(Version::new(26, 1, 0)),
2789 );
2790
2791 cache.cfg.build_version = Version::new(27, 1, 0);
2793 let client = cache.open(PersistLocation::new_in_mem()).await.unwrap();
2794 let (_, mut reader) = client.expect_open::<String, (), u64, i64>(shard_id).await;
2795 reader.downgrade_since(&Antichain::from_elem(2)).await;
2796 assert_eq!(
2797 fetch_catalog_upgrade_shard_version(&client, shard_id).await,
2798 Some(Version::new(26, 1, 0)),
2799 );
2800
2801 client
2803 .upgrade_version::<String, (), u64, i64>(shard_id, Diagnostics::for_tests())
2804 .await
2805 .unwrap();
2806 assert_eq!(
2807 fetch_catalog_upgrade_shard_version(&client, shard_id).await,
2808 Some(Version::new(27, 1, 0)),
2809 );
2810 }
2811}