Custom Resource Defintion Reference

This serves as a full reference for FlinkDeployment and FlinkSessionJob custom resource definitions, including all possible configuration parameters.

FlinkDeployment

Class: org.apache.flink.kubernetes.operator.crd.FlinkDeployment

Description: Custom resource that represents both Application and Session deployments.

ParameterTypeDocs
specorg.apache.flink.kubernetes.operator.crd.spec.FlinkDeploymentSpecSpec that describes a Flink application or session cluster deployment.
statusorg.apache.flink.kubernetes.operator.crd.status.FlinkDeploymentStatusLast observed status of the Flink deployment.

FlinkSessionJob

Class: org.apache.flink.kubernetes.operator.crd.FlinkSessionJob

Description: Custom resource that represents a session job deployment.

ParameterTypeDocs
specorg.apache.flink.kubernetes.operator.crd.spec.FlinkSessionJobSpecSpec that describes a session job deployment.
statusorg.apache.flink.kubernetes.operator.crd.status.FlinkSessionJobStatusLast observed status of the session job deployment.

Spec

CheckpointSpec

Class: org.apache.flink.kubernetes.operator.api.spec.CheckpointSpec

Description: Spec for checkpoint state snapshots. This is an empty class, used to instruct the operator to trigger a checkpoint.

ParameterTypeDocs

FlinkDeploymentSpec

Class: org.apache.flink.kubernetes.operator.api.spec.FlinkDeploymentSpec

Description: Spec that describes a Flink application or session cluster deployment.

ParameterTypeDocs
joborg.apache.flink.kubernetes.operator.api.spec.JobSpecJob specification for application deployments/session job. Null for session clusters.
restartNoncejava.lang.LongNonce used to manually trigger restart for the cluster/session job. In order to trigger restart, change the number to a different non-null value.
flinkConfigurationjava.util.Map<java.lang.String,java.lang.String>Flink configuration overrides for the Flink deployment or Flink session job.
imagejava.lang.StringFlink docker image used to start the Job and TaskManager pods.
imagePullPolicyjava.lang.StringImage pull policy of the Flink docker image.
serviceAccountjava.lang.StringKubernetes service used by the Flink deployment.
flinkVersionorg.apache.flink.kubernetes.operator.api.spec.FlinkVersionFlink image version.
ingressorg.apache.flink.kubernetes.operator.api.spec.IngressSpecIngress specs.
podTemplateio.fabric8.kubernetes.api.model.PodTemplateSpecBase pod template for job and task manager pods. Can be overridden by the jobManager and taskManager pod templates.
jobManagerorg.apache.flink.kubernetes.operator.api.spec.JobManagerSpecJobManager specs.
taskManagerorg.apache.flink.kubernetes.operator.api.spec.TaskManagerSpecTaskManager specs.
logConfigurationjava.util.Map<java.lang.String,java.lang.String>Log configuration overrides for the Flink deployment. Format logConfigFileName -> configContent.
modeorg.apache.flink.kubernetes.operator.api.spec.KubernetesDeploymentModeDeployment mode of the Flink cluster, native or standalone.

FlinkSessionJobSpec

Class: org.apache.flink.kubernetes.operator.api.spec.FlinkSessionJobSpec

Description: Spec that describes a Flink session job.

ParameterTypeDocs
joborg.apache.flink.kubernetes.operator.api.spec.JobSpecJob specification for application deployments/session job. Null for session clusters.
restartNoncejava.lang.LongNonce used to manually trigger restart for the cluster/session job. In order to trigger restart, change the number to a different non-null value.
flinkConfigurationjava.util.Map<java.lang.String,java.lang.String>Flink configuration overrides for the Flink deployment or Flink session job.
deploymentNamejava.lang.StringThe name of the target session cluster deployment.

FlinkStateSnapshotSpec

Class: org.apache.flink.kubernetes.operator.api.spec.FlinkStateSnapshotSpec

Description: Spec that describes a FlinkStateSnapshot.

ParameterTypeDocs
jobReferenceorg.apache.flink.kubernetes.operator.api.spec.JobReferenceSource to take a snapshot of. Not required if it’s a savepoint and alreadyExists is true.
savepointorg.apache.flink.kubernetes.operator.api.spec.SavepointSpecSpec in case of savepoint.
checkpointorg.apache.flink.kubernetes.operator.api.spec.CheckpointSpecSpec in case of checkpoint.
backoffLimitintMaximum number of retries before the snapshot is considered as failed. Set to -1 for unlimited or 0 for no retries.

FlinkVersion

Class: org.apache.flink.kubernetes.operator.api.spec.FlinkVersion

Description: Enumeration for supported Flink versions.

ValueDocs
v1_13No longer supported since 1.7 operator release.
v1_14No longer supported since 1.7 operator release.
v1_15Deprecated since 1.10 operator release.
v1_16
v1_17
v1_18
v1_19
v1_20

IngressSpec

Class: org.apache.flink.kubernetes.operator.api.spec.IngressSpec

Description: Ingress spec.

ParameterTypeDocs
templatejava.lang.StringIngress template for the JobManager service.
classNamejava.lang.StringIngress className for the Flink deployment.
annotationsjava.util.Map<java.lang.String,java.lang.String>Ingress annotations.
labelsjava.util.Map<java.lang.String,java.lang.String>Ingress labels.
tlsjava.util.List<io.fabric8.kubernetes.api.model.networking.v1.IngressTLS>Ingress tls.

JobKind

Class: org.apache.flink.kubernetes.operator.api.spec.JobKind

Description: Describes the Kubernetes kind of job reference.

ValueDocs
FlinkDeploymentFlinkDeployment CR kind.
FlinkSessionJobFlinkSessionJob CR kind.

JobManagerSpec

Class: org.apache.flink.kubernetes.operator.api.spec.JobManagerSpec

Description: JobManager spec.

ParameterTypeDocs
resourceorg.apache.flink.kubernetes.operator.api.spec.ResourceResource specification for the JobManager pods.
replicasintNumber of JobManager replicas. Must be 1 for non-HA deployments.
podTemplateio.fabric8.kubernetes.api.model.PodTemplateSpecJobManager pod template. It will be merged with FlinkDeploymentSpec.podTemplate.

JobReference

Class: org.apache.flink.kubernetes.operator.api.spec.JobReference

Description: Flink resource reference that can be a FlinkDeployment or FlinkSessionJob.

ParameterTypeDocs
kindorg.apache.flink.kubernetes.operator.api.spec.JobKindKind of the Flink resource, FlinkDeployment or FlinkSessionJob.
namejava.lang.StringName of the Flink resource.

JobSpec

Class: org.apache.flink.kubernetes.operator.api.spec.JobSpec

Description: Flink job spec.

ParameterTypeDocs
jarURIjava.lang.StringOptional URI of the job jar within the Flink docker container. For example: local:///opt/flink/examples/streaming/StateMachineExample.jar. If not specified the job jar should be available in the system classpath.
parallelismintParallelism of the Flink job.
entryClassjava.lang.StringFully qualified main class name of the Flink job.
argsjava.lang.String[]Arguments for the Flink job main class.
stateorg.apache.flink.kubernetes.operator.api.spec.JobStateDesired state for the job.
savepointTriggerNoncejava.lang.LongNonce used to manually trigger savepoint for the running job. In order to trigger a savepoint, change the number to a different non-null value.
initialSavepointPathjava.lang.StringSavepoint path used by the job the first time it is deployed or during savepoint redeployments (triggered by changing the savepointRedeployNonce).
checkpointTriggerNoncejava.lang.LongNonce used to manually trigger checkpoint for the running job. In order to trigger a checkpoint, change the number to a different non-null value.
upgradeModeorg.apache.flink.kubernetes.operator.api.spec.UpgradeModeUpgrade mode of the Flink job.
allowNonRestoredStatejava.lang.BooleanAllow checkpoint state that cannot be mapped to any job vertex in tasks.
savepointRedeployNoncejava.lang.LongNonce used to trigger a full redeployment of the job from the savepoint path specified in initialSavepointPath. In order to trigger redeployment, change the number to a different non-null value. Rollback is not possible after redeployment.

JobState

Class: org.apache.flink.kubernetes.operator.api.spec.JobState

Description: Enum describing the desired job state.

ValueDocs
runningJob is expected to be processing data.
suspendedProcessing is suspended with the intention of continuing later.

KubernetesDeploymentMode

Class: org.apache.flink.kubernetes.operator.api.spec.KubernetesDeploymentMode

Description: Enum to control Flink deployment mode on Kubernetes.

ValueDocs
nativeDeploys Flink using Flinks native Kubernetes support. Only supported for newer versions of Flink
standaloneDeploys Flink on-top of kubernetes in standalone mode.

Resource

Class: org.apache.flink.kubernetes.operator.api.spec.Resource

Description: Resource spec.

ParameterTypeDocs
cpujava.lang.DoubleAmount of CPU allocated to the pod.
memoryjava.lang.StringAmount of memory allocated to the pod. Example: 1024m, 1g
ephemeralStoragejava.lang.StringAmount of ephemeral storage allocated to the pod. Example: 1024m, 2G

SavepointSpec

Class: org.apache.flink.kubernetes.operator.api.spec.SavepointSpec

Description: Spec for savepoint state snapshots.

ParameterTypeDocs
pathjava.lang.StringOptional path for the savepoint.
formatTypeorg.apache.flink.kubernetes.operator.api.status.SavepointFormatTypeSavepoint format to use.
disposeOnDeletejava.lang.BooleanDispose the savepoints upon CR deletion.
alreadyExistsjava.lang.BooleanIndicates that the savepoint already exists on the given path. The Operator will not trigger any new savepoints, just update the status of the resource as a completed snapshot.

TaskManagerSpec

Class: org.apache.flink.kubernetes.operator.api.spec.TaskManagerSpec

Description: TaskManager spec.

ParameterTypeDocs
resourceorg.apache.flink.kubernetes.operator.api.spec.ResourceResource specification for the TaskManager pods.
replicasjava.lang.IntegerNumber of TaskManager replicas. If defined, takes precedence over parallelism
podTemplateio.fabric8.kubernetes.api.model.PodTemplateSpecTaskManager pod template. It will be merged with FlinkDeploymentSpec.podTemplate.

UpgradeMode

Class: org.apache.flink.kubernetes.operator.api.spec.UpgradeMode

Description: Enum to control Flink job upgrade behavior.

ValueDocs
savepointJob is upgraded by first taking a savepoint of the running job, shutting it down and restoring from the savepoint.
last-stateJob is upgraded using any latest checkpoint or savepoint available.
statelessJob is upgraded with empty state.

Status

Checkpoint

Class: org.apache.flink.kubernetes.operator.api.status.Checkpoint

Description: Represents information about a finished checkpoint.

ParameterTypeDocs
timeStamplongMillisecond timestamp at the start of the checkpoint operation.
triggerTypeorg.apache.flink.kubernetes.operator.api.status.SnapshotTriggerTypeCheckpoint trigger mechanism.
formatTypeorg.apache.flink.kubernetes.operator.api.status.CheckpointTypeCheckpoint format.
triggerNoncejava.lang.LongNonce value used when the checkpoint was triggered manually {@link SnapshotTriggerType#MANUAL}, null for other types of checkpoint.

CheckpointInfo

Class: org.apache.flink.kubernetes.operator.api.status.CheckpointInfo

Description: Stores checkpoint-related information.

ParameterTypeDocs
lastCheckpointorg.apache.flink.kubernetes.operator.api.status.CheckpointLast completed checkpoint by the operator.
triggerIdjava.lang.StringTrigger id of a pending checkpoint operation.
triggerTimestampjava.lang.LongTrigger timestamp of a pending checkpoint operation.
triggerTypeorg.apache.flink.kubernetes.operator.api.status.SnapshotTriggerTypeCheckpoint trigger mechanism.
formatTypeorg.apache.flink.kubernetes.operator.api.status.CheckpointTypeCheckpoint format.
lastPeriodicCheckpointTimestamplongTrigger timestamp of last periodic checkpoint operation.

CheckpointType

Class: org.apache.flink.kubernetes.operator.api.status.CheckpointType

Description: Checkpoint format type.

ValueDocs
FULL
INCREMENTAL
UNKNOWNCheckpoint format unknown, if the checkpoint was not triggered by the operator.
descriptionorg.apache.flink.configuration.description.InlineElement

FlinkDeploymentReconciliationStatus

Class: org.apache.flink.kubernetes.operator.api.status.FlinkDeploymentReconciliationStatus

Description: Status of the last reconcile step for the flink deployment.

ParameterTypeDocs
reconciliationTimestamplongEpoch timestamp of the last successful reconcile operation.
lastReconciledSpecjava.lang.StringLast reconciled deployment spec. Used to decide whether further reconciliation steps are necessary.
lastStableSpecjava.lang.StringLast stable deployment spec according to the specified stability condition. If a rollback strategy is defined this will be the target to roll back to.
stateorg.apache.flink.kubernetes.operator.api.status.ReconciliationStateDeployment state of the last reconciled spec.

FlinkDeploymentStatus

Class: org.apache.flink.kubernetes.operator.api.status.FlinkDeploymentStatus

Description: Last observed status of the Flink deployment.

ParameterTypeDocs
jobStatusorg.apache.flink.kubernetes.operator.api.status.JobStatusLast observed status of the Flink job on Application/Session cluster.
errorjava.lang.StringError information about the FlinkDeployment/FlinkSessionJob.
observedGenerationjava.lang.LongLast observed generation of the FlinkDeployment/FlinkSessionJob.
lifecycleStateorg.apache.flink.kubernetes.operator.api.lifecycle.ResourceLifecycleStateLifecycle state of the Flink resource (including being rolled back, failed etc.).
clusterInfojava.util.Map<java.lang.String,java.lang.String>Information from running clusters.
jobManagerDeploymentStatusorg.apache.flink.kubernetes.operator.api.status.JobManagerDeploymentStatusLast observed status of the JobManager deployment.
reconciliationStatusorg.apache.flink.kubernetes.operator.api.status.FlinkDeploymentReconciliationStatusStatus of the last reconcile operation.
taskManagerorg.apache.flink.kubernetes.operator.api.status.TaskManagerInfoInformation about the TaskManagers for the scale subresource.

FlinkSessionJobReconciliationStatus

Class: org.apache.flink.kubernetes.operator.api.status.FlinkSessionJobReconciliationStatus

Description: Status of the last reconcile step for the flink sessionjob.

ParameterTypeDocs
reconciliationTimestamplongEpoch timestamp of the last successful reconcile operation.
lastReconciledSpecjava.lang.StringLast reconciled deployment spec. Used to decide whether further reconciliation steps are necessary.
lastStableSpecjava.lang.StringLast stable deployment spec according to the specified stability condition. If a rollback strategy is defined this will be the target to roll back to.
stateorg.apache.flink.kubernetes.operator.api.status.ReconciliationStateDeployment state of the last reconciled spec.

FlinkSessionJobStatus

Class: org.apache.flink.kubernetes.operator.api.status.FlinkSessionJobStatus

Description: Last observed status of the Flink Session job.

ParameterTypeDocs
jobStatusorg.apache.flink.kubernetes.operator.api.status.JobStatusLast observed status of the Flink job on Application/Session cluster.
errorjava.lang.StringError information about the FlinkDeployment/FlinkSessionJob.
observedGenerationjava.lang.LongLast observed generation of the FlinkDeployment/FlinkSessionJob.
lifecycleStateorg.apache.flink.kubernetes.operator.api.lifecycle.ResourceLifecycleStateLifecycle state of the Flink resource (including being rolled back, failed etc.).
reconciliationStatusorg.apache.flink.kubernetes.operator.api.status.FlinkSessionJobReconciliationStatusStatus of the last reconcile operation.

FlinkStateSnapshotStatus

Class: org.apache.flink.kubernetes.operator.api.status.FlinkStateSnapshotStatus

Description: Last observed status of the Flink state snapshot.

ParameterTypeDocs

State

Class: org.apache.flink.kubernetes.operator.api.status.FlinkStateSnapshotStatus.State

Description: Describes state of a snapshot.

ValueDocs
COMPLETEDSnapshot was successful and available.
FAILEDError during snapshot.
IN_PROGRESSSnapshot in progress.
TRIGGER_PENDINGNot yet processed by the operator.
ABANDONEDSnapshot abandoned due to job failure/upgrade.
stateorg.apache.flink.kubernetes.operator.api.status.FlinkStateSnapshotStatus.State
triggerIdjava.lang.String
triggerTimestampjava.lang.String
resultTimestampjava.lang.String
pathjava.lang.String
errorjava.lang.String
failuresint

State

Class: org.apache.flink.kubernetes.operator.api.status.FlinkStateSnapshotStatus.State

Description: Describes state of a snapshot.

ValueDocs
COMPLETEDSnapshot was successful and available.
FAILEDError during snapshot.
IN_PROGRESSSnapshot in progress.
TRIGGER_PENDINGNot yet processed by the operator.
ABANDONEDSnapshot abandoned due to job failure/upgrade.

JobManagerDeploymentStatus

Class: org.apache.flink.kubernetes.operator.api.status.JobManagerDeploymentStatus

Description: Status of the Flink JobManager Kubernetes deployment.

ValueDocs
READYJobManager is running and ready to receive REST API calls.
DEPLOYED_NOT_READYJobManager is running but not ready yet to receive REST API calls.
DEPLOYINGJobManager process is starting up.
MISSINGJobManager deployment not found, probably not started or killed by user.
ERRORDeployment in terminal error, requires spec change for reconciliation to continue.

JobStatus

Class: org.apache.flink.kubernetes.operator.api.status.JobStatus

Description: Last observed status of the Flink job within an application deployment.

ParameterTypeDocs
jobNamejava.lang.StringName of the job.
jobIdjava.lang.StringFlink JobId of the Job.
stateorg.apache.flink.api.common.JobStatusLast observed state of the job.
startTimejava.lang.StringStart time of the job.
updateTimejava.lang.StringUpdate time of the job.
upgradeSavepointPathjava.lang.String
savepointInfoorg.apache.flink.kubernetes.operator.api.status.SavepointInfoInformation about pending and last savepoint for the job.
checkpointInfoorg.apache.flink.kubernetes.operator.api.status.CheckpointInfoInformation about pending and last checkpoint for the job.

ReconciliationState

Class: org.apache.flink.kubernetes.operator.api.status.ReconciliationState

Description: Current state of the reconciliation.

ValueDocs
DEPLOYEDThe lastReconciledSpec is currently deployed.
UPGRADINGThe spec is being upgraded.
ROLLING_BACKIn the process of rolling back to the lastStableSpec.
ROLLED_BACKRolled back to the lastStableSpec.

Savepoint

Class: org.apache.flink.kubernetes.operator.api.status.Savepoint

Description: Represents information about a finished savepoint.

ParameterTypeDocs
timeStamplongMillisecond timestamp at the start of the savepoint operation.
locationjava.lang.StringExternal pointer of the savepoint can be used to recover jobs.
triggerTypeorg.apache.flink.kubernetes.operator.api.status.SnapshotTriggerTypeSavepoint trigger mechanism.
formatTypeorg.apache.flink.kubernetes.operator.api.status.SavepointFormatTypeSavepoint format.
triggerNoncejava.lang.LongNonce value used when the savepoint was triggered manually {@link SnapshotTriggerType#MANUAL}, null for other types of savepoints.

SavepointFormatType

Class: org.apache.flink.kubernetes.operator.api.status.SavepointFormatType

Description: Savepoint format type.

ValueDocs
CANONICALA canonical, common for all state backends format.
NATIVEA format specific for the chosen state backend.
UNKNOWNSavepoint format unknown, if the savepoint was not triggered by the operator.

SavepointInfo

Class: org.apache.flink.kubernetes.operator.api.status.SavepointInfo

Description: Stores savepoint related information.

ParameterTypeDocs
lastSavepointorg.apache.flink.kubernetes.operator.api.status.SavepointLast completed savepoint by the operator.
triggerIdjava.lang.StringTrigger id of a pending savepoint operation.
triggerTimestampjava.lang.LongTrigger timestamp of a pending savepoint operation.
triggerTypeorg.apache.flink.kubernetes.operator.api.status.SnapshotTriggerTypeSavepoint trigger mechanism.
formatTypeorg.apache.flink.kubernetes.operator.api.status.SavepointFormatTypeSavepoint format.
savepointHistoryjava.util.List<org.apache.flink.kubernetes.operator.api.status.Savepoint>List of recent savepoints.
lastPeriodicSavepointTimestamplongTrigger timestamp of last periodic savepoint operation.

SnapshotTriggerType

Class: org.apache.flink.kubernetes.operator.api.status.SnapshotTriggerType

Description: Snapshot trigger mechanism.

ValueDocs
MANUALSnapshot manually triggered by changing a triggerNonce.
PERIODICSnapshot periodically triggered by the operator.
UPGRADESnapshot triggered during stateful upgrade.
UNKNOWNSnapshot trigger mechanism unknown, such as savepoint retrieved directly from Flink job.

TaskManagerInfo

Class: org.apache.flink.kubernetes.operator.api.status.TaskManagerInfo

Description: Last observed status of the Flink job within an application deployment.

ParameterTypeDocs
labelSelectorjava.lang.StringTaskManager label selector.
replicasintNumber of TaskManager replicas if defined in the spec.