This is an automated email from the ASF dual-hosted git repository.
rmetzger pushed a commit to branch release-1.13
in repository https://gitbox.apache.org/repos/asf/flink.git
The following commit(s) were added to refs/heads/release-1.13 by this push:
new be7a590 [FLINK-22495][docs] Add Reactive Mode section to K8s
be7a590 is described below
commit be7a590e05bd091caefbe2bb0a51b20107b79493
Author: Robert Metzger <[email protected]>
AuthorDate: Wed Apr 28 11:45:49 2021 +0200
[FLINK-22495][docs] Add Reactive Mode section to K8s
---
.../resource-providers/standalone/kubernetes.md | 81 +++++++++++++++++++++-
.../resource-providers/standalone/kubernetes.md | 81 +++++++++++++++++++++-
2 files changed, 160 insertions(+), 2 deletions(-)
diff --git
a/docs/content.zh/docs/deployment/resource-providers/standalone/kubernetes.md
b/docs/content.zh/docs/deployment/resource-providers/standalone/kubernetes.md
index 84c53ac..b9f51ce 100644
---
a/docs/content.zh/docs/deployment/resource-providers/standalone/kubernetes.md
+++
b/docs/content.zh/docs/deployment/resource-providers/standalone/kubernetes.md
@@ -95,7 +95,7 @@ You can tear down the cluster using the following commands:
### Deploy Application Cluster
-A *Flink Application cluster* is a dedicated cluster which runs a single
application.
+A *Flink Application cluster* is a dedicated cluster which runs a single
application, which needs to be available at deployment time.
A basic *Flink Application cluster* deployment in Kubernetes has three
components:
@@ -233,6 +233,15 @@ You can access the queryable state of TaskManager if you
create a `NodePort` ser
1. Run `kubectl create -f taskmanager-query-state-service.yaml` to create
the `NodePort` service for the `taskmanager` pod. The example of
`taskmanager-query-state-service.yaml` can be found in
[appendix](#common-cluster-resource-definitions).
2. Run `kubectl get svc flink-taskmanager-query-state` to get the
`<node-port>` of this service. Then you can create the
[QueryableStateClient(<public-node-ip>, <node-port>]({{< ref
"docs/dev/datastream/fault-tolerance/queryable_state" >}}#querying-state) to
submit state queries.
+### Using Standalone Kubernetes with Reactive Mode
+
+[Reactive Mode]({{< ref "docs/deployment/elastic_scaling" >}}#reactive-mode)
allows to run Flink in a mode, where the *Application Cluster* is always
adjusting the job parallelism to the available resources. In combination with
Kubernetes, the replica count of the TaskManager deployment determines the
available resources. Increasing the replica count will scale up the job,
reducing it will trigger a scale down. This can also be done automatically by
using a [Horizontal Pod Autoscaler](ht [...]
+
+To use Reactive Mode on Kubernetes, follow the same steps as for [deploying a
job using an Application Cluster](#deploy-application-cluster). But instead of
`flink-configuration-configmap.yaml` use this config map:
`flink-reactive-mode-configuration-configmap.yaml`. It contains the
`scheduler-mode: reactive` setting for Flink.
+
+Once you have deployed the *Application Cluster*, you can scale your job up or
down by changing the replica count in the `flink-taskmanager` deployment.
+
+
{{< top >}}
## Appendix
@@ -305,6 +314,76 @@ data:
logger.netty.level = OFF
```
+
+`flink-reactive-mode-configuration-configmap.yaml`
+
+```yaml
+apiVersion: v1
+kind: ConfigMap
+metadata:
+ name: flink-config
+ labels:
+ app: flink
+data:
+ flink-conf.yaml: |+
+ jobmanager.rpc.address: flink-jobmanager
+ taskmanager.numberOfTaskSlots: 2
+ blob.server.port: 6124
+ jobmanager.rpc.port: 6123
+ taskmanager.rpc.port: 6122
+ queryable-state.proxy.ports: 6125
+ jobmanager.memory.process.size: 1600m
+ taskmanager.memory.process.size: 1728m
+ parallelism.default: 2
+ scheduler-mode: reactive
+ execution.checkpointing.interval: 10s
+ log4j-console.properties: |+
+ # This affects logging for both user code and Flink
+ rootLogger.level = INFO
+ rootLogger.appenderRef.console.ref = ConsoleAppender
+ rootLogger.appenderRef.rolling.ref = RollingFileAppender
+
+ # Uncomment this if you want to _only_ change Flink's logging
+ #logger.flink.name = org.apache.flink
+ #logger.flink.level = INFO
+
+ # The following lines keep the log level of common libraries/connectors on
+ # log level INFO. The root logger does not override this. You have to
manually
+ # change the log levels here.
+ logger.akka.name = akka
+ logger.akka.level = INFO
+ logger.kafka.name= org.apache.kafka
+ logger.kafka.level = INFO
+ logger.hadoop.name = org.apache.hadoop
+ logger.hadoop.level = INFO
+ logger.zookeeper.name = org.apache.zookeeper
+ logger.zookeeper.level = INFO
+
+ # Log all infos to the console
+ appender.console.name = ConsoleAppender
+ appender.console.type = CONSOLE
+ appender.console.layout.type = PatternLayout
+ appender.console.layout.pattern = %d{yyyy-MM-dd HH:mm:ss,SSS} %-5p %-60c
%x - %m%n
+
+ # Log all infos in the given rolling file
+ appender.rolling.name = RollingFileAppender
+ appender.rolling.type = RollingFile
+ appender.rolling.append = false
+ appender.rolling.fileName = ${sys:log.file}
+ appender.rolling.filePattern = ${sys:log.file}.%i
+ appender.rolling.layout.type = PatternLayout
+ appender.rolling.layout.pattern = %d{yyyy-MM-dd HH:mm:ss,SSS} %-5p %-60c
%x - %m%n
+ appender.rolling.policies.type = Policies
+ appender.rolling.policies.size.type = SizeBasedTriggeringPolicy
+ appender.rolling.policies.size.size=100MB
+ appender.rolling.strategy.type = DefaultRolloverStrategy
+ appender.rolling.strategy.max = 10
+
+ # Suppress the irrelevant (wrong) warnings from the Netty channel handler
+ logger.netty.name =
org.apache.flink.shaded.akka.org.jboss.netty.channel.DefaultChannelPipeline
+ logger.netty.level = OFF
+```
+
`jobmanager-service.yaml` Optional service, which is only necessary for non-HA
mode.
```yaml
apiVersion: v1
diff --git
a/docs/content/docs/deployment/resource-providers/standalone/kubernetes.md
b/docs/content/docs/deployment/resource-providers/standalone/kubernetes.md
index 5fa36f6..98c5d9b 100644
--- a/docs/content/docs/deployment/resource-providers/standalone/kubernetes.md
+++ b/docs/content/docs/deployment/resource-providers/standalone/kubernetes.md
@@ -95,7 +95,7 @@ You can tear down the cluster using the following commands:
### Deploy Application Cluster
-A *Flink Application cluster* is a dedicated cluster which runs a single
application.
+A *Flink Application cluster* is a dedicated cluster which runs a single
application, which needs to be available at deployment time.
A basic *Flink Application cluster* deployment in Kubernetes has three
components:
@@ -233,6 +233,15 @@ You can access the queryable state of TaskManager if you
create a `NodePort` ser
1. Run `kubectl create -f taskmanager-query-state-service.yaml` to create
the `NodePort` service for the `taskmanager` pod. The example of
`taskmanager-query-state-service.yaml` can be found in
[appendix](#common-cluster-resource-definitions).
2. Run `kubectl get svc flink-taskmanager-query-state` to get the
`<node-port>` of this service. Then you can create the
[QueryableStateClient(<public-node-ip>, <node-port>]({{< ref
"docs/dev/datastream/fault-tolerance/queryable_state" >}}#querying-state) to
submit state queries.
+### Using Standalone Kubernetes with Reactive Mode
+
+[Reactive Mode]({{< ref "docs/deployment/elastic_scaling" >}}#reactive-mode)
allows to run Flink in a mode, where the *Application Cluster* is always
adjusting the job parallelism to the available resources. In combination with
Kubernetes, the replica count of the TaskManager deployment determines the
available resources. Increasing the replica count will scale up the job,
reducing it will trigger a scale down. This can also be done automatically by
using a [Horizontal Pod Autoscaler](ht [...]
+
+To use Reactive Mode on Kubernetes, follow the same steps as for [deploying a
job using an Application Cluster](#deploy-application-cluster). But instead of
`flink-configuration-configmap.yaml` use this config map:
`flink-reactive-mode-configuration-configmap.yaml`. It contains the
`scheduler-mode: reactive` setting for Flink.
+
+Once you have deployed the *Application Cluster*, you can scale your job up or
down by changing the replica count in the `flink-taskmanager` deployment.
+
+
{{< top >}}
## Appendix
@@ -305,6 +314,76 @@ data:
logger.netty.level = OFF
```
+
+`flink-reactive-mode-configuration-configmap.yaml`
+
+```yaml
+apiVersion: v1
+kind: ConfigMap
+metadata:
+ name: flink-config
+ labels:
+ app: flink
+data:
+ flink-conf.yaml: |+
+ jobmanager.rpc.address: flink-jobmanager
+ taskmanager.numberOfTaskSlots: 2
+ blob.server.port: 6124
+ jobmanager.rpc.port: 6123
+ taskmanager.rpc.port: 6122
+ queryable-state.proxy.ports: 6125
+ jobmanager.memory.process.size: 1600m
+ taskmanager.memory.process.size: 1728m
+ parallelism.default: 2
+ scheduler-mode: reactive
+ execution.checkpointing.interval: 10s
+ log4j-console.properties: |+
+ # This affects logging for both user code and Flink
+ rootLogger.level = INFO
+ rootLogger.appenderRef.console.ref = ConsoleAppender
+ rootLogger.appenderRef.rolling.ref = RollingFileAppender
+
+ # Uncomment this if you want to _only_ change Flink's logging
+ #logger.flink.name = org.apache.flink
+ #logger.flink.level = INFO
+
+ # The following lines keep the log level of common libraries/connectors on
+ # log level INFO. The root logger does not override this. You have to
manually
+ # change the log levels here.
+ logger.akka.name = akka
+ logger.akka.level = INFO
+ logger.kafka.name= org.apache.kafka
+ logger.kafka.level = INFO
+ logger.hadoop.name = org.apache.hadoop
+ logger.hadoop.level = INFO
+ logger.zookeeper.name = org.apache.zookeeper
+ logger.zookeeper.level = INFO
+
+ # Log all infos to the console
+ appender.console.name = ConsoleAppender
+ appender.console.type = CONSOLE
+ appender.console.layout.type = PatternLayout
+ appender.console.layout.pattern = %d{yyyy-MM-dd HH:mm:ss,SSS} %-5p %-60c
%x - %m%n
+
+ # Log all infos in the given rolling file
+ appender.rolling.name = RollingFileAppender
+ appender.rolling.type = RollingFile
+ appender.rolling.append = false
+ appender.rolling.fileName = ${sys:log.file}
+ appender.rolling.filePattern = ${sys:log.file}.%i
+ appender.rolling.layout.type = PatternLayout
+ appender.rolling.layout.pattern = %d{yyyy-MM-dd HH:mm:ss,SSS} %-5p %-60c
%x - %m%n
+ appender.rolling.policies.type = Policies
+ appender.rolling.policies.size.type = SizeBasedTriggeringPolicy
+ appender.rolling.policies.size.size=100MB
+ appender.rolling.strategy.type = DefaultRolloverStrategy
+ appender.rolling.strategy.max = 10
+
+ # Suppress the irrelevant (wrong) warnings from the Netty channel handler
+ logger.netty.name =
org.apache.flink.shaded.akka.org.jboss.netty.channel.DefaultChannelPipeline
+ logger.netty.level = OFF
+```
+
`jobmanager-service.yaml` Optional service, which is only necessary for non-HA
mode.
```yaml
apiVersion: v1