fromConfig(
* StreamExecutionEnvironment}.
*
* 3) Use the {@link CheckpointStorage} instance configured via the clusters
- * flink-conf.yaml.
+ * config.yaml.
*
*
4) Load a default {@link CheckpointStorage} instance.
*
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/state/filesystem/FsStateBackend.java b/flink-runtime/src/main/java/org/apache/flink/runtime/state/filesystem/FsStateBackend.java
index aded04d7b6b2e..ec219432faae4 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/state/filesystem/FsStateBackend.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/state/filesystem/FsStateBackend.java
@@ -67,7 +67,7 @@
* env.getCheckpointConfig().setCheckpointStorage("hdfs:///checkpoints");
* }
*
- *
If you are configuring your state backend via the {@code flink-conf.yaml} please make the
+ *
If you are configuring your state backend via the {@code config.yaml} please make the
* following changes set your state backend type to "hashmap" {@code state.backend.type: hashmap}.
*
*
This state backend holds the working state in the memory (JVM heap) of the TaskManagers. The
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/state/memory/MemoryStateBackend.java b/flink-runtime/src/main/java/org/apache/flink/runtime/state/memory/MemoryStateBackend.java
index 87d8d77803eb5..6345c2b4602b9 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/state/memory/MemoryStateBackend.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/state/memory/MemoryStateBackend.java
@@ -59,7 +59,7 @@
* env.getCheckpointConfig().setCheckpointStorage(new JobManagerCheckpointStorage());
* }
*
- *
If you are configuring your state backend via the {@code flink-conf.yaml} please make the
+ *
If you are configuring your state backend via the {@code config.yaml} please make the
* following changes:
*
*
{@code
diff --git a/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/EmbeddedRocksDBStateBackend.java b/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/EmbeddedRocksDBStateBackend.java
index 048515b5b449c..6d15c53c1a287 100644
--- a/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/EmbeddedRocksDBStateBackend.java
+++ b/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/EmbeddedRocksDBStateBackend.java
@@ -698,7 +698,7 @@ public void setPriorityQueueStateType(PriorityQueueStateType priorityQueueStateT
* Sets the predefined options for RocksDB.
*
* If user-configured options within {@link RocksDBConfigurableOptions} is set (through
- * flink-conf.yaml) or a user-defined options factory is set (via {@link
+ * config.yaml) or a user-defined options factory is set (via {@link
* #setRocksDBOptions(RocksDBOptionsFactory)}), then the options from the factory are applied on
* top of the here specified predefined options and customized options.
*
@@ -714,7 +714,7 @@ public void setPredefinedOptions(@Nonnull PredefinedOptions options) {
* PredefinedOptions#DEFAULT}.
*
*
If user-configured options within {@link RocksDBConfigurableOptions} is set (through
- * flink-conf.yaml) of a user-defined options factory is set (via {@link
+ * config.yaml) of a user-defined options factory is set (via {@link
* #setRocksDBOptions(RocksDBOptionsFactory)}), then the options from the factory are applied on
* top of the predefined and customized options.
*
diff --git a/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBOptions.java b/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBOptions.java
index 69576520fc387..e78d9fb0eb566 100644
--- a/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBOptions.java
+++ b/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBOptions.java
@@ -145,7 +145,7 @@ public class RocksDBOptions {
+ "This option only takes effect if neither '%s' nor '%s' are not configured. If none is configured "
+ "then each RocksDB column family state has its own memory caches (as controlled by the column "
+ "family options). "
- + "The relevant options for the shared resources (e.g. write-buffer-ratio) can be set on the same level (flink-conf.yaml)."
+ + "The relevant options for the shared resources (e.g. write-buffer-ratio) can be set on the same level (config.yaml)."
+ "Note, that this feature breaks resource isolation between the slots",
USE_MANAGED_MEMORY.key(), FIX_PER_SLOT_MEMORY_SIZE.key()));
diff --git a/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java b/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java
index 0c37a1e6bc5a9..dd3fcb70cf4ec 100644
--- a/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java
+++ b/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackend.java
@@ -61,8 +61,7 @@
* env.getCheckpointConfig().setCheckpointStorage("hdfs://checkpoints");
* }
*
- * If you are configuring your state backend via the {@code flink-conf.yaml} no changes are
- * required.
+ *
If you are configuring your state backend via the {@code config.yaml} no changes are required.
*
*
A State Backend that stores its state in {@code RocksDB}. This state backend can store very
* large state that exceeds memory and spills to disk.
@@ -399,7 +398,7 @@ public void setPriorityQueueStateType(PriorityQueueStateType priorityQueueStateT
* Sets the predefined options for RocksDB.
*
*
If user-configured options within {@link RocksDBConfigurableOptions} is set (through
- * flink-conf.yaml) or a user-defined options factory is set (via {@link
+ * config.yaml) or a user-defined options factory is set (via {@link
* #setRocksDBOptions(RocksDBOptionsFactory)}), then the options from the factory are applied on
* top of the here specified predefined options and customized options.
*
@@ -415,7 +414,7 @@ public void setPredefinedOptions(@Nonnull PredefinedOptions options) {
* PredefinedOptions#DEFAULT}.
*
*
If user-configured options within {@link RocksDBConfigurableOptions} is set (through
- * flink-conf.yaml) of a user-defined options factory is set (via {@link
+ * config.yaml) of a user-defined options factory is set (via {@link
* #setRocksDBOptions(RocksDBOptionsFactory)}), then the options from the factory are applied on
* top of the predefined and customized options.
*
diff --git a/flink-state-backends/flink-statebackend-rocksdb/src/test/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackendConfigTest.java b/flink-state-backends/flink-statebackend-rocksdb/src/test/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackendConfigTest.java
index 82a42dfdb3cd8..34d27e1bb5915 100644
--- a/flink-state-backends/flink-statebackend-rocksdb/src/test/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackendConfigTest.java
+++ b/flink-state-backends/flink-statebackend-rocksdb/src/test/java/org/apache/flink/contrib/streaming/state/RocksDBStateBackendConfigTest.java
@@ -249,7 +249,7 @@ public void testConfigureRocksDBPriorityQueueFactoryCacheSize() throws Exception
env.close();
}
- /** Validates that user custom configuration from code should override the flink-conf.yaml. */
+ /** Validates that user custom configuration from code should override the config.yaml. */
@Test
public void testConfigureTimerServiceLoadingFromApplication() throws Exception {
final MockEnvironment env = new MockEnvironmentBuilder().build();
@@ -524,7 +524,7 @@ public void testPredefinedOptions() throws Exception {
// verify that we would use PredefinedOptions.DEFAULT by default.
assertEquals(PredefinedOptions.DEFAULT, rocksDbBackend.getPredefinedOptions());
- // verify that user could configure predefined options via flink-conf.yaml
+ // verify that user could configure predefined options via config.yaml
Configuration configuration = new Configuration();
configuration.set(
RocksDBOptions.PREDEFINED_OPTIONS, PredefinedOptions.FLASH_SSD_OPTIMIZED.name());
@@ -640,7 +640,7 @@ public void testOptionsFactory() throws Exception {
String checkpointPath = tempFolder.newFolder().toURI().toString();
RocksDBStateBackend rocksDbBackend = new RocksDBStateBackend(checkpointPath);
- // verify that user-defined options factory could be configured via flink-conf.yaml
+ // verify that user-defined options factory could be configured via config.yaml
Configuration config = new Configuration();
config.setString(RocksDBOptions.OPTIONS_FACTORY.key(), TestOptionsFactory.class.getName());
config.setString(TestOptionsFactory.BACKGROUND_JOBS_OPTION.key(), "4");
diff --git a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/environment/StreamExecutionEnvironment.java b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/environment/StreamExecutionEnvironment.java
index f14f3d4400a1c..b179ead220d9f 100644
--- a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/environment/StreamExecutionEnvironment.java
+++ b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/environment/StreamExecutionEnvironment.java
@@ -2591,7 +2591,7 @@ public static StreamExecutionEnvironment getExecutionEnvironment() {
* execution environment, as returned by {@link #createLocalEnvironment(Configuration)}.
*
*
When executed from the command line the given configuration is stacked on top of the
- * global configuration which comes from the {@code flink-conf.yaml}, potentially overriding
+ * global configuration which comes from the {@code config.yaml}, potentially overriding
* duplicated options.
*
* @param configuration The configuration to instantiate the environment with.
diff --git a/flink-table/flink-sql-gateway/src/main/java/org/apache/flink/table/gateway/service/context/DefaultContext.java b/flink-table/flink-sql-gateway/src/main/java/org/apache/flink/table/gateway/service/context/DefaultContext.java
index 551e05ef962f8..eae416cd0a628 100644
--- a/flink-table/flink-sql-gateway/src/main/java/org/apache/flink/table/gateway/service/context/DefaultContext.java
+++ b/flink-table/flink-sql-gateway/src/main/java/org/apache/flink/table/gateway/service/context/DefaultContext.java
@@ -127,8 +127,8 @@ private static CustomCommandLine findActiveCommandLine(
// -------------------------------------------------------------------------------------------
/**
- * Build the {@link DefaultContext} from flink-conf.yaml, dynamic configuration and users
- * specified jars.
+ * Build the {@link DefaultContext} from config.yaml, dynamic configuration and users specified
+ * jars.
*
* @param dynamicConfig user specified configuration.
* @param dependencies user specified jars
diff --git a/flink-table/flink-sql-gateway/src/test/java/org/apache/flink/table/gateway/rest/util/SqlGatewayRestEndpointTestUtils.java b/flink-table/flink-sql-gateway/src/test/java/org/apache/flink/table/gateway/rest/util/SqlGatewayRestEndpointTestUtils.java
index 0c3e3ede7a624..65daeb4a3a819 100644
--- a/flink-table/flink-sql-gateway/src/test/java/org/apache/flink/table/gateway/rest/util/SqlGatewayRestEndpointTestUtils.java
+++ b/flink-table/flink-sql-gateway/src/test/java/org/apache/flink/table/gateway/rest/util/SqlGatewayRestEndpointTestUtils.java
@@ -43,7 +43,7 @@ public static Configuration getBaseConfig(Configuration flinkConf) {
return rebuildRestEndpointOptions(context.getEndpointOptions());
}
- /** Create the configuration generated from flink-conf.yaml. */
+ /** Create the configuration generated from config.yaml. */
public static Configuration getFlinkConfig(
String address, String bindAddress, String portRange) {
final Configuration config = new Configuration();
diff --git a/flink-table/flink-sql-gateway/src/test/java/org/apache/flink/table/gateway/service/context/SessionContextTest.java b/flink-table/flink-sql-gateway/src/test/java/org/apache/flink/table/gateway/service/context/SessionContextTest.java
index e5d22ed943e60..8ca455d31b4fa 100644
--- a/flink-table/flink-sql-gateway/src/test/java/org/apache/flink/table/gateway/service/context/SessionContextTest.java
+++ b/flink-table/flink-sql-gateway/src/test/java/org/apache/flink/table/gateway/service/context/SessionContextTest.java
@@ -184,7 +184,7 @@ void testCreateContextWithListeners() {
CatalogListener1.class.getName(),
CatalogListener2.class.getName()));
- // Find and create listeners from flink-conf.yaml for session
+ // Find and create listeners from config.yaml for session
flinkConfig.set(
TableConfigOptions.TABLE_CATALOG_MODIFICATION_LISTENERS,
Arrays.asList(CatalogFactory1.IDENTIFIER, CatalogFactory2.IDENTIFIER));
diff --git a/flink-table/flink-table-api-java/src/main/java/org/apache/flink/table/api/TableConfig.java b/flink-table/flink-table-api-java/src/main/java/org/apache/flink/table/api/TableConfig.java
index f80ad6393ecfa..3d3ffe29cca3f 100644
--- a/flink-table/flink-table-api-java/src/main/java/org/apache/flink/table/api/TableConfig.java
+++ b/flink-table/flink-table-api-java/src/main/java/org/apache/flink/table/api/TableConfig.java
@@ -50,7 +50,7 @@
* configuration can be set in any of the following layers (in the given order):
*
*
- * - {@code flink-conf.yaml},
+ *
- {@code config.yaml},
*
- CLI parameters,
*
- {@code StreamExecutionEnvironment} when bridging to DataStream API,
*
- {@link EnvironmentSettings.Builder#withConfiguration(Configuration)} / {@link
@@ -103,7 +103,7 @@ public TableConfig() {}
// Note to implementers:
// TableConfig is a ReadableConfig which is built once the TableEnvironment is created and
- // contains both the configuration defined in the execution context (flink-conf.yaml + CLI
+ // contains both the configuration defined in the execution context (config.yaml + CLI
// params), stored in rootConfiguration, but also any extra configuration defined by the user in
// the application, which has precedence over the execution configuration.
//
diff --git a/flink-test-utils-parent/flink-connector-test-utils/src/main/java/org/apache/flink/connector/testframe/container/FlinkContainersSettings.java b/flink-test-utils-parent/flink-connector-test-utils/src/main/java/org/apache/flink/connector/testframe/container/FlinkContainersSettings.java
index cd2bee6b39c6b..f3c79c4b35dea 100644
--- a/flink-test-utils-parent/flink-connector-test-utils/src/main/java/org/apache/flink/connector/testframe/container/FlinkContainersSettings.java
+++ b/flink-test-utils-parent/flink-connector-test-utils/src/main/java/org/apache/flink/connector/testframe/container/FlinkContainersSettings.java
@@ -283,8 +283,8 @@ public Builder jarPaths(Collection jarPaths) {
}
/**
- * Sets a single Flink configuration parameter (the options for flink-conf.yaml) and returns
- * a reference to this Builder enabling method chaining.
+ * Sets a single Flink configuration parameter (the options for config.yaml) and returns a
+ * reference to this Builder enabling method chaining.
*
* @param The type parameter.
* @param option The option.
diff --git a/flink-test-utils-parent/flink-connector-test-utils/src/main/java/org/apache/flink/connector/testframe/container/FlinkImageBuilder.java b/flink-test-utils-parent/flink-connector-test-utils/src/main/java/org/apache/flink/connector/testframe/container/FlinkImageBuilder.java
index 38b542e1310cf..1ee0c9f6999ac 100644
--- a/flink-test-utils-parent/flink-connector-test-utils/src/main/java/org/apache/flink/connector/testframe/container/FlinkImageBuilder.java
+++ b/flink-test-utils-parent/flink-connector-test-utils/src/main/java/org/apache/flink/connector/testframe/container/FlinkImageBuilder.java
@@ -125,7 +125,7 @@ public FlinkImageBuilder setJavaVersion(String javaVersion) {
}
/**
- * Sets Flink configuration. This configuration will be used for generating flink-conf.yaml for
+ * Sets Flink configuration. This configuration will be used for generating config.yaml for
* configuring JobManager and TaskManager.
*/
public FlinkImageBuilder setConfiguration(Configuration conf) {
@@ -209,12 +209,10 @@ public ImageFromDockerfile build() throws ImageBuildException {
final Path flinkConfFile = createTemporaryFlinkConfFile(conf, tempDirectory);
final Path log4jPropertiesFile = createTemporaryLog4jPropertiesFile(tempDirectory);
- // Copy flink-conf.yaml into image
- // NOTE: Before we change the default conf file in the flink-dist to 'config.yaml', we
- // need to use the legacy flink conf file 'flink-conf.yaml' here.
+ // Copy config.yaml into image
filesToCopy.put(
flinkConfFile,
- Paths.get(flinkHome, "conf", GlobalConfiguration.LEGACY_FLINK_CONF_FILENAME));
+ Paths.get(flinkHome, "conf", GlobalConfiguration.FLINK_CONF_FILENAME));
filesToCopy.put(
log4jPropertiesFile, Paths.get(flinkHome, "conf", LOG4J_PROPERTIES_FILENAME));
@@ -292,13 +290,10 @@ private String getJavaVersionSuffix() {
private Path createTemporaryFlinkConfFile(Configuration finalConfiguration, Path tempDirectory)
throws IOException {
- // Create a temporary flink-conf.yaml file and write merged configurations into it
- // NOTE: Before we change the default conf file in the flink-dist to 'config.yaml', we
- // need to use the legacy flink conf file 'flink-conf.yaml' here.
- Path flinkConfFile = tempDirectory.resolve(GlobalConfiguration.LEGACY_FLINK_CONF_FILENAME);
+ Path flinkConfFile = tempDirectory.resolve(GlobalConfiguration.FLINK_CONF_FILENAME);
Files.write(
flinkConfFile,
- ConfigurationUtils.convertConfigToWritableLines(finalConfiguration, true));
+ ConfigurationUtils.convertConfigToWritableLines(finalConfiguration, false));
return flinkConfFile;
}
diff --git a/flink-yarn-tests/src/test/java/org/apache/flink/yarn/YarnTestBase.java b/flink-yarn-tests/src/test/java/org/apache/flink/yarn/YarnTestBase.java
index fcf32ea13be52..ec7c6905ee7de 100644
--- a/flink-yarn-tests/src/test/java/org/apache/flink/yarn/YarnTestBase.java
+++ b/flink-yarn-tests/src/test/java/org/apache/flink/yarn/YarnTestBase.java
@@ -821,10 +821,9 @@ private static void start(
File flinkConfDirPath =
TestUtils.findFile(
- // NOTE: Before we change the default conf file in the flink-dist to
- // 'config.yaml', we need to use the legacy flink conf file
- // 'flink-conf.yaml' here.
- flinkDistRootDir, new ContainsName(new String[] {"flink-conf.yaml"}));
+ flinkDistRootDir,
+ new ContainsName(
+ new String[] {GlobalConfiguration.FLINK_CONF_FILENAME}));
assertThat(flinkConfDirPath).isNotNull();
final String confDirPath = flinkConfDirPath.getParentFile().getAbsolutePath();
@@ -839,7 +838,8 @@ private static void start(
FileUtils.copyDirectory(new File(confDirPath), tempConfPathForSecureRun);
BootstrapTools.writeConfiguration(
- globalConfiguration, new File(tempConfPathForSecureRun, "flink-conf.yaml"));
+ globalConfiguration,
+ new File(tempConfPathForSecureRun, GlobalConfiguration.FLINK_CONF_FILENAME));
String configDir = tempConfPathForSecureRun.getAbsolutePath();
diff --git a/flink-yarn/src/main/java/org/apache/flink/yarn/Utils.java b/flink-yarn/src/main/java/org/apache/flink/yarn/Utils.java
index 4f81077b16e81..5d9c77695e00e 100644
--- a/flink-yarn/src/main/java/org/apache/flink/yarn/Utils.java
+++ b/flink-yarn/src/main/java/org/apache/flink/yarn/Utils.java
@@ -485,7 +485,7 @@ static ContainerLaunchContext createTaskExecutorContext(
*
* @param flinkConfig The Flink configuration.
* @param tmParams Parameters for the task manager.
- * @param configDirectory The configuration directory for the flink-conf.yaml
+ * @param configDirectory The configuration directory for the config.yaml
* @param logDirectory The log directory.
* @param hasLogback Uses logback?
* @param hasLog4j Uses log4j?