Package org.apache.kafka.streams
Class StreamsConfig
java.lang.Object
org.apache.kafka.common.config.AbstractConfig
org.apache.kafka.streams.StreamsConfig
Configuration for a 
KafkaStreams instance.
 Can also be used to configure the Kafka Streams internal KafkaConsumer, KafkaProducer and Admin.
 To avoid consumer/producer/admin property conflicts, you should prefix those properties using
 consumerPrefix(String), producerPrefix(String) and adminClientPrefix(String), respectively.
 Example:
 // potentially wrong: sets "metadata.max.age.ms" to 1 minute for producer AND consumer
 Properties streamsProperties = new Properties();
 streamsProperties.put(ConsumerConfig.METADATA_MAX_AGE_CONFIG, 60000);
 // or
 streamsProperties.put(ProducerConfig.METADATA_MAX_AGE_CONFIG, 60000);
 // suggested:
 Properties streamsProperties = new Properties();
 // sets "metadata.max.age.ms" to 1 minute for consumer only
 streamsProperties.put(StreamsConfig.consumerPrefix(ConsumerConfig.METADATA_MAX_AGE_CONFIG), 60000);
 // sets "metadata.max.age.ms" to 1 minute for producer only
 streamsProperties.put(StreamsConfig.producerPrefix(ProducerConfig.METADATA_MAX_AGE_CONFIG), 60000);
 StreamsConfig streamsConfig = new StreamsConfig(streamsProperties);
 
 Properties streamsProperties = new Properties();
 // sets "my.custom.config" to "foo" for consumer only
 streamsProperties.put(StreamsConfig.consumerPrefix("my.custom.config"), "foo");
 // sets "my.custom.config" to "bar" for producer only
 streamsProperties.put(StreamsConfig.producerPrefix("my.custom.config"), "bar");
 // sets "my.custom.config2" to "boom" for all clients universally
 streamsProperties.put("my.custom.config2", "boom");
 // as a result, inside producer's serde class configure(..) function,
 // users can now read both key-value pairs "my.custom.config" -> "foo"
 // and "my.custom.config2" -> "boom" from the config map
 StreamsConfig streamsConfig = new StreamsConfig(streamsProperties);
 ProducerConfig.MAX_BLOCK_MS_CONFIG to be more resilient to non-available brokers you should also
 increase ConsumerConfig.MAX_POLL_INTERVAL_MS_CONFIG using the following guidance:
 
     max.poll.interval.ms > max.block.ms
 
 Kafka Streams requires at least the following properties to be set:
 
 By default, Kafka Streams does not allow users to overwrite the following properties (Streams setting shown in parentheses):
 - "group.id"(<application.id>) - Streams client will always use the application ID a consumer group ID
- "enable.auto.commit"(false) - Streams client will always disable/turn off auto committing
- "partition.assignment.strategy"(- StreamsPartitionAssignor) - Streams client will always use its own partition assignor
"processing.guarantee" is set to "exactly_once_v2",
 "exactly_once" (deprecated), or "exactly_once_beta" (deprecated), Kafka Streams does not
 allow users to overwrite the following properties (Streams setting shown in parentheses):
 - "isolation.level"(read_committed) - Consumers will always read committed data only
- "enable.idempotence"(true) - Producer will always have idempotency enabled
- 
Nested Class SummaryNested Classes
- 
Field SummaryFieldsModifier and TypeFieldDescriptionstatic final Stringacceptable.recovery.lagstatic final StringPrefix used to isolateadminconfigs from other client configs.static final Stringapplication.idstatic final Stringapplication.serverstatic final StringConfig value for parameter"processing.guarantee"for at-least-once processing guarantees.static final StringDeprecated.and will be removed in 4.0.0static final Stringbootstrap.serversstatic final Stringbuffered.records.per.partitionstatic final Stringstatic final Stringbuilt.in.metrics.versionstatic final StringDeprecated.since 3.4.0 Use"statestore.cache.max.bytes"instead.static final Stringstatic final Stringclient.idstatic final StringPrefix used to add arbitrary tags to a Kafka Stream's instance as key-value pairs.static final Stringcommit.interval.msstatic final Stringconnections.max.idle.msstatic final StringPrefix used to isolateconsumerconfigs from other client configs.static final Stringdefault.client.supplierstatic final Stringstatic final Stringdefault.deserialization.exception.handlerstatic final Stringstatic final StringDeprecated.static final StringDeprecated.static final StringDeprecated.static final Stringdefault key.serdestatic final Stringdefault.production.exception.handlerstatic final Stringdefault.timestamp.extractorstatic final Stringstatic final Stringdefault value.serdestatic final StringDeprecated.since 3.0.0 Use"windowed.inner.class.serde"instead.static final StringDeprecated.since 3.0.0 Use"windowed.inner.class.serde"instead.static final Stringdsl.store.suppliers.classstatic final intstatic final Stringenable.metrics.pushstatic final Stringstatic final StringDeprecated.since 3.0.0, will be removed in 4.0.static final StringDeprecated.since 3.0.0, will be removed in 4.0.static final StringConfig value for parameter"processing.guarantee"for exactly-once processing guarantees.static final StringPrefix used to overrideconsumerconfigs for the global consumer client from the general consumer client configs.static final StringDeprecated.static final StringPrefix used to overrideconsumerconfigs for the main consumer client from the general consumer client configs.static final intstatic final intstatic final intstatic final Stringmax.task.idle.msstatic final longstatic final Stringstatic final Stringmax.warmup.replicasstatic final StringConfig value for parameter"topology.optimization"for enabling the specific optimization that merges duplicated repartition topics.static final Stringmetadata.max.age.msstatic final Stringmetric.reportersstatic final StringConfig value for parameter"built.in.metrics.version"for the latest built-in metrics version.static final Stringmetrics.num.samplesstatic final Stringmetrics.record.levelstatic final Stringmetrics.sample.window.msstatic final StringConfig value for parameter"topology.optimization"for disabling topology optimizationstatic final Stringnum.standby.replicasstatic final Stringnum.stream.threadsstatic final StringConfig value for parameter"topology.optimization"for enabling topology optimizationstatic final Stringpoll.msstatic final Stringprobing.rebalance.interval.msstatic final Stringprocessing.guaranteestatic final StringPrefix used to isolateproducerconfigs from other client configs.static final Stringstatic final Stringstatic final Stringstatic final Stringrack.aware.assignment.strategystatic final Stringstatic final Stringstatic final Stringstatic final Stringrack.aware.assignment.tagsstatic final Stringstatic final Stringstatic final Stringreceive.buffer.bytesstatic final Stringreconnect.backoff.maxstatic final Stringreconnect.backoff.msstatic final Stringrepartition.purge.interval.msstatic final Stringreplication.factorstatic final Stringrequest.timeout.msstatic final StringPrefix used to overrideconsumerconfigs for the restore consumer client from the general consumer client configs.static final StringDeprecated.since 2.7static final Stringretry.backoff.msstatic final StringConfig value for parameter"topology.optimization"for enabling the specific optimization that reuses source topic as changelog topic for KTables.static final StringDeprecated.static final Stringrocksdb.config.setterstatic final Stringsecurity.protocolstatic final Stringsend.buffer.bytesstatic final StringConfig value for parameter"topology.optimization"for enabling the optimization that optimizes inner stream-stream joins into self-joins when both arguments are the same stream.static final Stringstate.cleanup.delaystatic final Stringstate.dirstatic final Stringstatestore.cache.max.bytesstatic final Stringstatic final Stringstatic final Stringtask.timeout.msstatic final Stringstatic final StringPrefix used to provide default topic configs to be applied when creating internal topics.static final StringDeprecated.since 2.7; useTOPOLOGY_OPTIMIZATION_CONFIGinsteadstatic final Stringtopology.optimizationstatic final StringConfig value for parameter"upgrade.from"for upgrading an application from version0.10.0.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version0.10.1.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version0.10.2.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version0.11.0.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version1.0.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version1.1.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version2.0.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version2.1.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version2.2.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version2.3.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version2.4.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version2.5.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version2.6.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version2.7.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version2.8.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version3.0.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version3.1.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version3.2.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version3.3.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version3.4.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version3.5.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version3.6.x.static final StringConfig value for parameter"upgrade.from"for upgrading an application from version3.7.x.static final Stringupgrade.fromstatic final Stringwindow.size.msstatic final Stringwindowstore.changelog.additional.retention.msstatic final StringFields inherited from class org.apache.kafka.common.config.AbstractConfigAUTOMATIC_CONFIG_PROVIDERS_PROPERTY, CONFIG_PROVIDERS_CONFIG
- 
Constructor SummaryConstructorsConstructorDescriptionStreamsConfig(Map<?, ?> props) Create a newStreamsConfigusing the given properties.
- 
Method SummaryModifier and TypeMethodDescriptionstatic StringadminClientPrefix(String adminClientProp) Prefix a property withADMIN_CLIENT_PREFIX.static StringclientTagPrefix(String clientTagKey) Prefix a client tag key withCLIENT_TAG_PREFIX.static ConfigDefReturn a copy of the config definition.static StringconsumerPrefix(String consumerProp) Prefix a property withCONSUMER_PREFIX.Serde<?>Return anconfiguredinstance ofkey Serde class.Serde<?>Return anconfiguredinstance ofvalue Serde class.getAdminConfigs(String clientId) Get the configs for theadmin client.Get the configured client tags set withCLIENT_TAG_PREFIXprefix.getGlobalConsumerConfigs(String clientId) Get the configs for theglobal consumer.Return configured KafkaClientSuppliergetMainConsumerConfigs(String groupId, String clientId, int threadIdx) Get the configs to themain consumer.getProducerConfigs(String clientId) Get the configs for theproducer.getRestoreConsumerConfigs(String clientId) Get the configs for therestore-consumer.static StringglobalConsumerPrefix(String consumerProp) Prefix a property withGLOBAL_CONSUMER_PREFIX.static voidstatic StringmainConsumerPrefix(String consumerProp) Prefix a property withMAIN_CONSUMER_PREFIX.static StringproducerPrefix(String producerProp) Prefix a property withPRODUCER_PREFIX.static StringrestoreConsumerPrefix(String consumerProp) Prefix a property withRESTORE_CONSUMER_PREFIX.static StringtopicPrefix(String topicProp) Prefix a property withTOPIC_PREFIXused to provide default topic configs to be applied when creating internal topics.Methods inherited from class org.apache.kafka.common.config.AbstractConfigdocumentationOf, equals, getBoolean, getClass, getConfiguredInstance, getConfiguredInstance, getConfiguredInstances, getConfiguredInstances, getConfiguredInstances, getDouble, getInt, getList, getLong, getPassword, getShort, getString, hashCode, ignore, logUnused, nonInternalValues, originals, originals, originalsStrings, originalsWithPrefix, originalsWithPrefix, typeOf, unused, values, valuesWithPrefixAllOrNothing, valuesWithPrefixOverride
- 
Field Details- 
DUMMY_THREAD_INDEXpublic static final int DUMMY_THREAD_INDEX- See Also:
 
- 
MAX_TASK_IDLE_MS_DISABLEDpublic static final long MAX_TASK_IDLE_MS_DISABLED- See Also:
 
- 
MAX_RACK_AWARE_ASSIGNMENT_TAG_LIST_SIZEpublic static final int MAX_RACK_AWARE_ASSIGNMENT_TAG_LIST_SIZE- See Also:
 
- 
MAX_RACK_AWARE_ASSIGNMENT_TAG_KEY_LENGTHpublic static final int MAX_RACK_AWARE_ASSIGNMENT_TAG_KEY_LENGTH- See Also:
 
- 
MAX_RACK_AWARE_ASSIGNMENT_TAG_VALUE_LENGTHpublic static final int MAX_RACK_AWARE_ASSIGNMENT_TAG_VALUE_LENGTH- See Also:
 
- 
TOPIC_PREFIXPrefix used to provide default topic configs to be applied when creating internal topics. These should be valid properties fromTopicConfig. It is recommended to usetopicPrefix(String).- See Also:
 
- 
CONSUMER_PREFIXPrefix used to isolateconsumerconfigs from other client configs. It is recommended to useconsumerPrefix(String)to add this prefix toconsumer properties.- See Also:
 
- 
MAIN_CONSUMER_PREFIXPrefix used to overrideconsumerconfigs for the main consumer client from the general consumer client configs. The override precedence is the following (from highest to lowest precedence): 1. main.consumer.[config-name] 2. consumer.[config-name] 3. [config-name]- See Also:
 
- 
RESTORE_CONSUMER_PREFIXPrefix used to overrideconsumerconfigs for the restore consumer client from the general consumer client configs. The override precedence is the following (from highest to lowest precedence): 1. restore.consumer.[config-name] 2. consumer.[config-name] 3. [config-name]- See Also:
 
- 
GLOBAL_CONSUMER_PREFIXPrefix used to overrideconsumerconfigs for the global consumer client from the general consumer client configs. The override precedence is the following (from highest to lowest precedence): 1. global.consumer.[config-name] 2. consumer.[config-name] 3. [config-name]- See Also:
 
- 
PRODUCER_PREFIXPrefix used to isolateproducerconfigs from other client configs. It is recommended to useproducerPrefix(String)to add this prefix toproducer properties.- See Also:
 
- 
ADMIN_CLIENT_PREFIXPrefix used to isolateadminconfigs from other client configs. It is recommended to useadminClientPrefix(String)to add this prefix toadmin client properties.- See Also:
 
- 
CLIENT_TAG_PREFIXPrefix used to add arbitrary tags to a Kafka Stream's instance as key-value pairs. Example: client.tag.zone=zone1 client.tag.cluster=cluster1- See Also:
 
- 
TOPOLOGY_OPTIMIZATION_CONFIGtopology.optimization- See Also:
 
- 
NO_OPTIMIZATIONConfig value for parameter"topology.optimization"for disabling topology optimization- See Also:
 
- 
OPTIMIZEConfig value for parameter"topology.optimization"for enabling topology optimization- See Also:
 
- 
REUSE_KTABLE_SOURCE_TOPICSConfig value for parameter"topology.optimization"for enabling the specific optimization that reuses source topic as changelog topic for KTables.- See Also:
 
- 
MERGE_REPARTITION_TOPICSConfig value for parameter"topology.optimization"for enabling the specific optimization that merges duplicated repartition topics.- See Also:
 
- 
SINGLE_STORE_SELF_JOINConfig value for parameter"topology.optimization"for enabling the optimization that optimizes inner stream-stream joins into self-joins when both arguments are the same stream.- See Also:
 
- 
UPGRADE_FROM_0100Config value for parameter"upgrade.from"for upgrading an application from version0.10.0.x.
- 
UPGRADE_FROM_0101Config value for parameter"upgrade.from"for upgrading an application from version0.10.1.x.
- 
UPGRADE_FROM_0102Config value for parameter"upgrade.from"for upgrading an application from version0.10.2.x.
- 
UPGRADE_FROM_0110Config value for parameter"upgrade.from"for upgrading an application from version0.11.0.x.
- 
UPGRADE_FROM_10Config value for parameter"upgrade.from"for upgrading an application from version1.0.x.
- 
UPGRADE_FROM_11Config value for parameter"upgrade.from"for upgrading an application from version1.1.x.
- 
UPGRADE_FROM_20Config value for parameter"upgrade.from"for upgrading an application from version2.0.x.
- 
UPGRADE_FROM_21Config value for parameter"upgrade.from"for upgrading an application from version2.1.x.
- 
UPGRADE_FROM_22Config value for parameter"upgrade.from"for upgrading an application from version2.2.x.
- 
UPGRADE_FROM_23Config value for parameter"upgrade.from"for upgrading an application from version2.3.x.
- 
UPGRADE_FROM_24Config value for parameter"upgrade.from"for upgrading an application from version2.4.x.
- 
UPGRADE_FROM_25Config value for parameter"upgrade.from"for upgrading an application from version2.5.x.
- 
UPGRADE_FROM_26Config value for parameter"upgrade.from"for upgrading an application from version2.6.x.
- 
UPGRADE_FROM_27Config value for parameter"upgrade.from"for upgrading an application from version2.7.x.
- 
UPGRADE_FROM_28Config value for parameter"upgrade.from"for upgrading an application from version2.8.x.
- 
UPGRADE_FROM_30Config value for parameter"upgrade.from"for upgrading an application from version3.0.x.
- 
UPGRADE_FROM_31Config value for parameter"upgrade.from"for upgrading an application from version3.1.x.
- 
UPGRADE_FROM_32Config value for parameter"upgrade.from"for upgrading an application from version3.2.x.
- 
UPGRADE_FROM_33Config value for parameter"upgrade.from"for upgrading an application from version3.3.x.
- 
UPGRADE_FROM_34Config value for parameter"upgrade.from"for upgrading an application from version3.4.x.
- 
UPGRADE_FROM_35Config value for parameter"upgrade.from"for upgrading an application from version3.5.x.
- 
UPGRADE_FROM_36Config value for parameter"upgrade.from"for upgrading an application from version3.6.x.
- 
UPGRADE_FROM_37Config value for parameter"upgrade.from"for upgrading an application from version3.7.x.
- 
AT_LEAST_ONCEConfig value for parameter"processing.guarantee"for at-least-once processing guarantees.- See Also:
 
- 
EXACTLY_ONCEDeprecated.since 3.0.0, will be removed in 4.0. Use"exactly_once_v2"instead.Config value for parameter"processing.guarantee"for exactly-once processing guarantees.Enabling exactly-once processing semantics requires broker version 0.11.0 or higher. If you enable this feature Kafka Streams will use more resources (like broker connections) compared to "at_least_once"and"exactly_once_v2".- See Also:
 
- 
EXACTLY_ONCE_BETADeprecated.since 3.0.0, will be removed in 4.0. Use"exactly_once_v2"instead.Config value for parameter"processing.guarantee"for exactly-once processing guarantees.Enabling exactly-once (beta) requires broker version 2.5 or higher. If you enable this feature Kafka Streams will use fewer resources (like broker connections) compared to the EXACTLY_ONCE(deprecated) case.- See Also:
 
- 
EXACTLY_ONCE_V2Config value for parameter"processing.guarantee"for exactly-once processing guarantees.Enabling exactly-once-v2 requires broker version 2.5 or higher. - See Also:
 
- 
METRICS_LATESTConfig value for parameter"built.in.metrics.version"for the latest built-in metrics version.- See Also:
 
- 
ACCEPTABLE_RECOVERY_LAG_CONFIGacceptable.recovery.lag- See Also:
 
- 
APPLICATION_ID_CONFIGapplication.id- See Also:
 
- 
APPLICATION_SERVER_CONFIGapplication.server- See Also:
 
- 
BOOTSTRAP_SERVERS_CONFIGbootstrap.servers- See Also:
 
- 
BUFFERED_RECORDS_PER_PARTITION_CONFIGbuffered.records.per.partition- See Also:
 
- 
BUFFERED_RECORDS_PER_PARTITION_DOC- See Also:
 
- 
BUILT_IN_METRICS_VERSION_CONFIGbuilt.in.metrics.version- See Also:
 
- 
CACHE_MAX_BYTES_BUFFERING_CONFIGDeprecated.since 3.4.0 Use"statestore.cache.max.bytes"instead.cache.max.bytes.buffering- See Also:
 
- 
CACHE_MAX_BYTES_BUFFERING_DOC- See Also:
 
- 
STATESTORE_CACHE_MAX_BYTES_CONFIGstatestore.cache.max.bytes- See Also:
 
- 
STATESTORE_CACHE_MAX_BYTES_DOC- See Also:
 
- 
CLIENT_ID_CONFIGclient.id- See Also:
 
- 
ENABLE_METRICS_PUSH_CONFIGenable.metrics.push- See Also:
 
- 
ENABLE_METRICS_PUSH_DOC- See Also:
 
- 
COMMIT_INTERVAL_MS_CONFIGcommit.interval.ms- See Also:
 
- 
REPARTITION_PURGE_INTERVAL_MS_CONFIGrepartition.purge.interval.ms- See Also:
 
- 
CONNECTIONS_MAX_IDLE_MS_CONFIGconnections.max.idle.ms- See Also:
 
- 
DEFAULT_DESERIALIZATION_EXCEPTION_HANDLER_CLASS_CONFIGdefault.deserialization.exception.handler- See Also:
 
- 
DEFAULT_DESERIALIZATION_EXCEPTION_HANDLER_CLASS_DOC- See Also:
 
- 
DEFAULT_PRODUCTION_EXCEPTION_HANDLER_CLASS_CONFIGdefault.production.exception.handler- See Also:
 
- 
DEFAULT_DSL_STORE_CONFIGDeprecated.default.dsl.store- See Also:
 
- 
DEFAULT_DSL_STORE_DOCDeprecated.- See Also:
 
- 
ROCKS_DBDeprecated.- See Also:
 
- 
IN_MEMORYDeprecated.- See Also:
 
- 
DEFAULT_DSL_STOREDeprecated.- See Also:
 
- 
DSL_STORE_SUPPLIERS_CLASS_CONFIGdsl.store.suppliers.class- See Also:
 
- 
DEFAULT_WINDOWED_KEY_SERDE_INNER_CLASSDeprecated.since 3.0.0 Use"windowed.inner.class.serde"instead.default.windowed.key.serde.inner- See Also:
 
- 
DEFAULT_WINDOWED_VALUE_SERDE_INNER_CLASSDeprecated.since 3.0.0 Use"windowed.inner.class.serde"instead.default.windowed.value.serde.inner- See Also:
 
- 
WINDOWED_INNER_CLASS_SERDE- See Also:
 
- 
DEFAULT_KEY_SERDE_CLASS_CONFIGdefault key.serde- See Also:
 
- 
DEFAULT_VALUE_SERDE_CLASS_CONFIGdefault value.serde- See Also:
 
- 
DEFAULT_TIMESTAMP_EXTRACTOR_CLASS_CONFIGdefault.timestamp.extractor- See Also:
 
- 
DEFAULT_TIMESTAMP_EXTRACTOR_CLASS_DOC- See Also:
 
- 
MAX_TASK_IDLE_MS_CONFIGmax.task.idle.ms- See Also:
 
- 
MAX_TASK_IDLE_MS_DOC- See Also:
 
- 
MAX_WARMUP_REPLICAS_CONFIGmax.warmup.replicas- See Also:
 
- 
METADATA_MAX_AGE_CONFIGmetadata.max.age.ms- See Also:
 
- 
METRICS_NUM_SAMPLES_CONFIGmetrics.num.samples- See Also:
 
- 
METRICS_RECORDING_LEVEL_CONFIGmetrics.record.level- See Also:
 
- 
METRIC_REPORTER_CLASSES_CONFIGmetric.reporters- See Also:
 
- 
METRICS_SAMPLE_WINDOW_MS_CONFIGmetrics.sample.window.ms- See Also:
 
- 
AUTO_INCLUDE_JMX_REPORTER_CONFIGDeprecated.and will be removed in 4.0.0auto.include.jmx.reporter- See Also:
 
- 
NUM_STANDBY_REPLICAS_CONFIGnum.standby.replicas- See Also:
 
- 
NUM_STREAM_THREADS_CONFIGnum.stream.threads- See Also:
 
- 
POLL_MS_CONFIGpoll.ms- See Also:
 
- 
PROBING_REBALANCE_INTERVAL_MS_CONFIGprobing.rebalance.interval.ms- See Also:
 
- 
PROCESSING_GUARANTEE_CONFIGprocessing.guarantee- See Also:
 
- 
RECEIVE_BUFFER_CONFIGreceive.buffer.bytes- See Also:
 
- 
RACK_AWARE_ASSIGNMENT_TAGS_CONFIGrack.aware.assignment.tags- See Also:
 
- 
RECONNECT_BACKOFF_MS_CONFIGreconnect.backoff.ms- See Also:
 
- 
RECONNECT_BACKOFF_MAX_MS_CONFIGreconnect.backoff.max- See Also:
 
- 
REPLICATION_FACTOR_CONFIGreplication.factor- See Also:
 
- 
REQUEST_TIMEOUT_MS_CONFIGrequest.timeout.ms- See Also:
 
- 
RETRIES_CONFIGDeprecated.since 2.7retriesThis config is ignored by Kafka Streams. Note, that the internal clients (producer, admin) are still impacted by this config. - See Also:
 
- 
RETRY_BACKOFF_MS_CONFIGretry.backoff.ms- See Also:
 
- 
ROCKSDB_CONFIG_SETTER_CLASS_CONFIGrocksdb.config.setter- See Also:
 
- 
SECURITY_PROTOCOL_CONFIGsecurity.protocol- See Also:
 
- 
SEND_BUFFER_CONFIGsend.buffer.bytes- See Also:
 
- 
STATE_CLEANUP_DELAY_MS_CONFIGstate.cleanup.delay- See Also:
 
- 
STATE_DIR_CONFIGstate.dir- See Also:
 
- 
TASK_TIMEOUT_MS_CONFIGtask.timeout.ms- See Also:
 
- 
TASK_TIMEOUT_MS_DOC- See Also:
 
- 
WINDOW_SIZE_MS_CONFIGwindow.size.ms- See Also:
 
- 
UPGRADE_FROM_CONFIGupgrade.from- See Also:
 
- 
WINDOW_STORE_CHANGE_LOG_ADDITIONAL_RETENTION_MS_CONFIGwindowstore.changelog.additional.retention.ms- See Also:
 
- 
DEFAULT_CLIENT_SUPPLIER_CONFIGdefault.client.supplier- See Also:
 
- 
DEFAULT_CLIENT_SUPPLIER_DOC- See Also:
 
- 
RACK_AWARE_ASSIGNMENT_STRATEGY_NONE- See Also:
 
- 
RACK_AWARE_ASSIGNMENT_STRATEGY_MIN_TRAFFIC- See Also:
 
- 
RACK_AWARE_ASSIGNMENT_STRATEGY_BALANCE_SUBTOPOLOGY- See Also:
 
- 
RACK_AWARE_ASSIGNMENT_STRATEGY_CONFIGrack.aware.assignment.strategy- See Also:
 
- 
RACK_AWARE_ASSIGNMENT_STRATEGY_DOC- See Also:
 
- 
RACK_AWARE_ASSIGNMENT_TRAFFIC_COST_CONFIG- See Also:
 
- 
RACK_AWARE_ASSIGNMENT_TRAFFIC_COST_DOC
- 
RACK_AWARE_ASSIGNMENT_NON_OVERLAP_COST_CONFIG- See Also:
 
- 
RACK_AWARE_ASSIGNMENT_NON_OVERLAP_COST_DOC
- 
TASK_ASSIGNOR_CLASS_CONFIG- See Also:
 
- 
TOPOLOGY_OPTIMIZATIONDeprecated.since 2.7; useTOPOLOGY_OPTIMIZATION_CONFIGinsteadtopology.optimization- See Also:
 
 
- 
- 
Constructor Details- 
StreamsConfigCreate a newStreamsConfigusing the given properties.- Parameters:
- props- properties that specify Kafka Streams and internal consumer/producer configuration
 
 
- 
- 
Method Details- 
consumerPrefixPrefix a property withCONSUMER_PREFIX. This is used to isolateconsumer configsfrom other client configs.- Parameters:
- consumerProp- the consumer property to be masked
- Returns:
- CONSUMER_PREFIX+- consumerProp
 
- 
mainConsumerPrefixPrefix a property withMAIN_CONSUMER_PREFIX. This is used to isolatemain consumer configsfrom other client configs.- Parameters:
- consumerProp- the consumer property to be masked
- Returns:
- MAIN_CONSUMER_PREFIX+- consumerProp
 
- 
restoreConsumerPrefixPrefix a property withRESTORE_CONSUMER_PREFIX. This is used to isolaterestore consumer configsfrom other client configs.- Parameters:
- consumerProp- the consumer property to be masked
- Returns:
- RESTORE_CONSUMER_PREFIX+- consumerProp
 
- 
clientTagPrefixPrefix a client tag key withCLIENT_TAG_PREFIX.- Parameters:
- clientTagKey- client tag key
- Returns:
- CLIENT_TAG_PREFIX+- clientTagKey
 
- 
globalConsumerPrefixPrefix a property withGLOBAL_CONSUMER_PREFIX. This is used to isolateglobal consumer configsfrom other client configs.- Parameters:
- consumerProp- the consumer property to be masked
- Returns:
- GLOBAL_CONSUMER_PREFIX+- consumerProp
 
- 
producerPrefixPrefix a property withPRODUCER_PREFIX. This is used to isolateproducer configsfrom other client configs.- Parameters:
- producerProp- the producer property to be masked
- Returns:
- PRODUCER_PREFIX + producerProp
 
- 
adminClientPrefixPrefix a property withADMIN_CLIENT_PREFIX. This is used to isolateadmin configsfrom other client configs.- Parameters:
- adminClientProp- the admin client property to be masked
- Returns:
- ADMIN_CLIENT_PREFIX + adminClientProp
 
- 
topicPrefixPrefix a property withTOPIC_PREFIXused to provide default topic configs to be applied when creating internal topics.- Parameters:
- topicProp- the topic property to be masked
- Returns:
- TOPIC_PREFIX + topicProp
 
- 
configDefReturn a copy of the config definition.- Returns:
- a copy of the config definition
 
- 
getMainConsumerConfigsGet the configs to themain consumer. Properties using the prefixMAIN_CONSUMER_PREFIXwill be used in favor over the properties prefixed withCONSUMER_PREFIXand the non-prefixed versions (read the override precedence ordering inMAIN_CONSUMER_PREFIX) except in the case ofConsumerConfig.BOOTSTRAP_SERVERS_CONFIGwhere we always use the non-prefixed version as we only support reading/writing from/to the same Kafka Cluster. If not specified byMAIN_CONSUMER_PREFIX, main consumer will share the general consumer configs prefixed byCONSUMER_PREFIX.- Parameters:
- groupId- consumer groupId
- clientId- clientId
- threadIdx- stream thread index
- Returns:
- Map of the consumer configuration.
 
- 
getRestoreConsumerConfigsGet the configs for therestore-consumer. Properties using the prefixRESTORE_CONSUMER_PREFIXwill be used in favor over the properties prefixed withCONSUMER_PREFIXand the non-prefixed versions (read the override precedence ordering inRESTORE_CONSUMER_PREFIX) except in the case ofConsumerConfig.BOOTSTRAP_SERVERS_CONFIGwhere we always use the non-prefixed version as we only support reading/writing from/to the same Kafka Cluster. If not specified byRESTORE_CONSUMER_PREFIX, restore consumer will share the general consumer configs prefixed byCONSUMER_PREFIX.- Parameters:
- clientId- clientId
- Returns:
- Map of the restore consumer configuration.
 
- 
getGlobalConsumerConfigsGet the configs for theglobal consumer. Properties using the prefixGLOBAL_CONSUMER_PREFIXwill be used in favor over the properties prefixed withCONSUMER_PREFIXand the non-prefixed versions (read the override precedence ordering inGLOBAL_CONSUMER_PREFIX) except in the case ofConsumerConfig.BOOTSTRAP_SERVERS_CONFIGwhere we always use the non-prefixed version as we only support reading/writing from/to the same Kafka Cluster. If not specified byGLOBAL_CONSUMER_PREFIX, global consumer will share the general consumer configs prefixed byCONSUMER_PREFIX.- Parameters:
- clientId- clientId
- Returns:
- Map of the global consumer configuration.
 
- 
getProducerConfigsGet the configs for theproducer. Properties using the prefixPRODUCER_PREFIXwill be used in favor over their non-prefixed versions except in the case ofProducerConfig.BOOTSTRAP_SERVERS_CONFIGwhere we always use the non-prefixed version as we only support reading/writing from/to the same Kafka Cluster.- Parameters:
- clientId- clientId
- Returns:
- Map of the producer configuration.
 
- 
getAdminConfigsGet the configs for theadmin client.- Parameters:
- clientId- clientId
- Returns:
- Map of the admin client configuration.
 
- 
getClientTagsGet the configured client tags set withCLIENT_TAG_PREFIXprefix.- Returns:
- Map of the client tags.
 
- 
verifyTopologyOptimizationConfigs
- 
getKafkaClientSupplierReturn configured KafkaClientSupplier- Returns:
- Configured KafkaClientSupplier
 
- 
defaultKeySerdeReturn anconfiguredinstance ofkey Serde class.- Returns:
- a configured instance of key Serde class
 
- 
defaultValueSerdeReturn anconfiguredinstance ofvalue Serde class.- Returns:
- an configured instance of value Serde class
 
- 
defaultTimestampExtractor
- 
defaultDeserializationExceptionHandler
- 
defaultProductionExceptionHandler
- 
main
 
-