1mod spiller;
16mod task_manager;
17pub(crate) mod test_utils;
18
19use std::cmp::Ordering;
20use std::collections::btree_map::Entry;
21use std::collections::{BTreeMap, BTreeSet, HashMap, HashSet, VecDeque};
22use std::fmt::{Debug, Display, Formatter};
23use std::future::{Future, poll_fn};
24use std::mem::{replace, swap, take};
25use std::sync::Arc;
26use std::task::{Context, Poll, ready};
27
28use futures::FutureExt;
29use itertools::Itertools;
30use more_asserts::assert_gt;
31use prometheus::{HistogramTimer, IntGauge};
32use risingwave_common::bitmap::BitmapBuilder;
33use risingwave_common::catalog::TableId;
34use risingwave_common::metrics::UintGauge;
35use risingwave_common::must_match;
36use risingwave_hummock_sdk::table_watermark::{
37 TableWatermarks, VnodeWatermark, WatermarkDirection, WatermarkSerdeType,
38};
39use risingwave_hummock_sdk::{HummockEpoch, HummockSstableObjectId, LocalSstableInfo};
40use task_manager::{TaskManager, UploadingTaskStatus};
41use thiserror_ext::AsReport;
42use tokio::sync::oneshot;
43use tokio::task::JoinHandle;
44use tracing::{debug, error, info, warn};
45
46use crate::hummock::event_handler::LocalInstanceId;
47use crate::hummock::event_handler::hummock_event_handler::{BufferTracker, send_sync_result};
48use crate::hummock::event_handler::uploader::spiller::Spiller;
49use crate::hummock::event_handler::uploader::uploader_imm::UploaderImm;
50use crate::hummock::local_version::pinned_version::PinnedVersion;
51use crate::hummock::shared_buffer::shared_buffer_batch::SharedBufferBatchId;
52use crate::hummock::store::version::StagingSstableInfo;
53use crate::hummock::{HummockError, HummockResult, ImmutableMemtable};
54use crate::mem_table::ImmId;
55use crate::monitor::HummockStateStoreMetrics;
56use crate::store::SealCurrentEpochOptions;
57
58fn take_before_epoch<T>(
60 data: &mut BTreeMap<HummockEpoch, T>,
61 epoch: HummockEpoch,
62) -> BTreeMap<HummockEpoch, T> {
63 let mut before_epoch_data = data.split_off(&(epoch + 1));
64 swap(&mut before_epoch_data, data);
65 before_epoch_data
66}
67
68type UploadTaskInput = HashMap<LocalInstanceId, Vec<UploaderImm>>;
69pub type UploadTaskPayload = HashMap<LocalInstanceId, Vec<ImmutableMemtable>>;
70
71#[derive(Debug)]
72pub struct UploadTaskOutput {
73 pub new_value_ssts: Vec<LocalSstableInfo>,
74 pub old_value_ssts: Vec<LocalSstableInfo>,
75 pub wait_poll_timer: Option<HistogramTimer>,
76}
77pub type SpawnUploadTask = Arc<
78 dyn Fn(UploadTaskPayload, UploadTaskInfo) -> JoinHandle<HummockResult<UploadTaskOutput>>
79 + Send
80 + Sync
81 + 'static,
82>;
83
84#[derive(Clone)]
85pub struct UploadTaskInfo {
86 pub task_size: usize,
87 pub epochs: Vec<HummockEpoch>,
88 pub imm_ids: HashMap<LocalInstanceId, Vec<ImmId>>,
89}
90
91impl Display for UploadTaskInfo {
92 fn fmt(&self, f: &mut Formatter<'_>) -> std::fmt::Result {
93 f.debug_struct("UploadTaskInfo")
94 .field("task_size", &self.task_size)
95 .field("epochs", &self.epochs)
96 .field("len(imm_ids)", &self.imm_ids.len())
97 .finish()
98 }
99}
100
101impl Debug for UploadTaskInfo {
102 fn fmt(&self, f: &mut Formatter<'_>) -> std::fmt::Result {
103 f.debug_struct("UploadTaskInfo")
104 .field("task_size", &self.task_size)
105 .field("epochs", &self.epochs)
106 .field("imm_ids", &self.imm_ids)
107 .finish()
108 }
109}
110
111mod uploader_imm {
112 use std::fmt::Formatter;
113 use std::ops::Deref;
114
115 use risingwave_common::metrics::UintGauge;
116
117 use crate::hummock::event_handler::uploader::UploaderContext;
118 use crate::mem_table::ImmutableMemtable;
119
120 pub(super) struct UploaderImm {
121 inner: ImmutableMemtable,
122 size_guard: UintGauge,
123 }
124
125 impl UploaderImm {
126 pub(super) fn new(imm: ImmutableMemtable, context: &UploaderContext) -> Self {
127 let size = imm.size();
128 let size_guard = context.stats.uploader_imm_size.clone();
129 size_guard.add(size as _);
130 Self {
131 inner: imm,
132 size_guard,
133 }
134 }
135
136 #[cfg(test)]
137 pub(super) fn for_test(imm: ImmutableMemtable) -> Self {
138 Self {
139 inner: imm,
140 size_guard: UintGauge::new("test", "test").unwrap(),
141 }
142 }
143 }
144
145 impl std::fmt::Debug for UploaderImm {
146 fn fmt(&self, f: &mut Formatter<'_>) -> std::fmt::Result {
147 self.inner.fmt(f)
148 }
149 }
150
151 impl Deref for UploaderImm {
152 type Target = ImmutableMemtable;
153
154 fn deref(&self) -> &Self::Target {
155 &self.inner
156 }
157 }
158
159 impl Drop for UploaderImm {
160 fn drop(&mut self) {
161 self.size_guard.sub(self.inner.size() as _);
162 }
163 }
164}
165
166#[derive(PartialEq, Eq, Hash, PartialOrd, Ord, Copy, Clone, Debug)]
167struct UploadingTaskId(usize);
168
169struct UploadingTask {
172 task_id: UploadingTaskId,
173 input: UploadTaskInput,
175 join_handle: JoinHandle<HummockResult<UploadTaskOutput>>,
176 task_info: UploadTaskInfo,
177 spawn_upload_task: SpawnUploadTask,
178 task_size_guard: UintGauge,
179 task_count_guard: IntGauge,
180}
181
182impl Drop for UploadingTask {
183 fn drop(&mut self) {
184 self.task_size_guard.sub(self.task_info.task_size as u64);
185 self.task_count_guard.dec();
186 }
187}
188
189impl Debug for UploadingTask {
190 fn fmt(&self, f: &mut Formatter<'_>) -> std::fmt::Result {
191 f.debug_struct("UploadingTask")
192 .field("input", &self.input)
193 .field("task_info", &self.task_info)
194 .finish()
195 }
196}
197
198fn get_payload_imm_ids(
199 payload: &UploadTaskPayload,
200) -> HashMap<LocalInstanceId, Vec<SharedBufferBatchId>> {
201 payload
202 .iter()
203 .map(|(instance_id, imms)| {
204 (
205 *instance_id,
206 imms.iter().map(|imm| imm.batch_id()).collect_vec(),
207 )
208 })
209 .collect()
210}
211
212impl UploadingTask {
213 const LOG_THRESHOLD_FOR_UPLOAD_TASK_SIZE: usize = 50 * (1 << 20);
215
216 fn input_to_payload(input: &UploadTaskInput) -> UploadTaskPayload {
217 input
218 .iter()
219 .map(|(instance_id, imms)| {
220 (
221 *instance_id,
222 imms.iter().map(|imm| (**imm).clone()).collect(),
223 )
224 })
225 .collect()
226 }
227
228 fn new(task_id: UploadingTaskId, input: UploadTaskInput, context: &UploaderContext) -> Self {
229 assert!(!input.is_empty());
230 let mut epochs = input
231 .iter()
232 .flat_map(|(_, imms)| imms.iter().flat_map(|imm| imm.epochs().iter().cloned()))
233 .sorted()
234 .dedup()
235 .collect_vec();
236
237 epochs.reverse();
239 let payload = Self::input_to_payload(&input);
240 let imm_ids = get_payload_imm_ids(&payload);
241 let task_size = input
242 .values()
243 .map(|imms| imms.iter().map(|imm| imm.size()).sum::<usize>())
244 .sum();
245 let task_info = UploadTaskInfo {
246 task_size,
247 epochs,
248 imm_ids,
249 };
250 context
251 .buffer_tracker
252 .global_upload_task_size()
253 .add(task_size as u64);
254 if task_info.task_size > Self::LOG_THRESHOLD_FOR_UPLOAD_TASK_SIZE {
255 info!("start upload task: {:?}", task_info);
256 } else {
257 debug!("start upload task: {:?}", task_info);
258 }
259 let join_handle = (context.spawn_upload_task)(payload, task_info.clone());
260 context.stats.uploader_uploading_task_count.inc();
261 Self {
262 task_id,
263 input,
264 join_handle,
265 task_info,
266 spawn_upload_task: context.spawn_upload_task.clone(),
267 task_size_guard: context.buffer_tracker.global_upload_task_size().clone(),
268 task_count_guard: context.stats.uploader_uploading_task_count.clone(),
269 }
270 }
271
272 fn poll_result(
274 &mut self,
275 cx: &mut Context<'_>,
276 ) -> Poll<HummockResult<Arc<StagingSstableInfo>>> {
277 Poll::Ready(match ready!(self.join_handle.poll_unpin(cx)) {
278 Ok(task_result) => task_result
279 .inspect(|_| {
280 if self.task_info.task_size > Self::LOG_THRESHOLD_FOR_UPLOAD_TASK_SIZE {
281 info!(task_info = ?self.task_info, "upload task finish");
282 } else {
283 debug!(task_info = ?self.task_info, "upload task finish");
284 }
285 })
286 .inspect_err(|e| error!(task_info = ?self.task_info, err = ?e.as_report(), "upload task failed"))
287 .map(|output| {
288 Arc::new(StagingSstableInfo::new(
289 output.new_value_ssts,
290 output.old_value_ssts,
291 self.task_info.epochs.clone(),
292 self.task_info.imm_ids.clone(),
293 self.task_info.task_size,
294 ))
295 }),
296
297 Err(err) => Err(HummockError::other(format!(
298 "fail to join upload join handle: {}",
299 err.as_report()
300 ))),
301 })
302 }
303
304 fn poll_ok_with_retry(&mut self, cx: &mut Context<'_>) -> Poll<Arc<StagingSstableInfo>> {
306 loop {
307 let result = ready!(self.poll_result(cx));
308 match result {
309 Ok(sstables) => return Poll::Ready(sstables),
310 Err(e) => {
311 error!(
312 error = %e.as_report(),
313 task_info = ?self.task_info,
314 "a flush task failed, start retry",
315 );
316 self.join_handle = (self.spawn_upload_task)(
317 Self::input_to_payload(&self.input),
318 self.task_info.clone(),
319 );
320 }
324 }
325 }
326 }
327
328 pub fn get_task_info(&self) -> &UploadTaskInfo {
329 &self.task_info
330 }
331}
332
333impl TableUnsyncData {
334 fn add_table_watermarks(
335 &mut self,
336 epoch: HummockEpoch,
337 table_watermarks: Vec<VnodeWatermark>,
338 direction: WatermarkDirection,
339 watermark_type: WatermarkSerdeType,
340 ) {
341 if table_watermarks.is_empty() {
342 return;
343 }
344 let vnode_count = table_watermarks[0].vnode_count();
345 for watermark in &table_watermarks {
346 assert_eq!(vnode_count, watermark.vnode_count());
347 }
348
349 fn apply_new_vnodes(
350 vnode_bitmap: &mut BitmapBuilder,
351 vnode_watermarks: &Vec<VnodeWatermark>,
352 ) {
353 for vnode_watermark in vnode_watermarks {
354 for vnode in vnode_watermark.vnode_bitmap().iter_ones() {
355 assert!(
356 !vnode_bitmap.is_set(vnode),
357 "vnode {} write multiple table watermarks",
358 vnode
359 );
360 vnode_bitmap.set(vnode, true);
361 }
362 }
363 }
364 match &mut self.table_watermarks {
365 Some((prev_direction, prev_watermarks, prev_watermark_type)) => {
366 assert_eq!(
367 *prev_direction, direction,
368 "table id {} new watermark direction not match with previous",
369 self.table_id
370 );
371 assert_eq!(
372 *prev_watermark_type, watermark_type,
373 "table id {} new watermark watermark_type not match with previous",
374 self.table_id
375 );
376 match prev_watermarks.entry(epoch) {
377 Entry::Occupied(mut entry) => {
378 let (prev_watermarks, vnode_bitmap) = entry.get_mut();
379 apply_new_vnodes(vnode_bitmap, &table_watermarks);
380 prev_watermarks.extend(table_watermarks);
381 }
382 Entry::Vacant(entry) => {
383 let mut vnode_bitmap = BitmapBuilder::zeroed(vnode_count);
384 apply_new_vnodes(&mut vnode_bitmap, &table_watermarks);
385 entry.insert((table_watermarks, vnode_bitmap));
386 }
387 }
388 }
389 None => {
390 let mut vnode_bitmap = BitmapBuilder::zeroed(vnode_count);
391 apply_new_vnodes(&mut vnode_bitmap, &table_watermarks);
392 self.table_watermarks = Some((
393 direction,
394 BTreeMap::from_iter([(epoch, (table_watermarks, vnode_bitmap))]),
395 watermark_type,
396 ));
397 }
398 }
399 }
400}
401
402impl UploaderData {
403 fn add_table_watermarks(
404 all_table_watermarks: &mut HashMap<TableId, TableWatermarks>,
405 table_id: TableId,
406 direction: WatermarkDirection,
407 watermarks: impl Iterator<Item = (HummockEpoch, Vec<VnodeWatermark>)>,
408 watermark_type: WatermarkSerdeType,
409 ) {
410 let mut table_watermarks: Option<TableWatermarks> = None;
411 for (epoch, watermarks) in watermarks {
412 match &mut table_watermarks {
413 Some(prev_watermarks) => {
414 assert_eq!(prev_watermarks.direction, direction);
415 assert_eq!(prev_watermarks.watermark_type, watermark_type);
416 prev_watermarks.add_new_epoch_watermarks(
417 epoch,
418 Arc::from(watermarks),
419 direction,
420 watermark_type,
421 );
422 }
423 None => {
424 table_watermarks = Some(TableWatermarks::single_epoch(
425 epoch,
426 watermarks,
427 direction,
428 watermark_type,
429 ));
430 }
431 }
432 }
433 if let Some(table_watermarks) = table_watermarks {
434 assert!(
435 all_table_watermarks
436 .insert(table_id, table_watermarks)
437 .is_none()
438 );
439 }
440 }
441}
442
443struct LocalInstanceEpochData {
444 epoch: HummockEpoch,
445 imms: VecDeque<UploaderImm>,
447 has_spilled: bool,
448}
449
450impl LocalInstanceEpochData {
451 fn new(epoch: HummockEpoch) -> Self {
452 Self {
453 epoch,
454 imms: VecDeque::new(),
455 has_spilled: false,
456 }
457 }
458
459 fn epoch(&self) -> HummockEpoch {
460 self.epoch
461 }
462
463 fn add_imm(&mut self, imm: UploaderImm) {
464 assert_eq!(imm.max_epoch(), imm.min_epoch());
465 assert_eq!(self.epoch, imm.min_epoch());
466 if let Some(prev_imm) = self.imms.front() {
467 assert_gt!(imm.batch_id(), prev_imm.batch_id());
468 }
469 self.imms.push_front(imm);
470 }
471
472 fn is_empty(&self) -> bool {
473 self.imms.is_empty()
474 }
475}
476
477struct LocalInstanceUnsyncData {
478 table_id: TableId,
479 instance_id: LocalInstanceId,
480 current_epoch_data: Option<LocalInstanceEpochData>,
482 sealed_data: VecDeque<LocalInstanceEpochData>,
484 flushing_imms: VecDeque<SharedBufferBatchId>,
486 is_destroyed: bool,
487}
488
489impl LocalInstanceUnsyncData {
490 fn new(table_id: TableId, instance_id: LocalInstanceId, init_epoch: HummockEpoch) -> Self {
491 Self {
492 table_id,
493 instance_id,
494 current_epoch_data: Some(LocalInstanceEpochData::new(init_epoch)),
495 sealed_data: VecDeque::new(),
496 flushing_imms: Default::default(),
497 is_destroyed: false,
498 }
499 }
500
501 fn add_imm(&mut self, imm: UploaderImm) {
502 assert!(!self.is_destroyed);
503 assert_eq!(self.table_id, imm.table_id);
504 self.current_epoch_data
505 .as_mut()
506 .expect("should be Some when adding new imm")
507 .add_imm(imm);
508 }
509
510 fn local_seal_epoch(&mut self, next_epoch: HummockEpoch) -> HummockEpoch {
511 assert!(!self.is_destroyed);
512 let data = self
513 .current_epoch_data
514 .as_mut()
515 .expect("should be Some when seal new epoch");
516 let current_epoch = data.epoch;
517 debug!(
518 instance_id = self.instance_id,
519 next_epoch, current_epoch, "local seal epoch"
520 );
521 assert_gt!(next_epoch, current_epoch);
522 let epoch_data = replace(data, LocalInstanceEpochData::new(next_epoch));
523 if !epoch_data.is_empty() {
524 self.sealed_data.push_front(epoch_data);
525 }
526 current_epoch
527 }
528
529 fn ack_flushed(&mut self, imm_ids: impl Iterator<Item = SharedBufferBatchId>) {
531 for imm_id in imm_ids {
532 assert_eq!(self.flushing_imms.pop_back().expect("should exist"), imm_id);
533 }
534 }
535
536 fn spill(&mut self, epoch: HummockEpoch) -> Vec<UploaderImm> {
537 let imms = if let Some(oldest_sealed_epoch) = self.sealed_data.back() {
538 match oldest_sealed_epoch.epoch.cmp(&epoch) {
539 Ordering::Less => {
540 unreachable!(
541 "should not spill at this epoch because there \
542 is unspilled data in previous epoch: prev epoch {}, spill epoch {}",
543 oldest_sealed_epoch.epoch, epoch
544 );
545 }
546 Ordering::Equal => {
547 let epoch_data = self.sealed_data.pop_back().unwrap();
548 assert_eq!(epoch, epoch_data.epoch);
549 epoch_data.imms
550 }
551 Ordering::Greater => VecDeque::new(),
552 }
553 } else {
554 let Some(current_epoch_data) = &mut self.current_epoch_data else {
555 return Vec::new();
556 };
557 match current_epoch_data.epoch.cmp(&epoch) {
558 Ordering::Less => {
559 assert!(
560 current_epoch_data.imms.is_empty(),
561 "should not spill at this epoch because there \
562 is unspilled data in current epoch epoch {}, spill epoch {}",
563 current_epoch_data.epoch,
564 epoch
565 );
566 VecDeque::new()
567 }
568 Ordering::Equal => {
569 if !current_epoch_data.imms.is_empty() {
570 current_epoch_data.has_spilled = true;
571 take(&mut current_epoch_data.imms)
572 } else {
573 VecDeque::new()
574 }
575 }
576 Ordering::Greater => VecDeque::new(),
577 }
578 };
579 self.add_flushing_imm(imms.iter().rev().map(|imm| imm.batch_id()));
580 imms.into_iter().collect()
581 }
582
583 fn add_flushing_imm(&mut self, imm_ids: impl Iterator<Item = SharedBufferBatchId>) {
584 for imm_id in imm_ids {
585 if let Some(prev_imm_id) = self.flushing_imms.front() {
586 assert_gt!(imm_id, *prev_imm_id);
587 }
588 self.flushing_imms.push_front(imm_id);
589 }
590 }
591
592 fn sync(&mut self, epoch: HummockEpoch) -> Vec<UploaderImm> {
595 let mut ret = Vec::new();
597 while let Some(epoch_data) = self.sealed_data.back()
598 && epoch_data.epoch() <= epoch
599 {
600 let imms = self.sealed_data.pop_back().expect("checked exist").imms;
601 self.add_flushing_imm(imms.iter().rev().map(|imm| imm.batch_id()));
602 ret.extend(imms.into_iter().rev());
603 }
604 ret.reverse();
606 if let Some(latest_epoch_data) = &self.current_epoch_data {
607 if latest_epoch_data.epoch <= epoch {
608 assert!(self.sealed_data.is_empty());
609 assert!(latest_epoch_data.is_empty());
610 assert!(!latest_epoch_data.has_spilled);
611 if cfg!(debug_assertions) {
612 panic!(
613 "sync epoch exceeds latest epoch, and the current instance should have been archived"
614 );
615 }
616 warn!(
617 instance_id = self.instance_id,
618 table_id = self.table_id.table_id,
619 "sync epoch exceeds latest epoch, and the current instance should have be archived"
620 );
621 self.current_epoch_data = None;
622 }
623 }
624 ret
625 }
626
627 fn assert_after_epoch(&self, epoch: HummockEpoch) {
628 if let Some(oldest_sealed_data) = self.sealed_data.back() {
629 assert!(!oldest_sealed_data.imms.is_empty());
630 assert_gt!(oldest_sealed_data.epoch, epoch);
631 } else if let Some(current_data) = &self.current_epoch_data {
632 if current_data.epoch <= epoch {
633 assert!(current_data.imms.is_empty() && !current_data.has_spilled);
634 }
635 }
636 }
637
638 fn is_finished(&self) -> bool {
639 self.is_destroyed && self.sealed_data.is_empty()
640 }
641}
642
643struct TableUnsyncData {
644 table_id: TableId,
645 instance_data: HashMap<LocalInstanceId, LocalInstanceUnsyncData>,
646 #[expect(clippy::type_complexity)]
647 table_watermarks: Option<(
648 WatermarkDirection,
649 BTreeMap<HummockEpoch, (Vec<VnodeWatermark>, BitmapBuilder)>,
650 WatermarkSerdeType,
651 )>,
652 spill_tasks: BTreeMap<HummockEpoch, VecDeque<UploadingTaskId>>,
653 unsync_epochs: BTreeMap<HummockEpoch, ()>,
654 stopped_next_epoch: Option<HummockEpoch>,
658 syncing_epochs: VecDeque<HummockEpoch>,
660 max_synced_epoch: Option<HummockEpoch>,
661}
662
663impl TableUnsyncData {
664 fn new(table_id: TableId, committed_epoch: Option<HummockEpoch>) -> Self {
665 Self {
666 table_id,
667 instance_data: Default::default(),
668 table_watermarks: None,
669 spill_tasks: Default::default(),
670 unsync_epochs: Default::default(),
671 stopped_next_epoch: None,
672 syncing_epochs: Default::default(),
673 max_synced_epoch: committed_epoch,
674 }
675 }
676
677 fn new_epoch(&mut self, epoch: HummockEpoch) {
678 debug!(table_id = ?self.table_id, epoch, "table new epoch");
679 if let Some(latest_epoch) = self.max_epoch() {
680 assert_gt!(epoch, latest_epoch);
681 }
682 self.unsync_epochs.insert(epoch, ());
683 }
684
685 #[expect(clippy::type_complexity)]
686 fn sync(
687 &mut self,
688 epoch: HummockEpoch,
689 ) -> (
690 impl Iterator<Item = (LocalInstanceId, Vec<UploaderImm>)> + '_,
691 Option<(
692 WatermarkDirection,
693 impl Iterator<Item = (HummockEpoch, Vec<VnodeWatermark>)> + use<>,
694 WatermarkSerdeType,
695 )>,
696 impl Iterator<Item = UploadingTaskId> + use<>,
697 BTreeMap<HummockEpoch, ()>,
698 ) {
699 if let Some(prev_epoch) = self.max_sync_epoch() {
700 assert_gt!(epoch, prev_epoch)
701 }
702 let epochs = take_before_epoch(&mut self.unsync_epochs, epoch);
703 assert_eq!(
704 *epochs.last_key_value().expect("non-empty").0,
705 epoch,
706 "{epochs:?} {epoch} {:?}",
707 self.table_id
708 );
709 self.syncing_epochs.push_front(epoch);
710 (
711 self.instance_data
712 .iter_mut()
713 .map(move |(instance_id, data)| (*instance_id, data.sync(epoch))),
714 self.table_watermarks
715 .as_mut()
716 .map(|(direction, watermarks, watermark_type)| {
717 let watermarks = take_before_epoch(watermarks, epoch)
718 .into_iter()
719 .map(|(epoch, (watermarks, _))| (epoch, watermarks));
720 (*direction, watermarks, *watermark_type)
721 }),
722 take_before_epoch(&mut self.spill_tasks, epoch)
723 .into_values()
724 .flat_map(|tasks| tasks.into_iter()),
725 epochs,
726 )
727 }
728
729 fn ack_synced(&mut self, sync_epoch: HummockEpoch) {
730 let min_sync_epoch = self.syncing_epochs.pop_back().expect("should exist");
731 assert_eq!(sync_epoch, min_sync_epoch);
732 self.max_synced_epoch = Some(sync_epoch);
733 }
734
735 fn ack_committed(&mut self, committed_epoch: HummockEpoch) {
736 let synced_epoch_advanced = {
737 if let Some(max_synced_epoch) = self.max_synced_epoch
738 && max_synced_epoch >= committed_epoch
739 {
740 false
741 } else {
742 true
743 }
744 };
745 if synced_epoch_advanced {
746 self.max_synced_epoch = Some(committed_epoch);
747 if let Some(min_syncing_epoch) = self.syncing_epochs.back() {
748 assert_gt!(*min_syncing_epoch, committed_epoch);
749 }
750 self.assert_after_epoch(committed_epoch);
751 }
752 }
753
754 fn assert_after_epoch(&self, epoch: HummockEpoch) {
755 self.instance_data
756 .values()
757 .for_each(|instance_data| instance_data.assert_after_epoch(epoch));
758 if let Some((_, watermarks, _)) = &self.table_watermarks
759 && let Some((oldest_epoch, _)) = watermarks.first_key_value()
760 {
761 assert_gt!(*oldest_epoch, epoch);
762 }
763 }
764
765 fn max_sync_epoch(&self) -> Option<HummockEpoch> {
766 self.syncing_epochs
767 .front()
768 .cloned()
769 .or(self.max_synced_epoch)
770 }
771
772 fn max_epoch(&self) -> Option<HummockEpoch> {
773 self.unsync_epochs
774 .last_key_value()
775 .map(|(epoch, _)| *epoch)
776 .or_else(|| self.max_sync_epoch())
777 }
778
779 fn is_empty(&self) -> bool {
780 self.instance_data.is_empty()
781 && self.syncing_epochs.is_empty()
782 && self.unsync_epochs.is_empty()
783 }
784}
785
786#[derive(Eq, Hash, PartialEq, Copy, Clone)]
787struct UnsyncEpochId(HummockEpoch, TableId);
788
789impl UnsyncEpochId {
790 fn epoch(&self) -> HummockEpoch {
791 self.0
792 }
793}
794
795fn get_unsync_epoch_id(epoch: HummockEpoch, table_ids: &HashSet<TableId>) -> Option<UnsyncEpochId> {
796 table_ids
797 .iter()
798 .min()
799 .map(|table_id| UnsyncEpochId(epoch, *table_id))
800}
801
802#[derive(Default)]
803struct UnsyncData {
808 table_data: HashMap<TableId, TableUnsyncData>,
809 instance_table_id: HashMap<LocalInstanceId, TableId>,
811 unsync_epochs: HashMap<UnsyncEpochId, HashSet<TableId>>,
812 spilled_data: HashMap<UploadingTaskId, (Arc<StagingSstableInfo>, HashSet<TableId>)>,
813}
814
815impl UnsyncData {
816 fn init_instance(
817 &mut self,
818 table_id: TableId,
819 instance_id: LocalInstanceId,
820 init_epoch: HummockEpoch,
821 ) {
822 debug!(
823 table_id = table_id.table_id,
824 instance_id, init_epoch, "init epoch"
825 );
826 let table_data = self
827 .table_data
828 .get_mut(&table_id)
829 .unwrap_or_else(|| panic!("should exist. {table_id:?}"));
830 assert!(
831 table_data
832 .instance_data
833 .insert(
834 instance_id,
835 LocalInstanceUnsyncData::new(table_id, instance_id, init_epoch)
836 )
837 .is_none()
838 );
839 assert!(
840 self.instance_table_id
841 .insert(instance_id, table_id)
842 .is_none()
843 );
844 assert!(table_data.unsync_epochs.contains_key(&init_epoch));
845 }
846
847 fn instance_data(
848 &mut self,
849 instance_id: LocalInstanceId,
850 ) -> Option<&mut LocalInstanceUnsyncData> {
851 self.instance_table_id
852 .get_mut(&instance_id)
853 .cloned()
854 .map(move |table_id| {
855 self.table_data
856 .get_mut(&table_id)
857 .expect("should exist")
858 .instance_data
859 .get_mut(&instance_id)
860 .expect("should exist")
861 })
862 }
863
864 fn add_imm(&mut self, instance_id: LocalInstanceId, imm: UploaderImm) {
865 self.instance_data(instance_id)
866 .expect("should exist")
867 .add_imm(imm);
868 }
869
870 fn local_seal_epoch(
871 &mut self,
872 instance_id: LocalInstanceId,
873 next_epoch: HummockEpoch,
874 opts: SealCurrentEpochOptions,
875 ) {
876 let table_id = self.instance_table_id[&instance_id];
877 let table_data = self.table_data.get_mut(&table_id).expect("should exist");
878 let instance_data = table_data
879 .instance_data
880 .get_mut(&instance_id)
881 .expect("should exist");
882 let epoch = instance_data.local_seal_epoch(next_epoch);
883 if !table_data.unsync_epochs.contains_key(&next_epoch) {
887 if let Some(stopped_next_epoch) = table_data.stopped_next_epoch {
888 if stopped_next_epoch != next_epoch {
889 let table_id = table_data.table_id.table_id;
890 let unsync_epochs = table_data.unsync_epochs.keys().collect_vec();
891 if cfg!(debug_assertions) {
892 panic!(
893 "table_id {} stop epoch {} different to prev stop epoch {}. unsync epochs: {:?}, syncing epochs {:?}, max_synced_epoch {:?}",
894 table_id,
895 next_epoch,
896 stopped_next_epoch,
897 unsync_epochs,
898 table_data.syncing_epochs,
899 table_data.max_synced_epoch
900 );
901 } else {
902 warn!(
903 table_id,
904 stopped_next_epoch,
905 next_epoch,
906 ?unsync_epochs,
907 syncing_epochs = ?table_data.syncing_epochs,
908 max_synced_epoch = ?table_data.max_synced_epoch,
909 "different stop epoch"
910 );
911 }
912 }
913 } else {
914 if let Some(max_epoch) = table_data.max_epoch() {
915 assert_gt!(next_epoch, max_epoch);
916 }
917 debug!(?table_id, epoch, next_epoch, "table data has stopped");
918 table_data.stopped_next_epoch = Some(next_epoch);
919 }
920 }
921 if let Some((direction, table_watermarks, watermark_type)) = opts.table_watermarks {
922 table_data.add_table_watermarks(epoch, table_watermarks, direction, watermark_type);
923 }
924 }
925
926 fn may_destroy_instance(&mut self, instance_id: LocalInstanceId) {
927 if let Some(table_id) = self.instance_table_id.get(&instance_id) {
928 debug!(instance_id, "destroy instance");
929 let table_data = self.table_data.get_mut(table_id).expect("should exist");
930 let instance_data = table_data
931 .instance_data
932 .get_mut(&instance_id)
933 .expect("should exist");
934 assert!(
935 !instance_data.is_destroyed,
936 "cannot destroy an instance for twice"
937 );
938 instance_data.is_destroyed = true;
939 }
940 }
941
942 fn clear_tables(&mut self, table_ids: &HashSet<TableId>, task_manager: &mut TaskManager) {
943 for table_id in table_ids {
944 if let Some(table_unsync_data) = self.table_data.remove(table_id) {
945 for task_id in table_unsync_data.spill_tasks.into_values().flatten() {
946 if let Some(task_status) = task_manager.abort_task(task_id) {
947 must_match!(task_status, UploadingTaskStatus::Spilling(spill_table_ids) => {
948 assert!(spill_table_ids.is_subset(table_ids));
949 });
950 }
951 if let Some((_, spill_table_ids)) = self.spilled_data.remove(&task_id) {
952 assert!(spill_table_ids.is_subset(table_ids));
953 }
954 }
955 assert!(
956 table_unsync_data
957 .instance_data
958 .values()
959 .all(|instance| instance.is_destroyed),
960 "should be clear when dropping the read version instance"
961 );
962 for instance_id in table_unsync_data.instance_data.keys() {
963 assert_eq!(
964 *table_id,
965 self.instance_table_id
966 .remove(instance_id)
967 .expect("should exist")
968 );
969 }
970 }
971 }
972 debug_assert!(
973 self.spilled_data
974 .values()
975 .all(|(_, spill_table_ids)| spill_table_ids.is_disjoint(table_ids))
976 );
977 self.unsync_epochs.retain(|_, unsync_epoch_table_ids| {
978 if !unsync_epoch_table_ids.is_disjoint(table_ids) {
979 assert!(unsync_epoch_table_ids.is_subset(table_ids));
980 false
981 } else {
982 true
983 }
984 });
985 assert!(
986 self.instance_table_id
987 .values()
988 .all(|table_id| !table_ids.contains(table_id))
989 );
990 }
991}
992
993impl UploaderData {
994 fn sync(
995 &mut self,
996 context: &UploaderContext,
997 sync_result_sender: oneshot::Sender<HummockResult<SyncedData>>,
998 sync_table_epochs: Vec<(HummockEpoch, HashSet<TableId>)>,
999 ) {
1000 let mut all_table_watermarks = HashMap::new();
1001 let mut uploading_tasks = HashSet::new();
1002 let mut spilled_tasks = BTreeSet::new();
1003 let mut all_table_ids = HashSet::new();
1004
1005 let mut flush_payload = HashMap::new();
1006
1007 for (epoch, table_ids) in &sync_table_epochs {
1008 let epoch = *epoch;
1009 for table_id in table_ids {
1010 assert!(
1011 all_table_ids.insert(*table_id),
1012 "duplicate sync table epoch: {:?} {:?}",
1013 all_table_ids,
1014 sync_table_epochs
1015 );
1016 }
1017 if let Some(UnsyncEpochId(_, min_table_id)) = get_unsync_epoch_id(epoch, table_ids) {
1018 let min_table_id_data = self
1019 .unsync_data
1020 .table_data
1021 .get_mut(&min_table_id)
1022 .expect("should exist");
1023 let epochs = take_before_epoch(&mut min_table_id_data.unsync_epochs.clone(), epoch);
1024 for epoch in epochs.keys() {
1025 assert_eq!(
1026 &self
1027 .unsync_data
1028 .unsync_epochs
1029 .remove(&UnsyncEpochId(*epoch, min_table_id))
1030 .expect("should exist"),
1031 table_ids
1032 );
1033 }
1034 for table_id in table_ids {
1035 let table_data = self
1036 .unsync_data
1037 .table_data
1038 .get_mut(table_id)
1039 .expect("should exist");
1040 let (unflushed_payload, table_watermarks, task_ids, table_unsync_epochs) =
1041 table_data.sync(epoch);
1042 assert_eq!(table_unsync_epochs, epochs);
1043 for (instance_id, payload) in unflushed_payload {
1044 if !payload.is_empty() {
1045 flush_payload.insert(instance_id, payload);
1046 }
1047 }
1048 table_data.instance_data.retain(|instance_id, data| {
1049 if data.is_finished() {
1051 assert_eq!(
1052 self.unsync_data.instance_table_id.remove(instance_id),
1053 Some(*table_id)
1054 );
1055 false
1056 } else {
1057 true
1058 }
1059 });
1060 if let Some((direction, watermarks, watermark_type)) = table_watermarks {
1061 Self::add_table_watermarks(
1062 &mut all_table_watermarks,
1063 *table_id,
1064 direction,
1065 watermarks,
1066 watermark_type,
1067 );
1068 }
1069 for task_id in task_ids {
1070 if self.unsync_data.spilled_data.contains_key(&task_id) {
1071 spilled_tasks.insert(task_id);
1072 } else {
1073 uploading_tasks.insert(task_id);
1074 }
1075 }
1076 }
1077 }
1078 }
1079
1080 let sync_id = {
1081 let sync_id = self.next_sync_id;
1082 self.next_sync_id += 1;
1083 SyncId(sync_id)
1084 };
1085
1086 if let Some(extra_flush_task_id) = self.task_manager.sync(
1087 context,
1088 sync_id,
1089 flush_payload,
1090 uploading_tasks.iter().cloned(),
1091 &all_table_ids,
1092 ) {
1093 uploading_tasks.insert(extra_flush_task_id);
1094 }
1095
1096 let uploaded = spilled_tasks
1098 .iter()
1099 .rev()
1100 .map(|task_id| {
1101 let (sst, spill_table_ids) = self
1102 .unsync_data
1103 .spilled_data
1104 .remove(task_id)
1105 .expect("should exist");
1106 assert!(
1107 spill_table_ids.is_subset(&all_table_ids),
1108 "spilled tabled ids {:?} not a subset of sync table id {:?}",
1109 spill_table_ids,
1110 all_table_ids
1111 );
1112 sst
1113 })
1114 .collect();
1115
1116 self.syncing_data.insert(
1117 sync_id,
1118 SyncingData {
1119 sync_table_epochs,
1120 remaining_uploading_tasks: uploading_tasks,
1121 uploaded,
1122 table_watermarks: all_table_watermarks,
1123 sync_result_sender,
1124 },
1125 );
1126
1127 self.check_upload_task_consistency();
1128 }
1129}
1130
1131impl UnsyncData {
1132 fn ack_flushed(&mut self, sstable_info: &StagingSstableInfo) {
1133 for (instance_id, imm_ids) in sstable_info.imm_ids() {
1134 if let Some(instance_data) = self.instance_data(*instance_id) {
1135 instance_data.ack_flushed(imm_ids.iter().rev().cloned());
1137 }
1138 }
1139 }
1140}
1141
1142struct SyncingData {
1143 sync_table_epochs: Vec<(HummockEpoch, HashSet<TableId>)>,
1144 remaining_uploading_tasks: HashSet<UploadingTaskId>,
1145 uploaded: VecDeque<Arc<StagingSstableInfo>>,
1147 table_watermarks: HashMap<TableId, TableWatermarks>,
1148 sync_result_sender: oneshot::Sender<HummockResult<SyncedData>>,
1149}
1150
1151#[derive(Debug)]
1152pub struct SyncedData {
1153 pub uploaded_ssts: VecDeque<Arc<StagingSstableInfo>>,
1154 pub table_watermarks: HashMap<TableId, TableWatermarks>,
1155}
1156
1157struct UploaderContext {
1158 pinned_version: PinnedVersion,
1159 spawn_upload_task: SpawnUploadTask,
1161 buffer_tracker: BufferTracker,
1162
1163 stats: Arc<HummockStateStoreMetrics>,
1164}
1165
1166impl UploaderContext {
1167 fn new(
1168 pinned_version: PinnedVersion,
1169 spawn_upload_task: SpawnUploadTask,
1170 buffer_tracker: BufferTracker,
1171 stats: Arc<HummockStateStoreMetrics>,
1172 ) -> Self {
1173 UploaderContext {
1174 pinned_version,
1175 spawn_upload_task,
1176 buffer_tracker,
1177 stats,
1178 }
1179 }
1180}
1181
1182#[derive(PartialEq, Eq, Hash, PartialOrd, Ord, Copy, Clone, Debug)]
1183struct SyncId(usize);
1184
1185#[derive(Default)]
1186struct UploaderData {
1187 unsync_data: UnsyncData,
1188
1189 syncing_data: BTreeMap<SyncId, SyncingData>,
1190
1191 task_manager: TaskManager,
1192 next_sync_id: usize,
1193}
1194
1195impl UploaderData {
1196 fn abort(self, err: impl Fn() -> HummockError) {
1197 self.task_manager.abort_all_tasks();
1198 for syncing_data in self.syncing_data.into_values() {
1199 send_sync_result(syncing_data.sync_result_sender, Err(err()));
1200 }
1201 }
1202
1203 fn clear_tables(&mut self, table_ids: HashSet<TableId>) {
1204 if table_ids.is_empty() {
1205 return;
1206 }
1207 self.unsync_data
1208 .clear_tables(&table_ids, &mut self.task_manager);
1209 self.syncing_data.retain(|sync_id, syncing_data| {
1210 if syncing_data
1211 .sync_table_epochs
1212 .iter()
1213 .any(|(_, sync_table_ids)| !sync_table_ids.is_disjoint(&table_ids))
1214 {
1215 assert!(
1216 syncing_data
1217 .sync_table_epochs
1218 .iter()
1219 .all(|(_, sync_table_ids)| sync_table_ids.is_subset(&table_ids))
1220 );
1221 for task_id in &syncing_data.remaining_uploading_tasks {
1222 match self
1223 .task_manager
1224 .abort_task(*task_id)
1225 .expect("should exist")
1226 {
1227 UploadingTaskStatus::Spilling(spill_table_ids) => {
1228 assert!(spill_table_ids.is_subset(&table_ids));
1229 }
1230 UploadingTaskStatus::Sync(task_sync_id) => {
1231 assert_eq!(sync_id, &task_sync_id);
1232 }
1233 }
1234 }
1235 false
1236 } else {
1237 true
1238 }
1239 });
1240
1241 self.check_upload_task_consistency();
1242 }
1243
1244 fn min_uncommitted_sst_id(&self) -> Option<HummockSstableObjectId> {
1245 self.unsync_data
1246 .spilled_data
1247 .values()
1248 .map(|(s, _)| s)
1249 .chain(self.syncing_data.values().flat_map(|s| s.uploaded.iter()))
1250 .filter_map(|s| {
1251 s.sstable_infos()
1252 .iter()
1253 .chain(s.old_value_sstable_infos())
1254 .map(|s| s.sst_info.sst_id)
1255 .min()
1256 })
1257 .min()
1258 }
1259}
1260
1261struct ErrState {
1262 failed_sync_table_epochs: Vec<(HummockEpoch, HashSet<TableId>)>,
1263 reason: String,
1264}
1265
1266enum UploaderState {
1267 Working(UploaderData),
1268 Err(ErrState),
1269}
1270
1271pub struct HummockUploader {
1284 state: UploaderState,
1285
1286 context: UploaderContext,
1287}
1288
1289impl HummockUploader {
1290 pub(super) fn new(
1291 state_store_metrics: Arc<HummockStateStoreMetrics>,
1292 pinned_version: PinnedVersion,
1293 spawn_upload_task: SpawnUploadTask,
1294 buffer_tracker: BufferTracker,
1295 ) -> Self {
1296 Self {
1297 state: UploaderState::Working(UploaderData::default()),
1298 context: UploaderContext::new(
1299 pinned_version,
1300 spawn_upload_task,
1301 buffer_tracker,
1302 state_store_metrics,
1303 ),
1304 }
1305 }
1306
1307 pub(super) fn buffer_tracker(&self) -> &BufferTracker {
1308 &self.context.buffer_tracker
1309 }
1310
1311 pub(super) fn hummock_version(&self) -> &PinnedVersion {
1312 &self.context.pinned_version
1313 }
1314
1315 pub(super) fn add_imm(&mut self, instance_id: LocalInstanceId, imm: ImmutableMemtable) {
1316 let UploaderState::Working(data) = &mut self.state else {
1317 return;
1318 };
1319 let imm = UploaderImm::new(imm, &self.context);
1320 data.unsync_data.add_imm(instance_id, imm);
1321 }
1322
1323 pub(super) fn init_instance(
1324 &mut self,
1325 instance_id: LocalInstanceId,
1326 table_id: TableId,
1327 init_epoch: HummockEpoch,
1328 ) {
1329 let UploaderState::Working(data) = &mut self.state else {
1330 return;
1331 };
1332 data.unsync_data
1333 .init_instance(table_id, instance_id, init_epoch);
1334 }
1335
1336 pub(super) fn local_seal_epoch(
1337 &mut self,
1338 instance_id: LocalInstanceId,
1339 next_epoch: HummockEpoch,
1340 opts: SealCurrentEpochOptions,
1341 ) {
1342 let UploaderState::Working(data) = &mut self.state else {
1343 return;
1344 };
1345 data.unsync_data
1346 .local_seal_epoch(instance_id, next_epoch, opts);
1347 }
1348
1349 pub(super) fn start_epoch(&mut self, epoch: HummockEpoch, table_ids: HashSet<TableId>) {
1350 let UploaderState::Working(data) = &mut self.state else {
1351 return;
1352 };
1353 debug!(epoch, ?table_ids, "start epoch");
1354 for table_id in &table_ids {
1355 let table_data = data
1356 .unsync_data
1357 .table_data
1358 .entry(*table_id)
1359 .or_insert_with(|| {
1360 TableUnsyncData::new(
1361 *table_id,
1362 self.context.pinned_version.table_committed_epoch(*table_id),
1363 )
1364 });
1365 table_data.new_epoch(epoch);
1366 }
1367 if let Some(unsync_epoch_id) = get_unsync_epoch_id(epoch, &table_ids) {
1368 assert!(
1369 data.unsync_data
1370 .unsync_epochs
1371 .insert(unsync_epoch_id, table_ids)
1372 .is_none()
1373 );
1374 }
1375 }
1376
1377 pub(super) fn start_sync_epoch(
1378 &mut self,
1379 sync_result_sender: oneshot::Sender<HummockResult<SyncedData>>,
1380 sync_table_epochs: Vec<(HummockEpoch, HashSet<TableId>)>,
1381 ) {
1382 let data = match &mut self.state {
1383 UploaderState::Working(data) => data,
1384 UploaderState::Err(ErrState {
1385 failed_sync_table_epochs,
1386 reason,
1387 }) => {
1388 let result = Err(HummockError::other(format!(
1389 "previous sync epoch {:?} failed due to [{}]",
1390 failed_sync_table_epochs, reason
1391 )));
1392 send_sync_result(sync_result_sender, result);
1393 return;
1394 }
1395 };
1396 debug!(?sync_table_epochs, "start sync epoch");
1397
1398 data.sync(&self.context, sync_result_sender, sync_table_epochs);
1399
1400 data.may_notify_sync_task(&self.context);
1401
1402 self.context
1403 .stats
1404 .uploader_syncing_epoch_count
1405 .set(data.syncing_data.len() as _);
1406 }
1407
1408 pub(crate) fn update_pinned_version(&mut self, pinned_version: PinnedVersion) {
1409 if let UploaderState::Working(data) = &mut self.state {
1410 for (table_id, info) in pinned_version.state_table_info.info() {
1412 if let Some(table_data) = data.unsync_data.table_data.get_mut(table_id) {
1413 table_data.ack_committed(info.committed_epoch);
1414 }
1415 }
1416 }
1417 self.context.pinned_version = pinned_version;
1418 }
1419
1420 pub(crate) fn may_flush(&mut self) -> bool {
1421 let UploaderState::Working(data) = &mut self.state else {
1422 return false;
1423 };
1424 if self.context.buffer_tracker.need_flush() {
1425 let mut spiller = Spiller::new(&mut data.unsync_data);
1426 let mut curr_batch_flush_size = 0;
1427 while self
1429 .context
1430 .buffer_tracker
1431 .need_more_flush(curr_batch_flush_size)
1432 && let Some((epoch, payload, spilled_table_ids)) = spiller.next_spilled_payload()
1433 {
1434 assert!(!payload.is_empty());
1435 {
1436 let (task_id, task_size, spilled_table_ids) =
1437 data.task_manager
1438 .spill(&self.context, spilled_table_ids, payload);
1439 for table_id in spilled_table_ids {
1440 spiller
1441 .unsync_data()
1442 .table_data
1443 .get_mut(table_id)
1444 .expect("should exist")
1445 .spill_tasks
1446 .entry(epoch)
1447 .or_default()
1448 .push_front(task_id);
1449 }
1450 curr_batch_flush_size += task_size;
1451 }
1452 }
1453 data.check_upload_task_consistency();
1454 curr_batch_flush_size > 0
1455 } else {
1456 false
1457 }
1458 }
1459
1460 pub(crate) fn clear(&mut self, table_ids: Option<HashSet<TableId>>) {
1461 if let Some(table_ids) = table_ids {
1462 if let UploaderState::Working(data) = &mut self.state {
1463 data.clear_tables(table_ids);
1464 }
1465 } else {
1466 if let UploaderState::Working(data) = replace(
1467 &mut self.state,
1468 UploaderState::Working(UploaderData::default()),
1469 ) {
1470 data.abort(|| HummockError::other("uploader is reset"));
1471 }
1472
1473 self.context.stats.uploader_syncing_epoch_count.set(0);
1474 }
1475 }
1476
1477 pub(crate) fn may_destroy_instance(&mut self, instance_id: LocalInstanceId) {
1478 let UploaderState::Working(data) = &mut self.state else {
1479 return;
1480 };
1481 data.unsync_data.may_destroy_instance(instance_id);
1482 }
1483
1484 pub(crate) fn min_uncommitted_sst_id(&self) -> Option<HummockSstableObjectId> {
1485 if let UploaderState::Working(ref u) = self.state {
1486 u.min_uncommitted_sst_id()
1487 } else {
1488 None
1489 }
1490 }
1491}
1492
1493impl UploaderData {
1494 fn may_notify_sync_task(&mut self, context: &UploaderContext) {
1495 while let Some((_, syncing_data)) = self.syncing_data.first_key_value()
1496 && syncing_data.remaining_uploading_tasks.is_empty()
1497 {
1498 let (_, syncing_data) = self.syncing_data.pop_first().expect("non-empty");
1499 let SyncingData {
1500 sync_table_epochs,
1501 remaining_uploading_tasks: _,
1502 uploaded,
1503 table_watermarks,
1504 sync_result_sender,
1505 } = syncing_data;
1506 context
1507 .stats
1508 .uploader_syncing_epoch_count
1509 .set(self.syncing_data.len() as _);
1510
1511 for (sync_epoch, table_ids) in sync_table_epochs {
1512 for table_id in table_ids {
1513 if let Some(table_data) = self.unsync_data.table_data.get_mut(&table_id) {
1514 table_data.ack_synced(sync_epoch);
1515 if table_data.is_empty() {
1516 self.unsync_data.table_data.remove(&table_id);
1517 }
1518 }
1519 }
1520 }
1521
1522 send_sync_result(
1523 sync_result_sender,
1524 Ok(SyncedData {
1525 uploaded_ssts: uploaded,
1526 table_watermarks,
1527 }),
1528 )
1529 }
1530 }
1531
1532 fn check_upload_task_consistency(&self) {
1533 #[cfg(debug_assertions)]
1534 {
1535 let mut spill_task_table_id_from_data: HashMap<_, HashSet<_>> = HashMap::new();
1536 for table_data in self.unsync_data.table_data.values() {
1537 for task_id in table_data
1538 .spill_tasks
1539 .iter()
1540 .flat_map(|(_, tasks)| tasks.iter())
1541 {
1542 assert!(
1543 spill_task_table_id_from_data
1544 .entry(*task_id)
1545 .or_default()
1546 .insert(table_data.table_id)
1547 );
1548 }
1549 }
1550 let syncing_task_id_from_data: HashMap<_, HashSet<_>> = self
1551 .syncing_data
1552 .iter()
1553 .filter_map(|(sync_id, data)| {
1554 if data.remaining_uploading_tasks.is_empty() {
1555 None
1556 } else {
1557 Some((*sync_id, data.remaining_uploading_tasks.clone()))
1558 }
1559 })
1560 .collect();
1561
1562 let mut spill_task_table_id_from_manager: HashMap<_, HashSet<_>> = HashMap::new();
1563 for (task_id, (_, table_ids)) in &self.unsync_data.spilled_data {
1564 spill_task_table_id_from_manager.insert(*task_id, table_ids.clone());
1565 }
1566 let mut syncing_task_from_manager: HashMap<_, HashSet<_>> = HashMap::new();
1567 for (task_id, status) in self.task_manager.tasks() {
1568 match status {
1569 UploadingTaskStatus::Spilling(table_ids) => {
1570 assert!(
1571 spill_task_table_id_from_manager
1572 .insert(task_id, table_ids.clone())
1573 .is_none()
1574 );
1575 }
1576 UploadingTaskStatus::Sync(sync_id) => {
1577 assert!(
1578 syncing_task_from_manager
1579 .entry(*sync_id)
1580 .or_default()
1581 .insert(task_id)
1582 );
1583 }
1584 }
1585 }
1586 assert_eq!(
1587 spill_task_table_id_from_data,
1588 spill_task_table_id_from_manager
1589 );
1590 assert_eq!(syncing_task_id_from_data, syncing_task_from_manager);
1591 }
1592 }
1593}
1594
1595impl HummockUploader {
1596 pub(super) fn next_uploaded_sst(
1597 &mut self,
1598 ) -> impl Future<Output = Arc<StagingSstableInfo>> + '_ {
1599 poll_fn(|cx| {
1600 let UploaderState::Working(data) = &mut self.state else {
1601 return Poll::Pending;
1602 };
1603
1604 if let Some((task_id, status, result)) = ready!(data.task_manager.poll_task_result(cx))
1605 {
1606 match result {
1607 Ok(sst) => {
1608 data.unsync_data.ack_flushed(&sst);
1609 match status {
1610 UploadingTaskStatus::Sync(sync_id) => {
1611 let syncing_data =
1612 data.syncing_data.get_mut(&sync_id).expect("should exist");
1613 syncing_data.uploaded.push_front(sst.clone());
1614 assert!(syncing_data.remaining_uploading_tasks.remove(&task_id));
1615 data.may_notify_sync_task(&self.context);
1616 }
1617 UploadingTaskStatus::Spilling(table_ids) => {
1618 data.unsync_data
1619 .spilled_data
1620 .insert(task_id, (sst.clone(), table_ids));
1621 }
1622 }
1623 data.check_upload_task_consistency();
1624 Poll::Ready(sst)
1625 }
1626 Err((sync_id, e)) => {
1627 let syncing_data =
1628 data.syncing_data.remove(&sync_id).expect("should exist");
1629 let failed_epochs = syncing_data.sync_table_epochs.clone();
1630 let data = must_match!(replace(
1631 &mut self.state,
1632 UploaderState::Err(ErrState {
1633 failed_sync_table_epochs: syncing_data.sync_table_epochs,
1634 reason: e.as_report().to_string(),
1635 }),
1636 ), UploaderState::Working(data) => data);
1637
1638 let _ = syncing_data
1639 .sync_result_sender
1640 .send(Err(HummockError::other(format!(
1641 "failed to sync: {:?}",
1642 e.as_report()
1643 ))));
1644
1645 data.abort(|| {
1646 HummockError::other(format!(
1647 "previous epoch {:?} failed to sync",
1648 failed_epochs
1649 ))
1650 });
1651 Poll::Pending
1652 }
1653 }
1654 } else {
1655 Poll::Pending
1656 }
1657 })
1658 }
1659}
1660
1661#[cfg(test)]
1662pub(crate) mod tests {
1663 use std::collections::{HashMap, HashSet};
1664 use std::future::{Future, poll_fn};
1665 use std::ops::Deref;
1666 use std::pin::pin;
1667 use std::sync::Arc;
1668 use std::sync::atomic::AtomicUsize;
1669 use std::sync::atomic::Ordering::SeqCst;
1670 use std::task::Poll;
1671
1672 use futures::FutureExt;
1673 use risingwave_common::catalog::TableId;
1674 use risingwave_common::util::epoch::EpochExt;
1675 use risingwave_hummock_sdk::HummockEpoch;
1676 use tokio::sync::oneshot;
1677
1678 use super::test_utils::*;
1679 use crate::hummock::event_handler::TEST_LOCAL_INSTANCE_ID;
1680 use crate::hummock::event_handler::uploader::{
1681 HummockUploader, SyncedData, UploadingTask, get_payload_imm_ids,
1682 };
1683 use crate::hummock::{HummockError, HummockResult};
1684 use crate::opts::StorageOpts;
1685
1686 impl HummockUploader {
1687 pub(super) fn start_single_epoch_sync(
1688 &mut self,
1689 epoch: HummockEpoch,
1690 sync_result_sender: oneshot::Sender<HummockResult<SyncedData>>,
1691 table_ids: HashSet<TableId>,
1692 ) {
1693 self.start_sync_epoch(sync_result_sender, vec![(epoch, table_ids)]);
1694 }
1695 }
1696
1697 #[tokio::test]
1698 pub async fn test_uploading_task_future() {
1699 let uploader_context = test_uploader_context(dummy_success_upload_future);
1700
1701 let imm = gen_imm(INITIAL_EPOCH).await;
1702 let imm_size = imm.size();
1703 let imm_ids = get_imm_ids(vec![&imm]);
1704 let mut task = UploadingTask::from_vec(vec![imm], &uploader_context);
1705 assert_eq!(imm_size, task.task_info.task_size);
1706 assert_eq!(imm_ids, task.task_info.imm_ids);
1707 assert_eq!(vec![INITIAL_EPOCH], task.task_info.epochs);
1708 let output = poll_fn(|cx| task.poll_result(cx)).await.unwrap();
1709 assert_eq!(
1710 output.sstable_infos(),
1711 &dummy_success_upload_output().new_value_ssts
1712 );
1713 assert_eq!(imm_size, output.imm_size());
1714 assert_eq!(&imm_ids, output.imm_ids());
1715 assert_eq!(&vec![INITIAL_EPOCH], output.epochs());
1716
1717 let uploader_context = test_uploader_context(dummy_fail_upload_future);
1718 let imm = gen_imm(INITIAL_EPOCH).await;
1719 let mut task = UploadingTask::from_vec(vec![imm], &uploader_context);
1720 let _ = poll_fn(|cx| task.poll_result(cx)).await.unwrap_err();
1721 }
1722
1723 #[tokio::test]
1724 pub async fn test_uploading_task_poll_result() {
1725 let uploader_context = test_uploader_context(dummy_success_upload_future);
1726 let mut task =
1727 UploadingTask::from_vec(vec![gen_imm(INITIAL_EPOCH).await], &uploader_context);
1728 let output = poll_fn(|cx| task.poll_result(cx)).await.unwrap();
1729 assert_eq!(
1730 output.sstable_infos(),
1731 &dummy_success_upload_output().new_value_ssts
1732 );
1733
1734 let uploader_context = test_uploader_context(dummy_fail_upload_future);
1735 let mut task =
1736 UploadingTask::from_vec(vec![gen_imm(INITIAL_EPOCH).await], &uploader_context);
1737 let _ = poll_fn(|cx| task.poll_result(cx)).await.unwrap_err();
1738 }
1739
1740 #[tokio::test]
1741 async fn test_uploading_task_poll_ok_with_retry() {
1742 let run_count = Arc::new(AtomicUsize::new(0));
1743 let fail_num = 10;
1744 let run_count_clone = run_count.clone();
1745 let uploader_context = test_uploader_context(move |payload, info| {
1746 let run_count = run_count.clone();
1747 async move {
1748 let ret = if run_count.load(SeqCst) < fail_num {
1750 Err(HummockError::other("fail"))
1751 } else {
1752 dummy_success_upload_future(payload, info).await
1753 };
1754 run_count.fetch_add(1, SeqCst);
1755 ret
1756 }
1757 });
1758 let mut task =
1759 UploadingTask::from_vec(vec![gen_imm(INITIAL_EPOCH).await], &uploader_context);
1760 let output = poll_fn(|cx| task.poll_ok_with_retry(cx)).await;
1761 assert_eq!(fail_num + 1, run_count_clone.load(SeqCst));
1762 assert_eq!(
1763 output.sstable_infos(),
1764 &dummy_success_upload_output().new_value_ssts
1765 );
1766 }
1767
1768 #[tokio::test]
1769 async fn test_uploader_basic() {
1770 let mut uploader = test_uploader(dummy_success_upload_future);
1771 let epoch1 = INITIAL_EPOCH.next_epoch();
1772 uploader.start_epochs_for_test([epoch1]);
1773 let imm = gen_imm(epoch1).await;
1774 uploader.init_instance(TEST_LOCAL_INSTANCE_ID, TEST_TABLE_ID, epoch1);
1775 uploader.add_imm(TEST_LOCAL_INSTANCE_ID, imm.clone());
1776 uploader.local_seal_epoch_for_test(TEST_LOCAL_INSTANCE_ID, epoch1);
1777
1778 let (sync_tx, sync_rx) = oneshot::channel();
1779 uploader.start_single_epoch_sync(epoch1, sync_tx, HashSet::from_iter([TEST_TABLE_ID]));
1780 assert_eq!(epoch1 as HummockEpoch, uploader.test_max_syncing_epoch());
1781 assert_eq!(1, uploader.data().syncing_data.len());
1782 let (_, syncing_data) = uploader.data().syncing_data.first_key_value().unwrap();
1783 assert_eq!(epoch1 as HummockEpoch, syncing_data.sync_table_epochs[0].0);
1784 assert!(syncing_data.uploaded.is_empty());
1785 assert!(!syncing_data.remaining_uploading_tasks.is_empty());
1786
1787 let staging_sst = uploader.next_uploaded_sst().await;
1788 assert_eq!(&vec![epoch1], staging_sst.epochs());
1789 assert_eq!(
1790 &HashMap::from_iter([(TEST_LOCAL_INSTANCE_ID, vec![imm.batch_id()])]),
1791 staging_sst.imm_ids()
1792 );
1793 assert_eq!(
1794 &dummy_success_upload_output().new_value_ssts,
1795 staging_sst.sstable_infos()
1796 );
1797
1798 match sync_rx.await {
1799 Ok(Ok(data)) => {
1800 let SyncedData {
1801 uploaded_ssts,
1802 table_watermarks,
1803 } = data;
1804 assert_eq!(1, uploaded_ssts.len());
1805 let staging_sst = &uploaded_ssts[0];
1806 assert_eq!(&vec![epoch1], staging_sst.epochs());
1807 assert_eq!(
1808 &HashMap::from_iter([(TEST_LOCAL_INSTANCE_ID, vec![imm.batch_id()])]),
1809 staging_sst.imm_ids()
1810 );
1811 assert_eq!(
1812 &dummy_success_upload_output().new_value_ssts,
1813 staging_sst.sstable_infos()
1814 );
1815 assert!(table_watermarks.is_empty());
1816 }
1817 _ => unreachable!(),
1818 };
1819 assert_eq!(epoch1, uploader.test_max_synced_epoch());
1820
1821 let new_pinned_version = uploader
1822 .context
1823 .pinned_version
1824 .new_pin_version(test_hummock_version(epoch1))
1825 .unwrap();
1826 uploader.update_pinned_version(new_pinned_version);
1827 assert_eq!(
1828 epoch1,
1829 uploader
1830 .context
1831 .pinned_version
1832 .table_committed_epoch(TEST_TABLE_ID)
1833 .unwrap()
1834 );
1835 }
1836
1837 #[tokio::test]
1838 async fn test_uploader_destroy_instance_before_sync() {
1839 let mut uploader = test_uploader(dummy_success_upload_future);
1840 let epoch1 = INITIAL_EPOCH.next_epoch();
1841 uploader.start_epochs_for_test([epoch1]);
1842 let imm = gen_imm(epoch1).await;
1843 uploader.init_instance(TEST_LOCAL_INSTANCE_ID, TEST_TABLE_ID, epoch1);
1844 uploader.add_imm(TEST_LOCAL_INSTANCE_ID, imm.clone());
1845 uploader.local_seal_epoch_for_test(TEST_LOCAL_INSTANCE_ID, epoch1);
1846 uploader.may_destroy_instance(TEST_LOCAL_INSTANCE_ID);
1847
1848 let (sync_tx, sync_rx) = oneshot::channel();
1849 uploader.start_single_epoch_sync(epoch1, sync_tx, HashSet::from_iter([TEST_TABLE_ID]));
1850 assert_eq!(epoch1 as HummockEpoch, uploader.test_max_syncing_epoch());
1851 assert_eq!(1, uploader.data().syncing_data.len());
1852 let (_, syncing_data) = uploader.data().syncing_data.first_key_value().unwrap();
1853 assert_eq!(epoch1 as HummockEpoch, syncing_data.sync_table_epochs[0].0);
1854 assert!(syncing_data.uploaded.is_empty());
1855 assert!(!syncing_data.remaining_uploading_tasks.is_empty());
1856
1857 let staging_sst = uploader.next_uploaded_sst().await;
1858 assert_eq!(&vec![epoch1], staging_sst.epochs());
1859 assert_eq!(
1860 &HashMap::from_iter([(TEST_LOCAL_INSTANCE_ID, vec![imm.batch_id()])]),
1861 staging_sst.imm_ids()
1862 );
1863 assert_eq!(
1864 &dummy_success_upload_output().new_value_ssts,
1865 staging_sst.sstable_infos()
1866 );
1867
1868 match sync_rx.await {
1869 Ok(Ok(data)) => {
1870 let SyncedData {
1871 uploaded_ssts,
1872 table_watermarks,
1873 } = data;
1874 assert_eq!(1, uploaded_ssts.len());
1875 let staging_sst = &uploaded_ssts[0];
1876 assert_eq!(&vec![epoch1], staging_sst.epochs());
1877 assert_eq!(
1878 &HashMap::from_iter([(TEST_LOCAL_INSTANCE_ID, vec![imm.batch_id()])]),
1879 staging_sst.imm_ids()
1880 );
1881 assert_eq!(
1882 &dummy_success_upload_output().new_value_ssts,
1883 staging_sst.sstable_infos()
1884 );
1885 assert!(table_watermarks.is_empty());
1886 }
1887 _ => unreachable!(),
1888 };
1889 assert!(
1890 !uploader
1891 .data()
1892 .unsync_data
1893 .table_data
1894 .contains_key(&TEST_TABLE_ID)
1895 );
1896 }
1897
1898 #[tokio::test]
1899 async fn test_empty_uploader_sync() {
1900 let mut uploader = test_uploader(dummy_success_upload_future);
1901 let epoch1 = INITIAL_EPOCH.next_epoch();
1902
1903 let (sync_tx, sync_rx) = oneshot::channel();
1904 uploader.start_epochs_for_test([epoch1]);
1905 uploader.init_instance(TEST_LOCAL_INSTANCE_ID, TEST_TABLE_ID, epoch1);
1906 uploader.local_seal_epoch_for_test(TEST_LOCAL_INSTANCE_ID, epoch1);
1907 uploader.start_single_epoch_sync(epoch1, sync_tx, HashSet::from_iter([TEST_TABLE_ID]));
1908 assert_eq!(epoch1, uploader.test_max_syncing_epoch());
1909
1910 assert_uploader_pending(&mut uploader).await;
1911
1912 match sync_rx.await {
1913 Ok(Ok(data)) => {
1914 assert!(data.uploaded_ssts.is_empty());
1915 }
1916 _ => unreachable!(),
1917 };
1918 assert_eq!(epoch1, uploader.test_max_synced_epoch());
1919 let new_pinned_version = uploader
1920 .context
1921 .pinned_version
1922 .new_pin_version(test_hummock_version(epoch1))
1923 .unwrap();
1924 uploader.update_pinned_version(new_pinned_version);
1925 assert!(uploader.data().syncing_data.is_empty());
1926 assert_eq!(
1927 epoch1,
1928 uploader
1929 .context
1930 .pinned_version
1931 .table_committed_epoch(TEST_TABLE_ID)
1932 .unwrap()
1933 );
1934 }
1935
1936 #[tokio::test]
1937 async fn test_uploader_empty_epoch() {
1938 let mut uploader = test_uploader(dummy_success_upload_future);
1939 let epoch1 = INITIAL_EPOCH.next_epoch();
1940 let epoch2 = epoch1.next_epoch();
1941 uploader.start_epochs_for_test([epoch1, epoch2]);
1942 let imm = gen_imm(epoch2).await;
1943 uploader.init_instance(TEST_LOCAL_INSTANCE_ID, TEST_TABLE_ID, epoch1);
1945 uploader.local_seal_epoch_for_test(TEST_LOCAL_INSTANCE_ID, epoch1);
1946 uploader.add_imm(TEST_LOCAL_INSTANCE_ID, imm);
1947
1948 let (sync_tx, sync_rx) = oneshot::channel();
1949 uploader.start_single_epoch_sync(epoch1, sync_tx, HashSet::from_iter([TEST_TABLE_ID]));
1950 assert_eq!(epoch1, uploader.test_max_syncing_epoch());
1951
1952 assert_uploader_pending(&mut uploader).await;
1953
1954 match sync_rx.await {
1955 Ok(Ok(data)) => {
1956 assert!(data.uploaded_ssts.is_empty());
1957 }
1958 _ => unreachable!(),
1959 };
1960 assert_eq!(epoch1, uploader.test_max_synced_epoch());
1961 let new_pinned_version = uploader
1962 .context
1963 .pinned_version
1964 .new_pin_version(test_hummock_version(epoch1))
1965 .unwrap();
1966 uploader.update_pinned_version(new_pinned_version);
1967 assert!(uploader.data().syncing_data.is_empty());
1968 assert_eq!(
1969 epoch1,
1970 uploader
1971 .context
1972 .pinned_version
1973 .table_committed_epoch(TEST_TABLE_ID)
1974 .unwrap()
1975 );
1976 }
1977
1978 #[tokio::test]
1979 async fn test_uploader_poll_empty() {
1980 let mut uploader = test_uploader(dummy_success_upload_future);
1981 let fut = uploader.next_uploaded_sst();
1982 let mut fut = pin!(fut);
1983 assert!(poll_fn(|cx| Poll::Ready(fut.as_mut().poll(cx).is_pending())).await);
1984 }
1985
1986 #[tokio::test]
1987 async fn test_uploader_empty_advance_mce() {
1988 let mut uploader = test_uploader(dummy_success_upload_future);
1989 let initial_pinned_version = uploader.context.pinned_version.clone();
1990 let epoch1 = INITIAL_EPOCH.next_epoch();
1991 let epoch2 = epoch1.next_epoch();
1992 let epoch3 = epoch2.next_epoch();
1993 let epoch4 = epoch3.next_epoch();
1994 let epoch5 = epoch4.next_epoch();
1995 let epoch6 = epoch5.next_epoch();
1996 let version1 = initial_pinned_version
1997 .new_pin_version(test_hummock_version(epoch1))
1998 .unwrap();
1999 let version2 = initial_pinned_version
2000 .new_pin_version(test_hummock_version(epoch2))
2001 .unwrap();
2002 let version3 = initial_pinned_version
2003 .new_pin_version(test_hummock_version(epoch3))
2004 .unwrap();
2005 let version4 = initial_pinned_version
2006 .new_pin_version(test_hummock_version(epoch4))
2007 .unwrap();
2008 let version5 = initial_pinned_version
2009 .new_pin_version(test_hummock_version(epoch5))
2010 .unwrap();
2011
2012 uploader.start_epochs_for_test([epoch6]);
2013 uploader.init_instance(TEST_LOCAL_INSTANCE_ID, TEST_TABLE_ID, epoch6);
2014
2015 uploader.update_pinned_version(version1);
2016 assert_eq!(epoch1, uploader.test_max_synced_epoch());
2017 assert_eq!(epoch1, uploader.test_max_syncing_epoch());
2018
2019 let imm = gen_imm(epoch6).await;
2020 uploader.add_imm(TEST_LOCAL_INSTANCE_ID, imm.clone());
2021 uploader.update_pinned_version(version2);
2022 assert_eq!(epoch2, uploader.test_max_synced_epoch());
2023 assert_eq!(epoch2, uploader.test_max_syncing_epoch());
2024
2025 uploader.local_seal_epoch_for_test(TEST_LOCAL_INSTANCE_ID, epoch6);
2026 uploader.update_pinned_version(version3);
2027 assert_eq!(epoch3, uploader.test_max_synced_epoch());
2028 assert_eq!(epoch3, uploader.test_max_syncing_epoch());
2029
2030 let (sync_tx, sync_rx) = oneshot::channel();
2031 uploader.start_single_epoch_sync(epoch6, sync_tx, HashSet::from_iter([TEST_TABLE_ID]));
2032 assert_eq!(epoch6, uploader.test_max_syncing_epoch());
2033 uploader.update_pinned_version(version4);
2034 assert_eq!(epoch4, uploader.test_max_synced_epoch());
2035 assert_eq!(epoch6, uploader.test_max_syncing_epoch());
2036
2037 let sst = uploader.next_uploaded_sst().await;
2038 assert_eq!(&get_imm_ids([&imm]), sst.imm_ids());
2039
2040 match sync_rx.await {
2041 Ok(Ok(data)) => {
2042 assert!(data.table_watermarks.is_empty());
2043 assert_eq!(1, data.uploaded_ssts.len());
2044 assert_eq!(&get_imm_ids([&imm]), data.uploaded_ssts[0].imm_ids());
2045 }
2046 _ => unreachable!(),
2047 }
2048
2049 uploader.update_pinned_version(version5);
2050 assert_eq!(epoch6, uploader.test_max_synced_epoch());
2051 assert_eq!(epoch6, uploader.test_max_syncing_epoch());
2052 }
2053
2054 #[tokio::test]
2055 async fn test_uploader_finish_in_order() {
2056 let config = StorageOpts {
2057 shared_buffer_capacity_mb: 1024 * 1024,
2058 shared_buffer_flush_ratio: 0.0,
2059 ..Default::default()
2060 };
2061 let (buffer_tracker, mut uploader, new_task_notifier) =
2062 prepare_uploader_order_test(&config, false);
2063
2064 let epoch1 = INITIAL_EPOCH.next_epoch();
2065 let epoch2 = epoch1.next_epoch();
2066 let epoch3 = epoch2.next_epoch();
2067 let epoch4 = epoch3.next_epoch();
2068 uploader.start_epochs_for_test([epoch1, epoch2, epoch3, epoch4]);
2069 let memory_limiter = buffer_tracker.get_memory_limiter().clone();
2070 let memory_limiter = Some(memory_limiter.deref());
2071
2072 let instance_id1 = 1;
2073 let instance_id2 = 2;
2074
2075 uploader.init_instance(instance_id1, TEST_TABLE_ID, epoch1);
2076 uploader.init_instance(instance_id2, TEST_TABLE_ID, epoch2);
2077
2078 let imm2 = gen_imm_with_limiter(epoch2, memory_limiter).await;
2080 uploader.add_imm(instance_id2, imm2.clone());
2081 let imm1_1 = gen_imm_with_limiter(epoch1, memory_limiter).await;
2082 uploader.add_imm(instance_id1, imm1_1.clone());
2083 let imm1_2 = gen_imm_with_limiter(epoch1, memory_limiter).await;
2084 uploader.add_imm(instance_id1, imm1_2.clone());
2085
2086 let epoch1_spill_payload12 =
2088 HashMap::from_iter([(instance_id1, vec![imm1_2.clone(), imm1_1.clone()])]);
2089 let epoch2_spill_payload = HashMap::from_iter([(instance_id2, vec![imm2.clone()])]);
2090 let (await_start1, finish_tx1) =
2091 new_task_notifier(get_payload_imm_ids(&epoch1_spill_payload12));
2092 let (await_start2, finish_tx2) =
2093 new_task_notifier(get_payload_imm_ids(&epoch2_spill_payload));
2094 uploader.may_flush();
2095 await_start1.await;
2096 await_start2.await;
2097
2098 assert_uploader_pending(&mut uploader).await;
2099
2100 finish_tx2.send(()).unwrap();
2101 assert_uploader_pending(&mut uploader).await;
2102
2103 finish_tx1.send(()).unwrap();
2104 let sst = uploader.next_uploaded_sst().await;
2105 assert_eq!(&get_payload_imm_ids(&epoch1_spill_payload12), sst.imm_ids());
2106 assert_eq!(&vec![epoch1], sst.epochs());
2107
2108 let sst = uploader.next_uploaded_sst().await;
2109 assert_eq!(&get_payload_imm_ids(&epoch2_spill_payload), sst.imm_ids());
2110 assert_eq!(&vec![epoch2], sst.epochs());
2111
2112 let imm1_3 = gen_imm_with_limiter(epoch1, memory_limiter).await;
2113 uploader.add_imm(instance_id1, imm1_3.clone());
2114 let epoch1_spill_payload3 = HashMap::from_iter([(instance_id1, vec![imm1_3.clone()])]);
2115 let (await_start1_3, finish_tx1_3) =
2116 new_task_notifier(get_payload_imm_ids(&epoch1_spill_payload3));
2117 uploader.may_flush();
2118 await_start1_3.await;
2119 let imm1_4 = gen_imm_with_limiter(epoch1, memory_limiter).await;
2120 uploader.add_imm(instance_id1, imm1_4.clone());
2121 let epoch1_sync_payload = HashMap::from_iter([(instance_id1, vec![imm1_4.clone()])]);
2122 let (await_start1_4, finish_tx1_4) =
2123 new_task_notifier(get_payload_imm_ids(&epoch1_sync_payload));
2124 uploader.local_seal_epoch_for_test(instance_id1, epoch1);
2125 let (sync_tx1, mut sync_rx1) = oneshot::channel();
2126 uploader.start_single_epoch_sync(epoch1, sync_tx1, HashSet::from_iter([TEST_TABLE_ID]));
2127 await_start1_4.await;
2128
2129 uploader.local_seal_epoch_for_test(instance_id1, epoch2);
2130 uploader.local_seal_epoch_for_test(instance_id2, epoch2);
2131
2132 let imm3_1 = gen_imm_with_limiter(epoch3, memory_limiter).await;
2138 let epoch3_spill_payload1 = HashMap::from_iter([(instance_id1, vec![imm3_1.clone()])]);
2139 uploader.add_imm(instance_id1, imm3_1.clone());
2140 let (await_start3_1, finish_tx3_1) =
2141 new_task_notifier(get_payload_imm_ids(&epoch3_spill_payload1));
2142 uploader.may_flush();
2143 await_start3_1.await;
2144 let imm3_2 = gen_imm_with_limiter(epoch3, memory_limiter).await;
2145 let epoch3_spill_payload2 = HashMap::from_iter([(instance_id2, vec![imm3_2.clone()])]);
2146 uploader.add_imm(instance_id2, imm3_2.clone());
2147 let (await_start3_2, finish_tx3_2) =
2148 new_task_notifier(get_payload_imm_ids(&epoch3_spill_payload2));
2149 uploader.may_flush();
2150 await_start3_2.await;
2151 let imm3_3 = gen_imm_with_limiter(epoch3, memory_limiter).await;
2152 uploader.add_imm(instance_id1, imm3_3.clone());
2153
2154 uploader.local_seal_epoch_for_test(instance_id1, epoch3);
2160 let imm4 = gen_imm_with_limiter(epoch4, memory_limiter).await;
2161 uploader.add_imm(instance_id1, imm4.clone());
2162 assert_uploader_pending(&mut uploader).await;
2163
2164 finish_tx3_1.send(()).unwrap();
2171 assert_uploader_pending(&mut uploader).await;
2172 finish_tx1_4.send(()).unwrap();
2173 assert_uploader_pending(&mut uploader).await;
2174 finish_tx1_3.send(()).unwrap();
2175
2176 let sst = uploader.next_uploaded_sst().await;
2177 assert_eq!(&get_payload_imm_ids(&epoch1_spill_payload3), sst.imm_ids());
2178
2179 assert!(poll_fn(|cx| Poll::Ready(sync_rx1.poll_unpin(cx).is_pending())).await);
2180
2181 let sst = uploader.next_uploaded_sst().await;
2182 assert_eq!(&get_payload_imm_ids(&epoch1_sync_payload), sst.imm_ids());
2183
2184 match sync_rx1.await {
2185 Ok(Ok(data)) => {
2186 assert_eq!(3, data.uploaded_ssts.len());
2187 assert_eq!(
2188 &get_payload_imm_ids(&epoch1_sync_payload),
2189 data.uploaded_ssts[0].imm_ids()
2190 );
2191 assert_eq!(
2192 &get_payload_imm_ids(&epoch1_spill_payload3),
2193 data.uploaded_ssts[1].imm_ids()
2194 );
2195 assert_eq!(
2196 &get_payload_imm_ids(&epoch1_spill_payload12),
2197 data.uploaded_ssts[2].imm_ids()
2198 );
2199 }
2200 _ => {
2201 unreachable!()
2202 }
2203 }
2204
2205 let (sync_tx2, sync_rx2) = oneshot::channel();
2213 uploader.start_single_epoch_sync(epoch2, sync_tx2, HashSet::from_iter([TEST_TABLE_ID]));
2214 uploader.local_seal_epoch_for_test(instance_id2, epoch3);
2215 let sst = uploader.next_uploaded_sst().await;
2216 assert_eq!(&get_payload_imm_ids(&epoch3_spill_payload1), sst.imm_ids());
2217
2218 match sync_rx2.await {
2219 Ok(Ok(data)) => {
2220 assert_eq!(data.uploaded_ssts.len(), 1);
2221 assert_eq!(
2222 &get_payload_imm_ids(&epoch2_spill_payload),
2223 data.uploaded_ssts[0].imm_ids()
2224 );
2225 }
2226 _ => {
2227 unreachable!("should be sync finish");
2228 }
2229 }
2230 assert_eq!(epoch2, uploader.test_max_synced_epoch());
2231
2232 uploader.local_seal_epoch_for_test(instance_id1, epoch4);
2240 uploader.local_seal_epoch_for_test(instance_id2, epoch4);
2241 let epoch4_sync_payload = HashMap::from_iter([(instance_id1, vec![imm4, imm3_3])]);
2242 let (await_start4_with_3_3, finish_tx4_with_3_3) =
2243 new_task_notifier(get_payload_imm_ids(&epoch4_sync_payload));
2244 let (sync_tx4, mut sync_rx4) = oneshot::channel();
2245 uploader.start_single_epoch_sync(epoch4, sync_tx4, HashSet::from_iter([TEST_TABLE_ID]));
2246 await_start4_with_3_3.await;
2247
2248 assert_uploader_pending(&mut uploader).await;
2256
2257 finish_tx3_2.send(()).unwrap();
2258 let sst = uploader.next_uploaded_sst().await;
2259 assert_eq!(&get_payload_imm_ids(&epoch3_spill_payload2), sst.imm_ids());
2260
2261 finish_tx4_with_3_3.send(()).unwrap();
2262 assert!(poll_fn(|cx| Poll::Ready(sync_rx4.poll_unpin(cx).is_pending())).await);
2263
2264 let sst = uploader.next_uploaded_sst().await;
2265 assert_eq!(&get_payload_imm_ids(&epoch4_sync_payload), sst.imm_ids());
2266
2267 match sync_rx4.await {
2268 Ok(Ok(data)) => {
2269 assert_eq!(3, data.uploaded_ssts.len());
2270 assert_eq!(
2271 &get_payload_imm_ids(&epoch4_sync_payload),
2272 data.uploaded_ssts[0].imm_ids()
2273 );
2274 assert_eq!(
2275 &get_payload_imm_ids(&epoch3_spill_payload2),
2276 data.uploaded_ssts[1].imm_ids()
2277 );
2278 assert_eq!(
2279 &get_payload_imm_ids(&epoch3_spill_payload1),
2280 data.uploaded_ssts[2].imm_ids(),
2281 )
2282 }
2283 _ => {
2284 unreachable!("should be sync finish");
2285 }
2286 }
2287 assert_eq!(epoch4, uploader.test_max_synced_epoch());
2288
2289 }
2297
2298 #[tokio::test]
2299 async fn test_uploader_frequently_flush() {
2300 let config = StorageOpts {
2301 shared_buffer_capacity_mb: 10,
2302 shared_buffer_flush_ratio: 0.8,
2303 shared_buffer_min_batch_flush_size_mb: 1,
2305 ..Default::default()
2306 };
2307 let (buffer_tracker, mut uploader, _new_task_notifier) =
2308 prepare_uploader_order_test(&config, true);
2309
2310 let epoch1 = INITIAL_EPOCH.next_epoch();
2311 let epoch2 = epoch1.next_epoch();
2312 uploader.start_epochs_for_test([epoch1, epoch2]);
2313 let instance_id1 = 1;
2314 let instance_id2 = 2;
2315 let flush_threshold = buffer_tracker.flush_threshold();
2316 let memory_limiter = buffer_tracker.get_memory_limiter().clone();
2317
2318 uploader.init_instance(instance_id1, TEST_TABLE_ID, epoch1);
2319 uploader.init_instance(instance_id2, TEST_TABLE_ID, epoch2);
2320
2321 let mut total_memory = 0;
2323 while total_memory < flush_threshold {
2324 let imm = gen_imm_with_limiter(epoch2, Some(memory_limiter.as_ref())).await;
2325 total_memory += imm.size();
2326 if total_memory > flush_threshold {
2327 break;
2328 }
2329 uploader.add_imm(instance_id2, imm);
2330 }
2331 let imm = gen_imm_with_limiter(epoch1, Some(memory_limiter.as_ref())).await;
2332 uploader.add_imm(instance_id1, imm);
2333 assert!(uploader.may_flush());
2334
2335 for _ in 0..10 {
2336 let imm = gen_imm_with_limiter(epoch1, Some(memory_limiter.as_ref())).await;
2337 uploader.add_imm(instance_id1, imm);
2338 assert!(!uploader.may_flush());
2339 }
2340 }
2341}