pub struct MetaOpts {Show 84 fields
pub enable_recovery: bool,
pub disable_automatic_parallelism_control: bool,
pub parallelism_control_batch_size: usize,
pub parallelism_control_trigger_period_sec: u64,
pub parallelism_control_trigger_first_delay_sec: u64,
pub in_flight_barrier_nums: usize,
pub max_idle_ms: u64,
pub compaction_deterministic_test: bool,
pub default_parallelism: DefaultParallelism,
pub vacuum_interval_sec: u64,
pub vacuum_spin_interval_ms: u64,
pub iceberg_gc_interval_sec: u64,
pub time_travel_vacuum_interval_sec: u64,
pub hummock_version_checkpoint_interval_sec: u64,
pub enable_hummock_data_archive: bool,
pub hummock_time_travel_snapshot_interval: u64,
pub hummock_time_travel_sst_info_fetch_batch_size: usize,
pub hummock_time_travel_sst_info_insert_batch_size: usize,
pub hummock_time_travel_epoch_version_insert_batch_size: usize,
pub hummock_gc_history_insert_batch_size: usize,
pub hummock_time_travel_filter_out_objects_batch_size: usize,
pub hummock_time_travel_filter_out_objects_v1: bool,
pub hummock_time_travel_filter_out_objects_list_version_batch_size: usize,
pub hummock_time_travel_filter_out_objects_list_delta_batch_size: usize,
pub min_delta_log_num_for_hummock_version_checkpoint: u64,
pub min_sst_retention_time_sec: u64,
pub full_gc_interval_sec: u64,
pub full_gc_object_limit: u64,
pub gc_history_retention_time_sec: u64,
pub max_inflight_time_travel_query: u64,
pub enable_committed_sst_sanity_check: bool,
pub periodic_compaction_interval_sec: u64,
pub node_num_monitor_interval_sec: u64,
pub protect_drop_table_with_incoming_sink: bool,
pub prometheus_endpoint: Option<String>,
pub prometheus_selector: Option<String>,
pub vpc_id: Option<String>,
pub security_group_id: Option<String>,
pub privatelink_endpoint_default_tags: Option<Vec<(String, String)>>,
pub periodic_space_reclaim_compaction_interval_sec: u64,
pub telemetry_enabled: bool,
pub periodic_ttl_reclaim_compaction_interval_sec: u64,
pub periodic_tombstone_reclaim_compaction_interval_sec: u64,
pub periodic_scheduling_compaction_group_split_interval_sec: u64,
pub do_not_config_object_storage_lifecycle: bool,
pub partition_vnode_count: u32,
pub table_high_write_throughput_threshold: u64,
pub table_low_write_throughput_threshold: u64,
pub compaction_task_max_heartbeat_interval_secs: u64,
pub compaction_task_max_progress_interval_secs: u64,
pub compaction_config: Option<CompactionConfig>,
pub hybrid_partition_node_count: u32,
pub event_log_enabled: bool,
pub event_log_channel_max_size: u32,
pub advertise_addr: String,
pub cached_traces_num: u32,
pub cached_traces_memory_limit_bytes: usize,
pub enable_trivial_move: bool,
pub enable_check_task_level_overlap: bool,
pub enable_dropped_column_reclaim: bool,
pub split_group_size_ratio: f64,
pub table_stat_high_write_throughput_ratio_for_split: f64,
pub table_stat_low_write_throughput_ratio_for_merge: f64,
pub table_stat_throuput_window_seconds_for_split: usize,
pub table_stat_throuput_window_seconds_for_merge: usize,
pub object_store_config: ObjectStoreConfig,
pub max_trivial_move_task_count_per_loop: usize,
pub max_get_task_probe_times: usize,
pub compact_task_table_size_partition_threshold_low: u64,
pub compact_task_table_size_partition_threshold_high: u64,
pub periodic_scheduling_compaction_group_merge_interval_sec: u64,
pub compaction_group_merge_dimension_threshold: f64,
pub secret_store_private_key: Option<Vec<u8>>,
pub temp_secret_file_dir: String,
pub actor_cnt_per_worker_parallelism_hard_limit: usize,
pub actor_cnt_per_worker_parallelism_soft_limit: usize,
pub license_key_path: Option<PathBuf>,
pub compute_client_config: RpcClientConfig,
pub stream_client_config: RpcClientConfig,
pub frontend_client_config: RpcClientConfig,
pub redact_sql_option_keywords: RedactSqlOptionKeywordsRef,
pub cdc_table_split_init_sleep_interval_splits: u64,
pub cdc_table_split_init_sleep_duration_millis: u64,
pub cdc_table_split_init_insert_batch_size: u64,
}Expand description
Options shared by all meta service instances
Fields§
§enable_recovery: boolWhether to enable the recovery of the cluster. If disabled, the meta service will exit on abnormal cases.
disable_automatic_parallelism_control: boolWhether to disable the auto-scaling feature.
parallelism_control_batch_size: usizeThe number of streaming jobs per scaling operation.
parallelism_control_trigger_period_sec: u64The period of parallelism control trigger.
parallelism_control_trigger_first_delay_sec: u64The first delay of parallelism control.
in_flight_barrier_nums: usizeThe maximum number of barriers in-flight in the compute nodes.
max_idle_ms: u64After specified seconds of idle (no mview or flush), the process will be exited. 0 for infinite, process will never be exited due to long idle time.
compaction_deterministic_test: boolWhether run in compaction detection test mode
default_parallelism: DefaultParallelismDefault parallelism of units for all streaming jobs.
vacuum_interval_sec: u64Interval of invoking a vacuum job, to remove stale metadata from meta store and objects from object store.
vacuum_spin_interval_ms: u64The spin interval inside a vacuum job. It avoids the vacuum job monopolizing resources of meta node.
iceberg_gc_interval_sec: u64Interval of invoking iceberg garbage collection, to expire old snapshots.
time_travel_vacuum_interval_sec: u64§hummock_version_checkpoint_interval_sec: u64Interval of hummock version checkpoint.
enable_hummock_data_archive: bool§hummock_time_travel_snapshot_interval: u64§hummock_time_travel_sst_info_fetch_batch_size: usize§hummock_time_travel_sst_info_insert_batch_size: usize§hummock_time_travel_epoch_version_insert_batch_size: usize§hummock_gc_history_insert_batch_size: usize§hummock_time_travel_filter_out_objects_batch_size: usize§hummock_time_travel_filter_out_objects_v1: bool§hummock_time_travel_filter_out_objects_list_version_batch_size: usize§hummock_time_travel_filter_out_objects_list_delta_batch_size: usize§min_delta_log_num_for_hummock_version_checkpoint: u64The minimum delta log number a new checkpoint should compact, otherwise the checkpoint
attempt is rejected. Greater value reduces object store IO, meanwhile it results in
more loss of in memory HummockVersionCheckpoint::stale_objects state when meta node is
restarted.
min_sst_retention_time_sec: u64Objects within min_sst_retention_time_sec won’t be deleted by hummock full GC, even they
are dangling.
full_gc_interval_sec: u64Interval of automatic hummock full GC.
full_gc_object_limit: u64Max number of object per full GC job can fetch.
gc_history_retention_time_sec: u64Duration in seconds to retain garbage collection history data.
max_inflight_time_travel_query: u64Max number of inflight time travel query.
enable_committed_sst_sanity_check: boolEnable sanity check when SSTs are committed
periodic_compaction_interval_sec: u64Schedule compaction for all compaction groups with this interval.
node_num_monitor_interval_sec: u64Interval of reporting the number of nodes in the cluster.
protect_drop_table_with_incoming_sink: boolWhether to protect the drop table operation with incoming sink.
prometheus_endpoint: Option<String>The Prometheus endpoint for Meta Dashboard Service. The Dashboard service uses this in the following ways:
- Query Prometheus for relevant metrics to find Stream Graph Bottleneck, and display it.
- Provide cluster diagnostics, at
/api/monitor/diagnoseto troubleshoot cluster. These are just examples which show how the Meta Dashboard Service queries Prometheus.
prometheus_selector: Option<String>The additional selector used when querying Prometheus.
vpc_id: Option<String>The VPC id of the cluster.
security_group_id: Option<String>A usable security group id to assign to a vpc endpoint
Default tag for the endpoint created when creating a privatelink connection.
Will be appended to the tags specified in the tags field in with clause in create connection.
periodic_space_reclaim_compaction_interval_sec: u64Schedule space_reclaim_compaction for all compaction groups with this interval.
telemetry_enabled: booltelemetry enabled in config file or not
periodic_ttl_reclaim_compaction_interval_sec: u64Schedule ttl_reclaim_compaction for all compaction groups with this interval.
periodic_tombstone_reclaim_compaction_interval_sec: u64Schedule tombstone_reclaim_compaction for all compaction groups with this interval.
periodic_scheduling_compaction_group_split_interval_sec: u64Schedule periodic_scheduling_compaction_group_split_interval_sec for all compaction groups with this interval.
do_not_config_object_storage_lifecycle: boolWhether config object storage bucket lifecycle to purge stale data.
partition_vnode_count: u32§table_high_write_throughput_threshold: u64threshold of high write throughput of state-table, unit: B/sec
table_low_write_throughput_threshold: u64threshold of low write throughput of state-table, unit: B/sec
compaction_task_max_heartbeat_interval_secs: u64§compaction_task_max_progress_interval_secs: u64§compaction_config: Option<CompactionConfig>§hybrid_partition_node_count: u32hybrid compaction group config
hybrid_partition_vnode_count determines the granularity of vnodes in the hybrid compaction group for SST alignment.
When hybrid_partition_vnode_count > 0, in hybrid compaction group
- Tables with high write throughput will be split at vnode granularity
- Tables with high size tables will be split by table granularity
When
hybrid_partition_vnode_count= 0,no longer be special alignment operations for the hybrid compaction group
event_log_enabled: bool§event_log_channel_max_size: u32§advertise_addr: String§cached_traces_num: u32The number of traces to be cached in-memory by the tracing collector embedded in the meta node.
cached_traces_memory_limit_bytes: usizeThe maximum memory usage in bytes for the tracing collector embedded in the meta node.
enable_trivial_move: booll0 picker whether to select trivial move task
enable_check_task_level_overlap: booll0 multi level picker whether to check the overlap accuracy between sub levels
enable_dropped_column_reclaim: bool§split_group_size_ratio: f64Whether to split the compaction group when the size of the group exceeds the threshold.
table_stat_high_write_throughput_ratio_for_split: f64To split the compaction group when the high throughput statistics of the group exceeds the threshold.
table_stat_low_write_throughput_ratio_for_merge: f64To merge the compaction group when the low throughput statistics of the group exceeds the threshold.
table_stat_throuput_window_seconds_for_split: usizeThe window seconds of table throughput statistic history for split compaction group.
table_stat_throuput_window_seconds_for_merge: usizeThe window seconds of table throughput statistic history for merge compaction group.
object_store_config: ObjectStoreConfigThe configuration of the object store
max_trivial_move_task_count_per_loop: usizeThe maximum number of trivial move tasks to be picked in a single loop
max_get_task_probe_times: usizeThe maximum number of times to probe for PullTaskEvent
compact_task_table_size_partition_threshold_low: u64§compact_task_table_size_partition_threshold_high: u64§periodic_scheduling_compaction_group_merge_interval_sec: u64§compaction_group_merge_dimension_threshold: f64§secret_store_private_key: Option<Vec<u8>>§temp_secret_file_dir: StringThe path of the temp secret file directory.
actor_cnt_per_worker_parallelism_hard_limit: usize§actor_cnt_per_worker_parallelism_soft_limit: usize§license_key_path: Option<PathBuf>§compute_client_config: RpcClientConfig§stream_client_config: RpcClientConfig§frontend_client_config: RpcClientConfig§redact_sql_option_keywords: RedactSqlOptionKeywordsRef§cdc_table_split_init_sleep_interval_splits: u64§cdc_table_split_init_sleep_duration_millis: u64§cdc_table_split_init_insert_batch_size: u64Implementations§
Trait Implementations§
Auto Trait Implementations§
impl Freeze for MetaOpts
impl RefUnwindSafe for MetaOpts
impl Send for MetaOpts
impl Sync for MetaOpts
impl Unpin for MetaOpts
impl UnwindSafe for MetaOpts
Blanket Implementations§
§impl<T> AsAny for T
impl<T> AsAny for T
§fn any_ref(&self) -> &(dyn Any + Sync + Send + 'static)
fn any_ref(&self) -> &(dyn Any + Sync + Send + 'static)
dyn Any reference to the object: Read more§fn as_any(self: Arc<T>) -> Arc<dyn Any + Sync + Send>
fn as_any(self: Arc<T>) -> Arc<dyn Any + Sync + Send>
Arc<dyn Any> reference to the object: Read more§fn into_any(self: Box<T>) -> Box<dyn Any + Sync + Send>
fn into_any(self: Box<T>) -> Box<dyn Any + Sync + Send>
Box<dyn Any>: Read more§fn type_name(&self) -> &'static str
fn type_name(&self) -> &'static str
std::any::type_name, since Any does not provide it and
Any::type_id is useless as a debugging aid (its Debug is just a mess of hex digits).Source§impl<T> BorrowMut<T> for Twhere
T: ?Sized,
impl<T> BorrowMut<T> for Twhere
T: ?Sized,
Source§fn borrow_mut(&mut self) -> &mut T
fn borrow_mut(&mut self) -> &mut T
Source§impl<T> CloneToUninit for Twhere
T: Clone,
impl<T> CloneToUninit for Twhere
T: Clone,
§impl<T> Conv for T
impl<T> Conv for T
§impl<T> FmtForward for T
impl<T> FmtForward for T
§fn fmt_binary(self) -> FmtBinary<Self>where
Self: Binary,
fn fmt_binary(self) -> FmtBinary<Self>where
Self: Binary,
self to use its Binary implementation when Debug-formatted.§fn fmt_display(self) -> FmtDisplay<Self>where
Self: Display,
fn fmt_display(self) -> FmtDisplay<Self>where
Self: Display,
self to use its Display implementation when
Debug-formatted.§fn fmt_lower_exp(self) -> FmtLowerExp<Self>where
Self: LowerExp,
fn fmt_lower_exp(self) -> FmtLowerExp<Self>where
Self: LowerExp,
self to use its LowerExp implementation when
Debug-formatted.§fn fmt_lower_hex(self) -> FmtLowerHex<Self>where
Self: LowerHex,
fn fmt_lower_hex(self) -> FmtLowerHex<Self>where
Self: LowerHex,
self to use its LowerHex implementation when
Debug-formatted.§fn fmt_octal(self) -> FmtOctal<Self>where
Self: Octal,
fn fmt_octal(self) -> FmtOctal<Self>where
Self: Octal,
self to use its Octal implementation when Debug-formatted.§fn fmt_pointer(self) -> FmtPointer<Self>where
Self: Pointer,
fn fmt_pointer(self) -> FmtPointer<Self>where
Self: Pointer,
self to use its Pointer implementation when
Debug-formatted.§fn fmt_upper_exp(self) -> FmtUpperExp<Self>where
Self: UpperExp,
fn fmt_upper_exp(self) -> FmtUpperExp<Self>where
Self: UpperExp,
self to use its UpperExp implementation when
Debug-formatted.§fn fmt_upper_hex(self) -> FmtUpperHex<Self>where
Self: UpperHex,
fn fmt_upper_hex(self) -> FmtUpperHex<Self>where
Self: UpperHex,
self to use its UpperHex implementation when
Debug-formatted.§fn fmt_list(self) -> FmtList<Self>where
&'a Self: for<'a> IntoIterator,
fn fmt_list(self) -> FmtList<Self>where
&'a Self: for<'a> IntoIterator,
§impl<T> FutureExt for T
impl<T> FutureExt for T
§fn with_context(self, otel_cx: Context) -> WithContext<Self>
fn with_context(self, otel_cx: Context) -> WithContext<Self>
§fn with_current_context(self) -> WithContext<Self>
fn with_current_context(self) -> WithContext<Self>
§impl<T> Instrument for T
impl<T> Instrument for T
§fn instrument(self, span: Span) -> Instrumented<Self>
fn instrument(self, span: Span) -> Instrumented<Self>
§fn in_current_span(self) -> Instrumented<Self>
fn in_current_span(self) -> Instrumented<Self>
Source§impl<T> Instrument for T
impl<T> Instrument for T
Source§fn instrument(self, span: Span) -> Instrumented<Self>
fn instrument(self, span: Span) -> Instrumented<Self>
Source§fn in_current_span(self) -> Instrumented<Self>
fn in_current_span(self) -> Instrumented<Self>
Source§impl<T> IntoEither for T
impl<T> IntoEither for T
Source§fn into_either(self, into_left: bool) -> Either<Self, Self>
fn into_either(self, into_left: bool) -> Either<Self, Self>
self into a Left variant of Either<Self, Self>
if into_left is true.
Converts self into a Right variant of Either<Self, Self>
otherwise. Read moreSource§fn into_either_with<F>(self, into_left: F) -> Either<Self, Self>
fn into_either_with<F>(self, into_left: F) -> Either<Self, Self>
self into a Left variant of Either<Self, Self>
if into_left(&self) returns true.
Converts self into a Right variant of Either<Self, Self>
otherwise. Read moreSource§impl<T> IntoRequest<T> for T
impl<T> IntoRequest<T> for T
Source§fn into_request(self) -> Request<T>
fn into_request(self) -> Request<T>
T in a tonic::Request§impl<T> IntoResult<T> for T
impl<T> IntoResult<T> for T
type Err = Infallible
fn into_result(self) -> Result<T, <T as IntoResult<T>>::Err>
§impl<M> MetricVecRelabelExt for M
impl<M> MetricVecRelabelExt for M
§fn relabel(
self,
metric_level: MetricLevel,
relabel_threshold: MetricLevel,
) -> RelabeledMetricVec<M>
fn relabel( self, metric_level: MetricLevel, relabel_threshold: MetricLevel, ) -> RelabeledMetricVec<M>
RelabeledMetricVec::with_metric_level].§fn relabel_n(
self,
metric_level: MetricLevel,
relabel_threshold: MetricLevel,
relabel_num: usize,
) -> RelabeledMetricVec<M>
fn relabel_n( self, metric_level: MetricLevel, relabel_threshold: MetricLevel, relabel_num: usize, ) -> RelabeledMetricVec<M>
RelabeledMetricVec::with_metric_level_relabel_n].§fn relabel_debug_1(
self,
relabel_threshold: MetricLevel,
) -> RelabeledMetricVec<M>
fn relabel_debug_1( self, relabel_threshold: MetricLevel, ) -> RelabeledMetricVec<M>
RelabeledMetricVec::with_metric_level_relabel_n] with metric_level set to
MetricLevel::Debug and relabel_num set to 1.§impl<T> Pipe for Twhere
T: ?Sized,
impl<T> Pipe for Twhere
T: ?Sized,
§fn pipe<R>(self, func: impl FnOnce(Self) -> R) -> Rwhere
Self: Sized,
fn pipe<R>(self, func: impl FnOnce(Self) -> R) -> Rwhere
Self: Sized,
§fn pipe_ref<'a, R>(&'a self, func: impl FnOnce(&'a Self) -> R) -> Rwhere
R: 'a,
fn pipe_ref<'a, R>(&'a self, func: impl FnOnce(&'a Self) -> R) -> Rwhere
R: 'a,
self and passes that borrow into the pipe function. Read more§fn pipe_ref_mut<'a, R>(&'a mut self, func: impl FnOnce(&'a mut Self) -> R) -> Rwhere
R: 'a,
fn pipe_ref_mut<'a, R>(&'a mut self, func: impl FnOnce(&'a mut Self) -> R) -> Rwhere
R: 'a,
self and passes that borrow into the pipe function. Read more§fn pipe_borrow<'a, B, R>(&'a self, func: impl FnOnce(&'a B) -> R) -> R
fn pipe_borrow<'a, B, R>(&'a self, func: impl FnOnce(&'a B) -> R) -> R
§fn pipe_borrow_mut<'a, B, R>(
&'a mut self,
func: impl FnOnce(&'a mut B) -> R,
) -> R
fn pipe_borrow_mut<'a, B, R>( &'a mut self, func: impl FnOnce(&'a mut B) -> R, ) -> R
§fn pipe_as_ref<'a, U, R>(&'a self, func: impl FnOnce(&'a U) -> R) -> R
fn pipe_as_ref<'a, U, R>(&'a self, func: impl FnOnce(&'a U) -> R) -> R
self, then passes self.as_ref() into the pipe function.§fn pipe_as_mut<'a, U, R>(&'a mut self, func: impl FnOnce(&'a mut U) -> R) -> R
fn pipe_as_mut<'a, U, R>(&'a mut self, func: impl FnOnce(&'a mut U) -> R) -> R
self, then passes self.as_mut() into the pipe
function.§fn pipe_deref<'a, T, R>(&'a self, func: impl FnOnce(&'a T) -> R) -> R
fn pipe_deref<'a, T, R>(&'a self, func: impl FnOnce(&'a T) -> R) -> R
self, then passes self.deref() into the pipe function.§impl<T> Pointable for T
impl<T> Pointable for T
§impl<T> Scope for T
impl<T> Scope for T
§impl<T> Tap for T
impl<T> Tap for T
§fn tap_borrow<B>(self, func: impl FnOnce(&B)) -> Self
fn tap_borrow<B>(self, func: impl FnOnce(&B)) -> Self
Borrow<B> of a value. Read more§fn tap_borrow_mut<B>(self, func: impl FnOnce(&mut B)) -> Self
fn tap_borrow_mut<B>(self, func: impl FnOnce(&mut B)) -> Self
BorrowMut<B> of a value. Read more§fn tap_ref<R>(self, func: impl FnOnce(&R)) -> Self
fn tap_ref<R>(self, func: impl FnOnce(&R)) -> Self
AsRef<R> view of a value. Read more§fn tap_ref_mut<R>(self, func: impl FnOnce(&mut R)) -> Self
fn tap_ref_mut<R>(self, func: impl FnOnce(&mut R)) -> Self
AsMut<R> view of a value. Read more§fn tap_deref<T>(self, func: impl FnOnce(&T)) -> Self
fn tap_deref<T>(self, func: impl FnOnce(&T)) -> Self
Deref::Target of a value. Read more§fn tap_deref_mut<T>(self, func: impl FnOnce(&mut T)) -> Self
fn tap_deref_mut<T>(self, func: impl FnOnce(&mut T)) -> Self
Deref::Target of a value. Read more§fn tap_dbg(self, func: impl FnOnce(&Self)) -> Self
fn tap_dbg(self, func: impl FnOnce(&Self)) -> Self
.tap() only in debug builds, and is erased in release builds.§fn tap_mut_dbg(self, func: impl FnOnce(&mut Self)) -> Self
fn tap_mut_dbg(self, func: impl FnOnce(&mut Self)) -> Self
.tap_mut() only in debug builds, and is erased in release
builds.§fn tap_borrow_dbg<B>(self, func: impl FnOnce(&B)) -> Self
fn tap_borrow_dbg<B>(self, func: impl FnOnce(&B)) -> Self
.tap_borrow() only in debug builds, and is erased in release
builds.§fn tap_borrow_mut_dbg<B>(self, func: impl FnOnce(&mut B)) -> Self
fn tap_borrow_mut_dbg<B>(self, func: impl FnOnce(&mut B)) -> Self
.tap_borrow_mut() only in debug builds, and is erased in release
builds.§fn tap_ref_dbg<R>(self, func: impl FnOnce(&R)) -> Self
fn tap_ref_dbg<R>(self, func: impl FnOnce(&R)) -> Self
.tap_ref() only in debug builds, and is erased in release
builds.§fn tap_ref_mut_dbg<R>(self, func: impl FnOnce(&mut R)) -> Self
fn tap_ref_mut_dbg<R>(self, func: impl FnOnce(&mut R)) -> Self
.tap_ref_mut() only in debug builds, and is erased in release
builds.§fn tap_deref_dbg<T>(self, func: impl FnOnce(&T)) -> Self
fn tap_deref_dbg<T>(self, func: impl FnOnce(&T)) -> Self
.tap_deref() only in debug builds, and is erased in release
builds.