From 93021109db4bdb6dc6db9464319b13250a792de9 Mon Sep 17 00:00:00 2001 From: Paul Lin Date: Sat, 9 Sep 2023 14:09:07 +0800 Subject: [PATCH] [KYUUBI #5264] [FLINK] Force disable Flink's session timeout ### _Why are the changes needed?_ Flink sessions are now managed by Kyuubi, hence disable session timeout from Flink itself. ### _How was this patch tested?_ - [ ] Add some test cases that check the changes thoroughly including negative and positive cases if possible - [ ] Add screenshots for manual tests if appropriate - [x] [Run test](https://kyuubi.readthedocs.io/en/master/contributing/code/testing.html#running-tests) locally before make a pull request ### _Was this patch authored or co-authored using generative AI tooling?_ No. Closes #5264 from link3280/disable_flink_session_timeout. Closes #5264 fff5c54d7 [Paul Lin] Force disable Flink's session timeout Authored-by: Paul Lin Signed-off-by: Paul Lin --- .../org/apache/kyuubi/engine/flink/FlinkSQLEngine.scala | 7 +++++++ 1 file changed, 7 insertions(+) diff --git a/externals/kyuubi-flink-sql-engine/src/main/scala/org/apache/kyuubi/engine/flink/FlinkSQLEngine.scala b/externals/kyuubi-flink-sql-engine/src/main/scala/org/apache/kyuubi/engine/flink/FlinkSQLEngine.scala index 7e4f31f8a..8838799bc 100644 --- a/externals/kyuubi-flink-sql-engine/src/main/scala/org/apache/kyuubi/engine/flink/FlinkSQLEngine.scala +++ b/externals/kyuubi-flink-sql-engine/src/main/scala/org/apache/kyuubi/engine/flink/FlinkSQLEngine.scala @@ -19,12 +19,14 @@ package org.apache.kyuubi.engine.flink import java.io.File import java.nio.file.Paths +import java.time.Duration import java.util.concurrent.CountDownLatch import scala.collection.JavaConverters._ import org.apache.flink.configuration.{Configuration, DeploymentOptions, GlobalConfiguration, PipelineOptions} import org.apache.flink.table.api.TableEnvironment +import org.apache.flink.table.gateway.api.config.SqlGatewayServiceConfigOptions import org.apache.flink.table.gateway.service.context.DefaultContext import org.apache.kyuubi.{Logging, Utils} @@ -146,6 +148,11 @@ object FlinkSQLEngine extends Logging { kyuubiConf.getOption(KYUUBI_SESSION_USER_KEY) .foreach(flinkConf.setString(KYUUBI_SESSION_USER_KEY, _)) + // force disable Flink's session timeout + flinkConf.set( + SqlGatewayServiceConfigOptions.SQL_GATEWAY_SESSION_IDLE_TIMEOUT, + Duration.ofMillis(0)) + executionTarget match { case "yarn-per-job" | "yarn-application" => if (flinkConf.containsKey("high-availability.cluster-id")) {