This is an automated email from the ASF dual-hosted git repository.
huweihua pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/flink.git
The following commit(s) were added to refs/heads/master by this push:
new b3557cfdc0a [FLINK-32771][SlotManager] Remove the invalid config
option slotmanager.request-timeout
b3557cfdc0a is described below
commit b3557cfdc0a2897356027c16d97161c11680c802
Author: Weihua Hu <[email protected]>
AuthorDate: Mon Aug 14 19:35:51 2023 +0800
[FLINK-32771][SlotManager] Remove the invalid config option
slotmanager.request-timeout
This closes #23210
---
.../configuration/ResourceManagerOptions.java | 15 ++-------
.../slotmanager/SlotManagerConfiguration.java | 25 ---------------
.../slotmanager/DeclarativeSlotManagerBuilder.java | 8 -----
.../SlotManagerConfigurationBuilder.java | 8 -----
.../slotmanager/SlotManagerConfigurationTest.java | 37 ----------------------
5 files changed, 2 insertions(+), 91 deletions(-)
diff --git
a/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java
b/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java
index 6a681fb7396..5e6e4b8dc1d 100644
---
a/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java
+++
b/flink-core/src/main/java/org/apache/flink/configuration/ResourceManagerOptions.java
@@ -166,24 +166,13 @@ public class ResourceManagerOptions {
.defaultValue(Duration.ofMillis(50))
.withDescription("The delay of the declare needed
resources.");
- /**
- * The timeout for a slot request to be discarded, in milliseconds.
- *
- * @deprecated Use {@link JobManagerOptions#SLOT_REQUEST_TIMEOUT}.
- */
- @Deprecated
- public static final ConfigOption<Long> SLOT_REQUEST_TIMEOUT =
- ConfigOptions.key("slotmanager.request-timeout")
- .longType()
- .defaultValue(-1L)
- .withDescription("The timeout for a slot request to be
discarded.");
-
/**
* Time in milliseconds of the start-up period of a standalone cluster.
During this time,
* resource manager of the standalone cluster expects new task executors
to be registered, and
* will not fail slot requests that can not be satisfied by any current
registered slots. After
* this time, it will fail pending and new coming requests immediately
that can not be satisfied
- * by registered slots. If not set, {@link #SLOT_REQUEST_TIMEOUT} will be
used by default.
+ * by registered slots. If not set, {@link
JobManagerOptions#SLOT_REQUEST_TIMEOUT} will be used
+ * by default.
*/
public static final ConfigOption<Long>
STANDALONE_CLUSTER_STARTUP_PERIOD_TIME =
ConfigOptions.key("resourcemanager.standalone.start-up-time")
diff --git
a/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java
b/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java
index fe8442ee04f..d34f0a6e9f5 100644
---
a/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java
+++
b/flink-runtime/src/main/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfiguration.java
@@ -23,7 +23,6 @@ import org.apache.flink.api.common.time.Time;
import org.apache.flink.configuration.AkkaOptions;
import org.apache.flink.configuration.ClusterOptions;
import org.apache.flink.configuration.Configuration;
-import org.apache.flink.configuration.JobManagerOptions;
import org.apache.flink.configuration.MemorySize;
import org.apache.flink.configuration.ResourceManagerOptions;
import org.apache.flink.configuration.TaskManagerOptions;
@@ -42,7 +41,6 @@ public class SlotManagerConfiguration {
private static final Logger LOGGER =
LoggerFactory.getLogger(SlotManagerConfiguration.class);
private final Time taskManagerRequestTimeout;
- private final Time slotRequestTimeout;
private final Time taskManagerTimeout;
private final Duration requirementCheckDelay;
private final Duration declareNeededResourceDelay;
@@ -58,7 +56,6 @@ public class SlotManagerConfiguration {
public SlotManagerConfiguration(
Time taskManagerRequestTimeout,
- Time slotRequestTimeout,
Time taskManagerTimeout,
Duration requirementCheckDelay,
Duration declareNeededResourceDelay,
@@ -73,7 +70,6 @@ public class SlotManagerConfiguration {
int redundantTaskManagerNum) {
this.taskManagerRequestTimeout =
Preconditions.checkNotNull(taskManagerRequestTimeout);
- this.slotRequestTimeout =
Preconditions.checkNotNull(slotRequestTimeout);
this.taskManagerTimeout =
Preconditions.checkNotNull(taskManagerTimeout);
this.requirementCheckDelay =
Preconditions.checkNotNull(requirementCheckDelay);
this.declareNeededResourceDelay =
Preconditions.checkNotNull(declareNeededResourceDelay);
@@ -95,10 +91,6 @@ public class SlotManagerConfiguration {
return taskManagerRequestTimeout;
}
- public Time getSlotRequestTimeout() {
- return slotRequestTimeout;
- }
-
public Time getTaskManagerTimeout() {
return taskManagerTimeout;
}
@@ -154,7 +146,6 @@ public class SlotManagerConfiguration {
final Time rpcTimeout =
Time.fromDuration(configuration.get(AkkaOptions.ASK_TIMEOUT_DURATION));
- final Time slotRequestTimeout = getSlotRequestTimeout(configuration);
final Time taskManagerTimeout =
Time.milliseconds(
configuration.getLong(ResourceManagerOptions.TASK_MANAGER_TIMEOUT));
@@ -185,7 +176,6 @@ public class SlotManagerConfiguration {
return new SlotManagerConfiguration(
rpcTimeout,
- slotRequestTimeout,
taskManagerTimeout,
requirementCheckDelay,
declareNeededResourceDelay,
@@ -200,21 +190,6 @@ public class SlotManagerConfiguration {
redundantTaskManagerNum);
}
- private static Time getSlotRequestTimeout(final Configuration
configuration) {
- final long slotRequestTimeoutMs;
- if
(configuration.contains(ResourceManagerOptions.SLOT_REQUEST_TIMEOUT)) {
- LOGGER.warn(
- "Config key {} is deprecated; use {} instead.",
- ResourceManagerOptions.SLOT_REQUEST_TIMEOUT,
- JobManagerOptions.SLOT_REQUEST_TIMEOUT);
- slotRequestTimeoutMs =
-
configuration.getLong(ResourceManagerOptions.SLOT_REQUEST_TIMEOUT);
- } else {
- slotRequestTimeoutMs =
configuration.getLong(JobManagerOptions.SLOT_REQUEST_TIMEOUT);
- }
- return Time.milliseconds(slotRequestTimeoutMs);
- }
-
private static CPUResource getMaxTotalCpu(
final Configuration configuration,
final WorkerResourceSpec defaultWorkerResourceSpec,
diff --git
a/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/DeclarativeSlotManagerBuilder.java
b/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/DeclarativeSlotManagerBuilder.java
index 4fd874abde6..bfc291a120c 100644
---
a/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/DeclarativeSlotManagerBuilder.java
+++
b/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/DeclarativeSlotManagerBuilder.java
@@ -39,7 +39,6 @@ public class DeclarativeSlotManagerBuilder {
private boolean evenlySpreadOutSlots;
private final ScheduledExecutor scheduledExecutor;
private Time taskManagerRequestTimeout;
- private Time slotRequestTimeout;
private Time taskManagerTimeout;
private boolean waitResultConsumedBeforeRelease;
private WorkerResourceSpec defaultWorkerResourceSpec;
@@ -56,7 +55,6 @@ public class DeclarativeSlotManagerBuilder {
this.evenlySpreadOutSlots = false;
this.scheduledExecutor = scheduledExecutor;
this.taskManagerRequestTimeout = TestingUtils.infiniteTime();
- this.slotRequestTimeout = TestingUtils.infiniteTime();
this.taskManagerTimeout = TestingUtils.infiniteTime();
this.waitResultConsumedBeforeRelease = true;
this.defaultWorkerResourceSpec = WorkerResourceSpec.ZERO;
@@ -82,11 +80,6 @@ public class DeclarativeSlotManagerBuilder {
return this;
}
- public DeclarativeSlotManagerBuilder setSlotRequestTimeout(Time
slotRequestTimeout) {
- this.slotRequestTimeout = slotRequestTimeout;
- return this;
- }
-
public DeclarativeSlotManagerBuilder setTaskManagerTimeout(Time
taskManagerTimeout) {
this.taskManagerTimeout = taskManagerTimeout;
return this;
@@ -155,7 +148,6 @@ public class DeclarativeSlotManagerBuilder {
final SlotManagerConfiguration slotManagerConfiguration =
new SlotManagerConfiguration(
taskManagerRequestTimeout,
- slotRequestTimeout,
taskManagerTimeout,
requirementCheckDelay,
declareNeededResourceDelay,
diff --git
a/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfigurationBuilder.java
b/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfigurationBuilder.java
index 11c69282cb3..745ae5babb3 100644
---
a/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfigurationBuilder.java
+++
b/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfigurationBuilder.java
@@ -30,7 +30,6 @@ import java.time.Duration;
/** Builder for {@link SlotManagerConfiguration}. */
public class SlotManagerConfigurationBuilder {
private Time taskManagerRequestTimeout;
- private Time slotRequestTimeout;
private Time taskManagerTimeout;
private Duration requirementCheckDelay;
private Duration declareNeededResourceDelay;
@@ -45,7 +44,6 @@ public class SlotManagerConfigurationBuilder {
private SlotManagerConfigurationBuilder() {
this.taskManagerRequestTimeout = TestingUtils.infiniteTime();
- this.slotRequestTimeout = TestingUtils.infiniteTime();
this.taskManagerTimeout = TestingUtils.infiniteTime();
this.requirementCheckDelay =
ResourceManagerOptions.REQUIREMENTS_CHECK_DELAY.defaultValue();
this.declareNeededResourceDelay =
@@ -71,11 +69,6 @@ public class SlotManagerConfigurationBuilder {
return this;
}
- public SlotManagerConfigurationBuilder setSlotRequestTimeout(Time
slotRequestTimeout) {
- this.slotRequestTimeout = slotRequestTimeout;
- return this;
- }
-
public SlotManagerConfigurationBuilder setTaskManagerTimeout(Time
taskManagerTimeout) {
this.taskManagerTimeout = taskManagerTimeout;
return this;
@@ -138,7 +131,6 @@ public class SlotManagerConfigurationBuilder {
public SlotManagerConfiguration build() {
return new SlotManagerConfiguration(
taskManagerRequestTimeout,
- slotRequestTimeout,
taskManagerTimeout,
requirementCheckDelay,
declareNeededResourceDelay,
diff --git
a/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfigurationTest.java
b/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfigurationTest.java
index 136a48d2bad..65ef36645c6 100644
---
a/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfigurationTest.java
+++
b/flink-runtime/src/test/java/org/apache/flink/runtime/resourcemanager/slotmanager/SlotManagerConfigurationTest.java
@@ -19,7 +19,6 @@
package org.apache.flink.runtime.resourcemanager.slotmanager;
import org.apache.flink.configuration.Configuration;
-import org.apache.flink.configuration.JobManagerOptions;
import org.apache.flink.configuration.MemorySize;
import org.apache.flink.configuration.ResourceManagerOptions;
import org.apache.flink.runtime.resourcemanager.WorkerResourceSpec;
@@ -32,42 +31,6 @@ import static org.assertj.core.api.Assertions.assertThat;
/** Tests for {@link SlotManagerConfiguration}. */
class SlotManagerConfigurationTest {
-
- /**
- * Tests that {@link SlotManagerConfiguration#getSlotRequestTimeout()}
returns the value
- * configured under key {@link JobManagerOptions#SLOT_REQUEST_TIMEOUT}.
- */
- @Test
- void testSetSlotRequestTimeout() throws Exception {
- final long slotIdleTimeout = 42;
-
- final Configuration configuration = new Configuration();
- configuration.setLong(JobManagerOptions.SLOT_REQUEST_TIMEOUT,
slotIdleTimeout);
- final SlotManagerConfiguration slotManagerConfiguration =
- SlotManagerConfiguration.fromConfiguration(configuration,
WorkerResourceSpec.ZERO);
-
- assertThat(slotIdleTimeout)
-
.isEqualTo(slotManagerConfiguration.getSlotRequestTimeout().toMilliseconds());
- }
-
- /**
- * Tests that {@link ResourceManagerOptions#SLOT_REQUEST_TIMEOUT} is
preferred over {@link
- * JobManagerOptions#SLOT_REQUEST_TIMEOUT} if set.
- */
- @Test
- void testPreferLegacySlotRequestTimeout() throws Exception {
- final long legacySlotIdleTimeout = 42;
-
- final Configuration configuration = new Configuration();
- configuration.setLong(ResourceManagerOptions.SLOT_REQUEST_TIMEOUT,
legacySlotIdleTimeout);
- configuration.setLong(JobManagerOptions.SLOT_REQUEST_TIMEOUT, 300000L);
- final SlotManagerConfiguration slotManagerConfiguration =
- SlotManagerConfiguration.fromConfiguration(configuration,
WorkerResourceSpec.ZERO);
-
- assertThat(legacySlotIdleTimeout)
-
.isEqualTo(slotManagerConfiguration.getSlotRequestTimeout().toMilliseconds());
- }
-
@Test
void testComputeMaxTotalCpu() throws Exception {
final Configuration configuration = new Configuration();