Skip to main content

mz_storage_types/
dyncfgs.rs

1// Copyright Materialize, Inc. and contributors. All rights reserved.
2//
3// Use of this software is governed by the Business Source License
4// included in the LICENSE file.
5//
6// As of the Change Date specified in that file, in accordance with
7// the Business Source License, use of this software will be governed
8// by the Apache License, Version 2.0.
9
10//! Dyncfgs used by the storage layer. Despite their name, these can be used
11//! "statically" during rendering, or dynamically within timely operators.
12
13use mz_dyncfg::{Config, ConfigSet};
14use std::time::Duration;
15
16/// When dataflows observe an invariant violation it is either due to a bug or due to the cluster
17/// being shut down. This configuration defines the amount of time to wait before panicking the
18/// process, which will register the invariant violation.
19pub const CLUSTER_SHUTDOWN_GRACE_PERIOD: Config<Duration> = Config::new(
20    "storage_cluster_shutdown_grace_period",
21    Duration::from_secs(10 * 60),
22    "When dataflows observe an invariant violation it is either due to a bug or due to \
23        the cluster being shut down. This configuration defines the amount of time to \
24        wait before panicking the process, which will register the invariant violation.",
25);
26
27// Flow control
28
29/// Whether rendering should use `mz_join_core` rather than DD's `JoinCore::join_core`.
30/// Configuration for basic hydration backpressure.
31pub const DELAY_SOURCES_PAST_REHYDRATION: Config<bool> = Config::new(
32    "storage_dataflow_delay_sources_past_rehydration",
33    // This was original `false`, but it is not enabled everywhere.
34    true,
35    "Whether or not to delay sources producing values in some scenarios \
36        (namely, upsert) till after rehydration is finished",
37);
38
39/// Whether storage dataflows should suspend execution while downstream operators are still
40/// processing data.
41pub const SUSPENDABLE_SOURCES: Config<bool> = Config::new(
42    "storage_dataflow_suspendable_sources",
43    true,
44    "Whether storage dataflows should suspend execution while downstream operators are still \
45        processing data.",
46);
47
48// Controller
49
50/// When enabled, force-downgrade the controller's since handle on the shard
51/// during shard finalization.
52pub const STORAGE_DOWNGRADE_SINCE_DURING_FINALIZATION: Config<bool> = Config::new(
53    "storage_downgrade_since_during_finalization",
54    // This was original `false`, but it is not enabled everywhere.
55    true,
56    "When enabled, force-downgrade the controller's since handle on the shard\
57    during shard finalization",
58);
59
60/// The interval of time to keep when truncating the replica metrics history.
61pub const REPLICA_METRICS_HISTORY_RETENTION_INTERVAL: Config<Duration> = Config::new(
62    "replica_metrics_history_retention_interval",
63    Duration::from_secs(60 * 60 * 24 * 30), // 30 days
64    "The interval of time to keep when truncating the replica metrics history.",
65);
66
67/// The interval of time to keep when truncating the wallclock lag history.
68pub const WALLCLOCK_LAG_HISTORY_RETENTION_INTERVAL: Config<Duration> = Config::new(
69    "wallclock_lag_history_retention_interval",
70    Duration::from_secs(60 * 60 * 24 * 30), // 30 days
71    "The interval of time to keep when truncating the wallclock lag history.",
72);
73
74/// The interval of time to keep when truncating the wallclock lag histogram.
75pub const WALLCLOCK_GLOBAL_LAG_HISTOGRAM_RETENTION_INTERVAL: Config<Duration> = Config::new(
76    "wallclock_global_lag_histogram_retention_interval",
77    Duration::from_secs(60 * 60 * 24 * 30), // 30 days
78    "The interval of time to keep when truncating the wallclock lag histogram.",
79);
80
81// Kafka
82
83/// Rules for enriching the `client.id` property of Kafka clients with
84/// additional data.
85///
86/// The configuration value must be a JSON array of objects containing keys
87/// named `pattern` and `payload`, both of type string. Rules are checked in the
88/// order they are defined. The rule's pattern must be a regular expression
89/// understood by the Rust `regex` crate. If the rule's pattern matches the
90/// address of any broker in the connection, then the payload is appended to the
91/// client ID. A rule's payload is always prefixed with `-`, to separate it from
92/// the preceding data in the client ID.
93pub const KAFKA_CLIENT_ID_ENRICHMENT_RULES: Config<fn() -> serde_json::Value> = Config::new(
94    "kafka_client_id_enrichment_rules",
95    || serde_json::json!([]),
96    "Rules for enriching the `client.id` property of Kafka clients with additional data.",
97);
98
99/// The maximum time we will wait before re-polling rdkafka to see if new partitions/data are
100/// available.
101pub const KAFKA_POLL_MAX_WAIT: Config<Duration> = Config::new(
102    "kafka_poll_max_wait",
103    Duration::from_secs(1),
104    "The maximum time we will wait before re-polling rdkafka to see if new partitions/data are \
105    available.",
106);
107
108pub const KAFKA_DEFAULT_AWS_PRIVATELINK_ENDPOINT_IDENTIFICATION_ALGORITHM: Config<&'static str> =
109    Config::new(
110        "kafka_default_aws_privatelink_endpoint_identification_algorithm",
111        // Default to no hostname verification, which is the default in versions of `librdkafka <1.9.2`.
112        "none",
113        "The value we set for the 'ssl.endpoint.identification.algorithm' option in the Kafka \
114    Connection config. default: 'none'",
115    );
116
117pub const KAFKA_BUFFERED_EVENT_RESIZE_THRESHOLD_ELEMENTS: Config<usize> = Config::new(
118    "kafka_buffered_event_resize_threshold_elements",
119    1000,
120    "In the Kafka sink operator we might need to buffer messages before emitting them. As a \
121        performance optimization we reuse the buffer allocations, but shrink it to retain at \
122        most this number of elements.",
123);
124
125/// Sets retry.backoff.ms in librdkafka for sources and sinks.
126/// See <https://docs.confluent.io/platform/current/clients/librdkafka/html/md_CONFIGURATION.html>
127pub const KAFKA_RETRY_BACKOFF: Config<Duration> = Config::new(
128    "kafka_retry_backoff",
129    Duration::from_millis(100),
130    "Sets retry.backoff.ms in librdkafka for sources and sinks.",
131);
132
133/// Sets retry.backoff.max.ms in librdkafka for sources and sinks.
134/// See <https://docs.confluent.io/platform/current/clients/librdkafka/html/md_CONFIGURATION.html>
135pub const KAFKA_RETRY_BACKOFF_MAX: Config<Duration> = Config::new(
136    "kafka_retry_backoff_max",
137    Duration::from_secs(1),
138    "Sets retry.backoff.max.ms in librdkafka for sources and sinks.",
139);
140
141/// Sets reconnect.backoff.ms in librdkafka for sources and sinks.
142/// See <https://docs.confluent.io/platform/current/clients/librdkafka/html/md_CONFIGURATION.html>
143pub const KAFKA_RECONNECT_BACKOFF: Config<Duration> = Config::new(
144    "kafka_reconnect_backoff",
145    Duration::from_millis(100),
146    "Sets reconnect.backoff.ms in librdkafka for sources and sinks.",
147);
148
149/// Sets reconnect.backoff.max.ms in librdkafka for sources and sinks.
150/// We default to 30s instead of 10s to avoid constant reconnection attempts in the event of
151/// auth changes or unavailability.
152/// See <https://docs.confluent.io/platform/current/clients/librdkafka/html/md_CONFIGURATION.html>
153pub const KAFKA_RECONNECT_BACKOFF_MAX: Config<Duration> = Config::new(
154    "kafka_reconnect_backoff_max",
155    Duration::from_secs(30),
156    "Sets reconnect.backoff.max.ms in librdkafka for sources and sinks.",
157);
158
159// MySQL
160
161/// Replication heartbeat interval requested from the MySQL server.
162pub const MYSQL_REPLICATION_HEARTBEAT_INTERVAL: Config<Duration> = Config::new(
163    "mysql_replication_heartbeat_interval",
164    Duration::from_secs(30),
165    "Replication heartbeat interval requested from the MySQL server.",
166);
167
168// Postgres
169
170/// Interval to poll `confirmed_flush_lsn` to get a resumption lsn.
171pub const PG_FETCH_SLOT_RESUME_LSN_INTERVAL: Config<Duration> = Config::new(
172    "postgres_fetch_slot_resume_lsn_interval",
173    Duration::from_millis(500),
174    "Interval to poll `confirmed_flush_lsn` to get a resumption lsn.",
175);
176
177/// Interval to re-validate the schemas of ingested tables.
178pub const PG_SCHEMA_VALIDATION_INTERVAL: Config<Duration> = Config::new(
179    "pg_schema_validation_interval",
180    Duration::from_secs(15),
181    "Interval to re-validate the schemas of ingested tables.",
182);
183
184/// Controls behavior of PG Source when the upstream DB timeline changes. The default behavior
185/// is to emit a definite error forcing source recreation. In cases of HA, the upstream DB may
186/// provide guarantees of failover without loss of data (e.g. CloudSQL maintenance). Changing this
187/// flag puts the onus on the customer to recreate the source if the upstream DB changes timeline
188/// in a way that introduces data loss (e.g. manual failover, restore, etc.).
189pub static PG_SOURCE_VALIDATE_TIMELINE: Config<bool> = Config::new(
190    "pg_source_validate_timeline",
191    true,
192    "Whether to treat a timeline switch as a definite error",
193);
194
195/// Controls behavior of the SQL Server source when the upstream DB restore history changes. The
196/// default behavior is to emit a definite error, forcing source recreation.  In cases of Always
197/// On Availability Group (AOAG), the upstream DB may guarantee continuity without loss of data.
198/// Changing this flag puts the onus on the customer to recreate the source if the upstream DB
199/// changes in a way that introduces data loss.
200pub static SQL_SERVER_SOURCE_VALIDATE_RESTORE_HISTORY: Config<bool> = Config::new(
201    "sql_server_source_validate_restore_history",
202    true,
203    "Whether to treat a restore history change as a definite error",
204);
205
206// Networking
207
208/// Whether or not to enforce that external connection addresses are global
209/// (not private or local) when resolving them.
210pub const ENFORCE_EXTERNAL_ADDRESSES: Config<bool> = Config::new(
211    "storage_enforce_external_addresses",
212    false,
213    "Whether or not to enforce that external connection addresses are global \
214          (not private or local) when resolving them",
215);
216
217// Upsert
218
219/// Whether or not to prevent buffering the entire _upstream_ snapshot in
220/// memory when processing it in memory. This is generally understood to reduce
221/// memory consumption.
222///
223/// When false, in general the memory utilization while processing the snapshot is:
224/// # of snapshot updates + (# of unique keys in snapshot * N), where N is some small
225/// integer number of buffers
226///
227/// When true, in general the memory utilization while processing the snapshot is:
228/// # of snapshot updates + (RocksDB buffers + # of keys in batch produced by upstream) * # of
229/// workers.
230///
231/// Without hydration flow control, which is not yet implemented, there are workloads that may
232/// cause the latter to use more memory, which is why we offer this configuration.
233pub const STORAGE_UPSERT_PREVENT_SNAPSHOT_BUFFERING: Config<bool> = Config::new(
234    "storage_upsert_prevent_snapshot_buffering",
235    true,
236    "Prevent snapshot buffering in upsert.",
237);
238
239/// Whether to enable the merge operator in upsert for the RocksDB backend.
240pub const STORAGE_ROCKSDB_USE_MERGE_OPERATOR: Config<bool> = Config::new(
241    "storage_rocksdb_use_merge_operator",
242    true,
243    "Use the native rocksdb merge operator where possible.",
244);
245
246/// If `storage_upsert_prevent_snapshot_buffering` is true, this prevents the upsert
247/// operator from buffering too many events from the upstream snapshot. In the absence
248/// of hydration flow control, this could prevent certain workloads from causing egregiously
249/// large writes to RocksDB.
250pub const STORAGE_UPSERT_MAX_SNAPSHOT_BATCH_BUFFERING: Config<Option<usize>> = Config::new(
251    "storage_upsert_max_snapshot_batch_buffering",
252    None,
253    "Limit snapshot buffering in upsert.",
254);
255
256// RocksDB
257
258/// How many times to try to cleanup old RocksDB DB's on disk before giving up.
259pub const STORAGE_ROCKSDB_CLEANUP_TRIES: Config<usize> = Config::new(
260    "storage_rocksdb_cleanup_tries",
261    5,
262    "How many times to try to cleanup old RocksDB DB's on disk before giving up.",
263);
264
265/// Delay interval when reconnecting to a source / sink after halt.
266pub const STORAGE_SUSPEND_AND_RESTART_DELAY: Config<Duration> = Config::new(
267    "storage_suspend_and_restart_delay",
268    Duration::from_secs(5),
269    "Delay interval when reconnecting to a source / sink after halt.",
270);
271
272/// Whether to use the new continual feedback upsert operator.
273pub const STORAGE_USE_CONTINUAL_FEEDBACK_UPSERT: Config<bool> = Config::new(
274    "storage_use_continual_feedback_upsert",
275    true,
276    "Whether to use the new continual feedback upsert operator.",
277);
278
279/// The interval at which the storage server performs maintenance tasks.
280pub const STORAGE_SERVER_MAINTENANCE_INTERVAL: Config<Duration> = Config::new(
281    "storage_server_maintenance_interval",
282    Duration::from_millis(10),
283    "The interval at which the storage server performs maintenance tasks. Zero enables maintenance on every iteration.",
284);
285
286/// If set, iteratively search the progress topic for a progress record with increasing lookback.
287pub const SINK_PROGRESS_SEARCH: Config<bool> = Config::new(
288    "storage_sink_progress_search",
289    true,
290    "If set, iteratively search the progress topic for a progress record with increasing lookback.",
291);
292
293/// Configure how to behave when trying to create an existing topic with specified configs.
294pub const SINK_ENSURE_TOPIC_CONFIG: Config<&'static str> = Config::new(
295    "storage_sink_ensure_topic_config",
296    "skip",
297    "If `skip`, don't check the config of existing topics; if `check`, fetch the config and \
298    warn if it does not match the expected configs; if `alter`, attempt to change the upstream to \
299    match the expected configs.",
300);
301
302/// Configure mz-ore overflowing type behavior.
303pub const ORE_OVERFLOWING_BEHAVIOR: Config<&'static str> = Config::new(
304    "ore_overflowing_behavior",
305    "soft_panic",
306    "Overflow behavior for Overflowing types. One of 'ignore', 'panic', 'soft_panic'.",
307);
308
309/// The time after which we delete per-replica statistics (for sources and
310/// sinks) after there have been no updates.
311///
312/// This time is opportunistic, statistics are not guaranteed to be deleted
313/// after the retention time runs out.
314pub const STATISTICS_RETENTION_DURATION: Config<Duration> = Config::new(
315    "storage_statistics_retention_duration",
316    Duration::from_secs(86_400), /* one day */
317    "The time after which we delete per replica statistics (for sources and sinks) after there have been no updates.",
318);
319
320/// Adds the full set of all storage `Config`s.
321pub fn all_dyncfgs(configs: ConfigSet) -> ConfigSet {
322    configs
323        .add(&CLUSTER_SHUTDOWN_GRACE_PERIOD)
324        .add(&DELAY_SOURCES_PAST_REHYDRATION)
325        .add(&ENFORCE_EXTERNAL_ADDRESSES)
326        .add(&KAFKA_BUFFERED_EVENT_RESIZE_THRESHOLD_ELEMENTS)
327        .add(&KAFKA_CLIENT_ID_ENRICHMENT_RULES)
328        .add(&KAFKA_DEFAULT_AWS_PRIVATELINK_ENDPOINT_IDENTIFICATION_ALGORITHM)
329        .add(&KAFKA_POLL_MAX_WAIT)
330        .add(&KAFKA_RETRY_BACKOFF)
331        .add(&KAFKA_RETRY_BACKOFF_MAX)
332        .add(&KAFKA_RECONNECT_BACKOFF)
333        .add(&KAFKA_RECONNECT_BACKOFF_MAX)
334        .add(&MYSQL_REPLICATION_HEARTBEAT_INTERVAL)
335        .add(&ORE_OVERFLOWING_BEHAVIOR)
336        .add(&PG_FETCH_SLOT_RESUME_LSN_INTERVAL)
337        .add(&PG_SCHEMA_VALIDATION_INTERVAL)
338        .add(&PG_SOURCE_VALIDATE_TIMELINE)
339        .add(&REPLICA_METRICS_HISTORY_RETENTION_INTERVAL)
340        .add(&SINK_ENSURE_TOPIC_CONFIG)
341        .add(&SINK_PROGRESS_SEARCH)
342        .add(&SQL_SERVER_SOURCE_VALIDATE_RESTORE_HISTORY)
343        .add(&STORAGE_DOWNGRADE_SINCE_DURING_FINALIZATION)
344        .add(&STORAGE_ROCKSDB_CLEANUP_TRIES)
345        .add(&STORAGE_ROCKSDB_USE_MERGE_OPERATOR)
346        .add(&STORAGE_SERVER_MAINTENANCE_INTERVAL)
347        .add(&STORAGE_SUSPEND_AND_RESTART_DELAY)
348        .add(&STORAGE_UPSERT_MAX_SNAPSHOT_BATCH_BUFFERING)
349        .add(&STORAGE_UPSERT_PREVENT_SNAPSHOT_BUFFERING)
350        .add(&STORAGE_USE_CONTINUAL_FEEDBACK_UPSERT)
351        .add(&SUSPENDABLE_SOURCES)
352        .add(&WALLCLOCK_GLOBAL_LAG_HISTOGRAM_RETENTION_INTERVAL)
353        .add(&WALLCLOCK_LAG_HISTORY_RETENTION_INTERVAL)
354        .add(&crate::sources::sql_server::CDC_CLEANUP_CHANGE_TABLE)
355        .add(&crate::sources::sql_server::CDC_CLEANUP_CHANGE_TABLE_MAX_DELETES)
356        .add(&crate::sources::sql_server::MAX_LSN_WAIT)
357        .add(&crate::sources::sql_server::SNAPSHOT_PROGRESS_REPORT_INTERVAL)
358        .add(&STATISTICS_RETENTION_DURATION)
359}