MergeTree tables settings
System table system.merge_tree_settings
shows the globally set MergeTree settings.
MergeTree settings can be set in the merge_tree
section of the server config file, or specified for each MergeTree
table individually in
the SETTINGS
clause of the CREATE TABLE
statement.
Example for customizing setting max_suspicious_broken_parts
:
Configure the default for all MergeTree
tables in the server configuration file:
Set for a particular table:
Change the settings for a particular table using ALTER TABLE ... MODIFY SETTING
:
MergeTree settings
add_minmax_index_for_numeric_columns
Type | Default value |
---|---|
Type Bool | Default value 0 |
When enabled, min-max (skipping) indices are added for all numeric columns of the table.
add_minmax_index_for_string_columns
Type | Default value |
---|---|
Type Bool | Default value 0 |
When enabled, min-max (skipping) indices are added for all string columns of the table.
allow_experimental_reverse_key
Type | Default value |
---|---|
Type Bool | Default value 0 |
Enables support for descending sort order in MergeTree sorting keys. This setting is particularly useful for time series analysis and Top-N queries, allowing data to be stored in reverse chronological order to optimize query performance.
With allow_experimental_reverse_key
enabled, you can define descending sort
orders within the ORDER BY
clause of a MergeTree table. This enables the
use of more efficient ReadInOrder
optimizations instead of ReadInReverseOrder
for descending queries.
Example
By using ORDER BY time DESC
in the query, ReadInOrder
is applied.
Default Value: false
allow_reduce_blocking_parts_task
Type | Default value |
---|---|
Type Bool | Default value 1 |
Background task which reduces blocking parts for shared merge tree tables. Only in ClickHouse Cloud
allow_summing_columns_in_partition_or_order_key
Type | Default value |
---|---|
Type Bool | Default value 0 |
When enabled, allows summing columns in a SummingMergeTree table to be used in the partition or sorting key.
columns_and_secondary_indices_sizes_lazy_calculation
Type | Default value |
---|---|
Type Bool | Default value 1 |
Calculate columns and secondary indices sizes lazily on first request instead of on table initialization.
deduplicate_merge_projection_mode
Type | Default value |
---|---|
Type DeduplicateMergeProjectionMode | Default value throw |
Whether to allow create projection for the table with non-classic MergeTree,
that is not (Replicated, Shared) MergeTree. Ignore option is purely for
compatibility which might result in incorrect answer. Otherwise, if allowed,
what is the action when merge projections, either drop or rebuild. So classic
MergeTree would ignore this setting. It also controls OPTIMIZE DEDUPLICATE
as well, but has effect on all MergeTree family members. Similar to the
option lightweight_mutation_projection_mode
, it is also part level.
Possible values:
ignore
throw
drop
rebuild
default_compression_codec
Specifies the default compression codec to be used if none is defined for a particular column in the table declaration. Compression codec selecting order for a column:
- Compression codec defined for the column in the table declaration
- Compression codec defined in
default_compression_codec
(this setting) - Default compression codec defined in
compression
settings Default value: an empty string (not defined).
enable_max_bytes_limit_for_min_age_to_force_merge
Type | Default value |
---|---|
Type Bool | Default value 0 |
If settings min_age_to_force_merge_seconds
and
min_age_to_force_merge_on_partition_only
should respect setting
max_bytes_to_merge_at_max_space_in_pool
.
Possible values:
true
false
enable_replacing_merge_with_cleanup_for_min_age_to_force_merge
Type | Default value |
---|---|
Type Bool | Default value 0 |
Whether to use CLEANUP merges for ReplacingMergeTree when merging partitions
down to a single part. Requires allow_experimental_replacing_merge_with_cleanup
,
min_age_to_force_merge_seconds
and min_age_to_force_merge_on_partition_only
to be enabled.
Possible values:
true
false
enforce_index_structure_match_on_partition_manipulation
Type | Default value |
---|---|
Type Bool | Default value 0 |
If this setting is enabled for destination table of a partition manipulation
query (ATTACH/MOVE/REPLACE PARTITION
), the indices and projections must be
identical between the source and destination tables. Otherwise, the destination
table can have a superset of the source table's indices and projections.
materialize_skip_indexes_on_merge
Type | Default value |
---|---|
Type Bool | Default value 1 |
When enabled, merges build and store skip indices for new parts. Otherwise they can be created/stored by explicit MATERIALIZE INDEX
max_merge_delayed_streams_for_parallel_write
Type | Default value |
---|---|
Type UInt64 | Default value 40 |
The maximum number of streams (columns) that can be flushed in parallel (analog of max_insert_delayed_streams_for_parallel_write for merges). Works only for Vertical merges.
max_postpone_time_for_failed_replicated_fetches_ms
Type | Default value |
---|---|
Type UInt64 | Default value 60000 |
The maximum postpone time for failed replicated fetches.
max_postpone_time_for_failed_replicated_merges_ms
Type | Default value |
---|---|
Type UInt64 | Default value 60000 |
The maximum postpone time for failed replicated merges.
max_postpone_time_for_failed_replicated_tasks_ms
Type | Default value |
---|---|
Type UInt64 | Default value 300000 |
The maximum postpone time for failed replicated task. The value is used if the task is not a fetch, merge or mutation.
merge_max_bytes_to_prewarm_cache
Type | Default value |
---|---|
Type UInt64 | Default value 1073741824 |
Only available in ClickHouse Cloud. Maximal size of part (compact or packed) to prewarm cache during merge.
merge_total_max_bytes_to_prewarm_cache
Type | Default value |
---|---|
Type UInt64 | Default value 16106127360 |
Only available in ClickHouse Cloud. Maximal size of parts in total to prewarm cache during merge.
min_bytes_to_prewarm_caches
Type | Default value |
---|---|
Type UInt64 | Default value 0 |
Minimal size (uncompressed bytes) to prewarm mark cache and primary index cache for new parts
notify_newest_block_number
Type | Default value |
---|---|
Type Bool | Default value 0 |
Notify newest block number to SharedJoin or SharedSet. Only in ClickHouse Cloud.
number_of_partitions_to_consider_for_merge
Type | Default value |
---|---|
Type UInt64 | Default value 10 |
Only available in ClickHouse Cloud. Up to top N partitions which we will consider for merge. Partitions picked in a random weighted way where weight is amount of data parts which can be merged in this partition.
prewarm_primary_key_cache
Type | Default value |
---|---|
Type Bool | Default value 0 |
If true primary index cache will be prewarmed by saving marks to mark cache on inserts, merges, fetches and on startup of server
reduce_blocking_parts_sleep_ms
Type | Default value |
---|---|
Type UInt64 | Default value 5000 |
Only available in ClickHouse Cloud. Minimum time to wait before trying to reduce blocking parts again after no ranges were dropped/replaced. A lower setting will trigger tasks in background_schedule_pool frequently which results in large amount of requests to zookeeper in large-scale clusters
refresh_parts_interval
Type | Default value |
---|---|
Type Seconds | Default value 0 |
If it is greater than zero - refresh the list of data parts from the underlying filesystem to check if the data was updated under the hood. It can be set only if the table is located on readonly disks (which means that this is a readonly replica, while data is being written by another replica).
shared_merge_tree_create_per_replica_metadata_nodes
Type | Default value |
---|---|
Type Bool | Default value 1 |
Enables creation of per-replica /metadata and /columns nodes in ZooKeeper. Only available in ClickHouse Cloud
shared_merge_tree_enable_keeper_parts_extra_data
Type | Default value |
---|---|
Type Bool | Default value 0 |
Enables writing attributes into virtual parts and committing blocks in keeper
shared_merge_tree_enable_outdated_parts_check
Type | Default value |
---|---|
Type Bool | Default value 1 |
Enable outdated parts check. Only available in ClickHouse Cloud
shared_merge_tree_idle_parts_update_seconds
Type | Default value |
---|---|
Type UInt64 | Default value 3600 |
Interval in seconds for parts update without being triggered by ZooKeeper watch in the shared merge tree. Only available in ClickHouse Cloud
shared_merge_tree_initial_parts_update_backoff_ms
Type | Default value |
---|---|
Type UInt64 | Default value 50 |
Initial backoff for parts update. Only available in ClickHouse Cloud
shared_merge_tree_interserver_http_connection_timeout_ms
Type | Default value |
---|---|
Type UInt64 | Default value 100 |
Timeouts for interserver HTTP connection. Only available in ClickHouse Cloud
shared_merge_tree_interserver_http_timeout_ms
Type | Default value |
---|---|
Type UInt64 | Default value 10000 |
Timeouts for interserver HTTP communication. Only available in ClickHouse Cloud
shared_merge_tree_leader_update_period_random_add_seconds
Type | Default value |
---|---|
Type UInt64 | Default value 10 |
Add uniformly distributed value from 0 to x seconds to shared_merge_tree_leader_update_period to avoid thundering herd effect. Only available in ClickHouse Cloud
shared_merge_tree_leader_update_period_seconds
Type | Default value |
---|---|
Type UInt64 | Default value 30 |
Maximum period to recheck leadership for parts update. Only available in ClickHouse Cloud
shared_merge_tree_max_outdated_parts_to_process_at_once
Type | Default value |
---|---|
Type UInt64 | Default value 1000 |
Maximum amount of outdated parts leader will try to confirm for removal at one HTTP request. Only available in ClickHouse Cloud.
shared_merge_tree_max_parts_update_backoff_ms
Type | Default value |
---|---|
Type UInt64 | Default value 5000 |
Max backoff for parts update. Only available in ClickHouse Cloud
shared_merge_tree_max_parts_update_leaders_in_total
Type | Default value |
---|---|
Type UInt64 | Default value 6 |
Maximum number of parts update leaders. Only available in ClickHouse Cloud
shared_merge_tree_max_parts_update_leaders_per_az
Type | Default value |
---|---|
Type UInt64 | Default value 2 |
Maximum number of parts update leaders. Only available in ClickHouse Cloud
shared_merge_tree_max_replicas_for_parts_deletion
Type | Default value |
---|---|
Type UInt64 | Default value 10 |
Max replicas which will participate in parts deletion (killer thread). Only available in ClickHouse Cloud
shared_merge_tree_max_replicas_to_merge_parts_for_each_parts_range
Type | Default value |
---|---|
Type UInt64 | Default value 5 |
Max replicas which will try to assign potentially conflicting merges (allow to avoid redundant conflicts in merges assignment). 0 means disabled. Only available in ClickHouse Cloud
shared_merge_tree_max_suspicious_broken_parts
Type | Default value |
---|---|
Type UInt64 | Default value 0 |
Max broken parts for SMT, if more - deny automatic detach.
shared_merge_tree_max_suspicious_broken_parts_bytes
Type | Default value |
---|---|
Type UInt64 | Default value 0 |
Max size of all broken parts for SMT, if more - deny automatic detach.
shared_merge_tree_memo_ids_remove_timeout_seconds
Type | Default value |
---|---|
Type Int64 | Default value 1800 |
How long we store insert memoization ids to avoid wrong actions during insert retries. Only available in ClickHouse Cloud
shared_merge_tree_postpone_next_merge_for_locally_merged_parts_ms
Type | Default value |
---|---|
Type UInt64 | Default value 0 |
Time to keep a locally merged part without starting a new merge containing this part. Gives other replicas a chance fetch the part and start this merge. Only available in ClickHouse Cloud.
shared_merge_tree_postpone_next_merge_for_locally_merged_parts_rows_threshold
Type | Default value |
---|---|
Type UInt64 | Default value 1000000 |
Minimum size of part (in rows) to postpone assigning a next merge just after merging it locally. Only available in ClickHouse Cloud.
shared_merge_tree_range_for_merge_window_size
Type | Default value |
---|---|
Type UInt64 | Default value 10 |
Time to keep a locally merged part without starting a new merge containing this part. Gives other replicas a chance fetch the part and start this merge. Only available in ClickHouse Cloud
shared_merge_tree_read_virtual_parts_from_leader
Type | Default value |
---|---|
Type Bool | Default value 1 |
Read virtual parts from leader when possible. Only available in ClickHouse Cloud
shared_merge_tree_try_fetch_part_in_memory_data_from_replicas
Type | Default value |
---|---|
Type Bool | Default value 0 |
If enabled all the replicas try to fetch part in memory data (like primary key, partition info and so on) from other replicas where it already exists.
shared_merge_tree_use_metadata_hints_cache
Type | Default value |
---|---|
Type Bool | Default value 1 |
Enables requesting FS cache hints from in-memory cache on other replicas. Only available in ClickHouse Cloud
shared_merge_tree_use_outdated_parts_compact_format
Type | Default value |
---|---|
Type Bool | Default value 0 |
Use compact format for outdated parts: reduces load to Keeper, improves outdated parts processing. Only available in ClickHouse Cloud
shared_merge_tree_use_too_many_parts_count_from_virtual_parts
Type | Default value |
---|---|
Type Bool | Default value 0 |
If enabled too many parts counter will rely on shared data in Keeper, not on local replica state. Only available in ClickHouse Cloud
table_disk
Type | Default value |
---|---|
Type Bool | Default value 0 |
This is table disk, the path/endpoint should point to the table data, not to the database data. Can be set only for s3_plain/s3_plain_rewritable/web.
use_primary_key_cache
Type | Default value |
---|---|
Type Bool | Default value 0 |
Use cache for primary index instead of saving all indexes in memory. Can be useful for very large tables
zero_copy_merge_mutation_min_parts_size_sleep_no_scale_before_lock
Type | Default value |
---|---|
Type UInt64 | Default value 0 |
If zero copy replication is enabled sleep random amount of time up to 500ms before trying to lock for merge or mutation.