Apache Flink 进阶(一):Runtime 核心机制剖析

本文涉及的产品
实时计算 Flink 版,1000CU*H 3个月
简介: Flink 的整体架构如图 1 所示。Flink 是可以运行在多种不同的环境中的,例如,它可以通过单进程多线程的方式直接运行,从而提供调试的能力。它也可以运行在 Yarn 或者 K8S 这种资源管理系统上面,也可以在各种云环境中执行。

作者:高赟(云骞)

1. 综述

本文主要介绍 Flink Runtime 的作业执行的核心机制。首先介绍 Flink Runtime 的整体架构以及 Job 的基本执行流程,然后介绍在这个过程,Flink 是怎么进行资源管理、作业调度以及错误恢复的。最后,本文还将简要介绍 Flink Runtime 层当前正在进行的一些工作。

2. Flink Runtime 整体架构

Flink 的整体架构如图 1 所示。Flink 是可以运行在多种不同的环境中的,例如,它可以通过单进程多线程的方式直接运行,从而提供调试的能力。它也可以运行在 Yarn 或者 K8S 这种资源管理系统上面,也可以在各种云环境中执行。

图片 1.png

图1. Flink 的整体架构,其中 Runtime 层对不同的执行环境提供了一套统一的分布式执行引擎。

针对不同的执行环境,Flink 提供了一套统一的分布式作业执行引擎,也就是 Flink Runtime 这层。Flink 在 Runtime 层之上提供了 DataStream 和 DataSet 两套 API,分别用来编写流作业与批作业,以及一组更高级的 API 来简化特定作业的编写。本文主要介绍 Flink Runtime 层的整体架构。

Flink Runtime 层的主要架构如图 2 所示,它展示了一个 Flink 集群的基本结构。Flink Runtime 层的整个架构主要是在 FLIP-6 中实现的,整体来说,它采用了标准 master-slave 的结构,其中左侧白色圈中的部分即是 master,它负责管理整个集群中的资源和作业;而右侧的两个 TaskExecutor 则是 Slave,负责提供具体的资源并实际执行作业。

图片 2.png

图2. Flink 集群的基本结构。Flink Runtime 层采用了标准的 master-slave 架构。

其中,Master 部分又包含了三个组件,即 Dispatcher、ResourceManager 和 JobManager。其中,Dispatcher 负责接收用户提供的作业,并且负责为这个新提交的作业拉起一个新的 JobManager 组件。ResourceManager 负责资源的管理,在整个 Flink 集群中只有一个 ResourceManager。JobManager 负责管理作业的执行,在一个 Flink 集群中可能有多个作业同时执行,每个作业都有自己的 JobManager 组件。这三个组件都包含在 AppMaster 进程中。

基于上述结构,当用户提交作业的时候,提交脚本会首先启动一个 Client进程负责作业的编译与提交。它首先将用户编写的代码编译为一个 JobGraph,在这个过程,它还会进行一些检查或优化等工作,例如判断哪些 Operator 可以 Chain 到同一个 Task 中。然后,Client 将产生的 JobGraph 提交到集群中执行。此时有两种情况,一种是类似于 Standalone 这种 Session 模式,AM 会预先启动,此时 Client 直接与 Dispatcher 建立连接并提交作业即可。另一种是 Per-Job 模式,AM 不会预先启动,此时 Client 将首先向资源管理系统 (如Yarn、K8S)申请资源来启动 AM,然后再向 AM 中的 Dispatcher 提交作业。

当作业到 Dispatcher 后,Dispatcher 会首先启动一个 JobManager 组件,然后 JobManager 会向 ResourceManager 申请资源来启动作业中具体的任务。这时根据 Session 和 Per-Job 模式的区别, TaskExecutor 可能已经启动或者尚未启动。如果是前者,此时 ResourceManager 中已有记录了 TaskExecutor 注册的资源,可以直接选取空闲资源进行分配。否则,ResourceManager 也需要首先向外部资源管理系统申请资源来启动 TaskExecutor,然后等待 TaskExecutor 注册相应资源后再继续选择空闲资源进程分配。目前 Flink 中 TaskExecutor 的资源是通过 Slot 来描述的,一个 Slot 一般可以执行一个具体的 Task,但在一些情况下也可以执行多个相关联的 Task,这部分内容将在下文进行详述。ResourceManager 选择到空闲的 Slot 之后,就会通知相应的 TM “将该 Slot 分配分 JobManager XX ”,然后 TaskExecutor 进行相应的记录后,会向 JobManager 进行注册。JobManager 收到 TaskExecutor 注册上来的 Slot 后,就可以实际提交 Task 了。

TaskExecutor 收到 JobManager 提交的 Task 之后,会启动一个新的线程来执行该 Task。Task 启动后就会开始进行预先指定的计算,并通过数据 Shuffle 模块互相交换数据。

以上就是 Flink Runtime 层执行作业的基本流程。可以看出,Flink 支持两种不同的模式,即 Per-job 模式与 Session 模式。如图 3 所示,Per-job 模式下整个 Flink 集群只执行单个作业,即每个作业会独享 Dispatcher 和 ResourceManager 组件。此外,Per-job 模式下 AppMaster 和 TaskExecutor 都是按需申请的。因此,Per-job 模式更适合运行执行时间较长的大作业,这些作业对稳定性要求较高,并且对申请资源的时间不敏感。与之对应,在 Session 模式下,Flink 预先启动 AppMaster 以及一组 TaskExecutor,然后在整个集群的生命周期中会执行多个作业。可以看出,Session 模式更适合规模小,执行时间短的作业。

图片 3.png

图3. Flink Runtime 支持两种作业执行的模式。

3. 资源管理与作业调度

本节对 Flink 中资源管理与作业调度的功能进行更深入的说明。实际上,作业调度可以看做是对资源和任务进行匹配的过程。如上节所述,在 Flink 中,资源是通过 Slot 来表示的,每个 Slot 可以用来执行不同的 Task。而在另一端,任务即 Job 中实际的 Task,它包含了待执行的用户逻辑。调度的主要目的就是为了给 Task 找到匹配的 Slot。逻辑上来说,每个 Slot 都应该有一个向量来描述它所能提供的各种资源的量,每个 Task 也需要相应的说明它所需要的各种资源的量。但是实际上在 1.9 之前,Flink 是不支持细粒度的资源描述的,而是统一的认为每个 Slot 提供的资源和 Task 需要的资源都是相同的。从 1.9 开始,Flink 开始增加对细粒度的资源匹配的支持的实现,但这部分功能目前仍在完善中。

作业调度的基础是首先提供对资源的管理,因此我们首先来看下 Flink 中资源管理的实现。如上文所述,Flink 中的资源是由 TaskExecutor 上的 Slot 来表示的。如图 4 所示,在 ResourceManager 中,有一个子组件叫做 SlotManager,它维护了当前集群中所有 TaskExecutor 上的 Slot 的信息与状态,如该 Slot 在哪个 TaskExecutor 中,该 Slot 当前是否空闲等。当 JobManger 来为特定 Task 申请资源的时候,根据当前是 Per-job 还是 Session 模式,ResourceManager 可能会去申请资源来启动新的 TaskExecutor。当 TaskExecutor 启动之后,它会通过服务发现找到当前活跃的 ResourceManager 并进行注册。在注册信息中,会包含该 TaskExecutor中所有 Slot 的信息。 ResourceManager 收到注册信息后,其中的 SlotManager 就会记录下相应的 Slot 信息。当 JobManager 为某个 Task 来申请资源时, SlotManager 就会从当前空闲的 Slot 中按一定规则选择一个空闲的 Slot 进行分配。当分配完成后,如第 2 节所述,RM 会首先向 TaskManager 发送 RPC 要求将选定的 Slot 分配给特定的 JobManager。TaskManager 如果还没有执行过该 JobManager 的 Task 的话,它需要首先向相应的 JobManager 建立连接,然后发送提供 Slot 的 RPC 请求。在 JobManager 中,所有 Task 的请求会缓存到 SlotPool 中。当有 Slot 被提供之后,SlotPool 会从缓存的请求中选择相应的请求并结束相应的请求过程。

图片 4.png

图4. Flink 中资源管理功能各模块交互关系。

当 Task 结束之后,无论是正常结束还是异常结束,都会通知 JobManager 相应的结束状态,然后在 TaskManager 端将 Slot 标记为已占用但未执行任务的状态。JobManager 会首先将相应的 Slot 缓存到 SlotPool 中,但不会立即释放。这种方式避免了如果将 Slot 直接还给 ResourceManager,在任务异常结束之后需要重启时,需要立刻重新申请 Slot 的问题。通过延时释放,Failover 的 Task 可以尽快调度回原来的 TaskManager,从而加快 Failover 的速度。当 SlotPool 中缓存的 Slot 超过指定的时间仍未使用时,SlotPool 就会发起释放该 Slot 的过程。与申请 Slot 的过程对应,SlotPool 会首先通知 TaskManager 来释放该 Slot,然后 TaskExecutor 通知 ResourceManager 该 Slot 已经被释放,从而最终完成释放的逻辑。

除了正常的通信逻辑外,在 ResourceManager 和 TaskExecutor 之间还存在定时的心跳消息来同步 Slot 的状态。在分布式系统中,消息的丢失、错乱不可避免,这些问题会在分布式系统的组件中引入不一致状态,如果没有定时消息,那么组件无法从这些不一致状态中恢复。此外,当组件之间长时间未收到对方的心跳时,就会认为对应的组件已经失效,并进入到 Failover 的流程。

在 Slot 管理基础上,Flink 可以将 Task 调度到相应的 Slot 当中。如上文所述,Flink 尚未完全引入细粒度的资源匹配,默认情况下,每个 Slot 可以分配给一个 Task。但是,这种方式在某些情况下会导致资源利用率不高。如图 5 所示,假如 A、B、C 依次执行计算逻辑,那么给 A、B、C 分配分配单独的 Slot 就会导致资源利用率不高。为了解决这一问题,Flink 提供了 Share Slot 的机制。如图 5 所示,基于 Share Slot,每个 Slot 中可以部署来自不同 JobVertex 的多个任务,但是不能部署来自同一个 JobVertex 的 Task。如图5所示,每个 Slot 中最多可以部署同一个 A、B 或 C 的 Task,但是可以同时部署 A、B 和 C 的各一个 Task。当单个 Task 占用资源较少时,Share Slot 可以提高资源利用率。 此外,Share Slot 也提供了一种简单的保持负载均衡的方式。

图片 5.png

图5.Flink Share Slot 示例。使用 Share Slot 可以在每个 Slot 中部署来自不同 JobVertex 的多个 Task。

基于上述 Slot 管理和分配的逻辑,JobManager 负责维护作业中 Task执行的状态。如上文所述,Client 端会向 JobManager 提交一个 JobGraph,它代表了作业的逻辑结构。JobManager 会根据 JobGraph 按并发展开,从而得到 JobManager 中关键的 ExecutionGraph。ExecutionGraph 的结构如图 5 所示,与 JobGraph 相比,ExecutionGraph 中对于每个 Task 与中间结果等均创建了对应的对象,从而可以维护这些实体的信息与状态。

图片 6.png

图6.Flink 中的 JobGraph 与 ExecutionGraph。ExecutionGraph 是 JobGraph 按并发展开所形成的,它是 JobMaster 中的核心数据结构。

在一个 Flink Job 中是包含多个 Task 的,因此另一个关键的问题是在 Flink 中按什么顺序来调度 Task。如图 7 所示,目前 Flink 提供了两种基本的调度逻辑,即 Eager 调度与 Lazy From Source。Eager 调度如其名子所示,它会在作业启动时申请资源将所有的 Task 调度起来。这种调度算法主要用来调度可能没有终止的流作业。与之对应,Lazy From Source 则是从 Source 开始,按拓扑顺序来进行调度。简单来说,Lazy From Source 会先调度没有上游任务的 Source 任务,当这些任务执行完成时,它会将输出数据缓存到内存或者写入到磁盘中。然后,对于后续的任务,当它的前驱任务全部执行完成后,Flink 就会将这些任务调度起来。这些任务会从读取上游缓存的输出数据进行自己的计算。这一过程继续进行直到所有的任务完成计算。

图片 7.png

图7. Flink 中两种基本的调度策略。其中 Eager 调度适用于流作业,而Lazy From Source 适用于批作业。

4. 错误恢复

在 Flink 作业的执行过程中,除正常执行的流程外,还有可能由于环境等原因导致各种类型的错误。整体上来说,错误可能分为两大类:Task 执行出现错误或 Flink 集群的 Master 出现错误。由于错误不可避免,为了提高可用性,Flink 需要提供自动错误恢复机制来进行重试。

对于第一类 Task 执行错误,Flink 提供了多种不同的错误恢复策略。如图 8 所示,第一种策略是 Restart-all,即直接重启所有的 Task。对于 Flink 的流任务,由于 Flink 提供了 Checkpoint 机制,因此当任务重启后可以直接从上次的 Checkpoint 开始继续执行。因此这种方式更适合于流作业。第二类错误恢复策略是 Restart-individual,它只适用于 Task 之间没有数据传输的情况。这种情况下,我们可以直接重启出错的任务。

图片 8.png

图8.Restart-all 错误恢复策略示例。该策略会直接重启所有的 Task。

图片 9.png

图9.Restart-individual 错误恢复策略示例。该策略只适用于 Task之间不需要数据传输的作业,对于这种作业可以只重启出现错误的 Task。

由于 Flink 的批作业没有 Checkpoint 机制,因此对于需要数据传输的作业,直接重启所有 Task 会导致作业从头计算,从而导致一定的性能问题。为了增强对 Batch 作业,Flink 在1.9中引入了一种新的Region-Based的Failover策略。在一个 Flink 的 Batch 作业中 Task 之间存在两种数据传输方式,一种是 Pipeline 类型的方式,这种方式上下游 Task 之间直接通过网络传输数据,因此需要上下游同时运行;另外一种是 Blocking 类型的试,如上节所述,这种方式下,上游的 Task 会首先将数据进行缓存,因此上下游的 Task 可以单独执行。基于这两种类型的传输,Flink 将 ExecutionGraph 中使用 Pipeline 方式传输数据的 Task 的子图叫做 Region,从而将整个 ExecutionGraph 划分为多个子图。可以看出,Region 内的 Task 必须同时重启,而不同 Region 的 Task 由于在 Region 边界存在 Blocking 的边,因此,可以单独重启下游 Region 中的 Task。

基于这一思路,如果某个 Region 中的某个 Task 执行出现错误,可以分两种情况进行考虑。如图 8 所示,如果是由于 Task 本身的问题发生错误,那么可以只重启该 Task 所属的 Region 中的 Task,这些 Task 重启之后,可以直接拉取上游 Region 缓存的输出结果继续进行计算。

另一方面,如图如果错误是由于读取上游结果出现问题,如网络连接中断、缓存上游输出数据的 TaskExecutor 异常退出等,那么还需要重启上游 Region 来重新产生相应的数据。在这种情况下,如果上游 Region 输出的数据分发方式不是确定性的(如 KeyBy、Broadcast 是确定性的分发方式,而 Rebalance、Random 则不是,因为每次执行会产生不同的分发结果),为保证结果正确性,还需要同时重启上游 Region 所有的下游 Region。

图片 10.png

图10.Region-based 错误恢复策略示例一。如果是由于下游任务本身导致的错误,可以只重启下游对应的 Region。

图片 11.png

图11.Region-based 错误恢复策略示例二。如果是由于上游失败导致的错误,那么需要同时重启上游的 Region 和下游的 Region。实际上,如果下游的输出使用了非确定的数据分割方式,为了保持数据一致性,还需要同时重启所有上游 Region 的下游 Region。

除了 Task 本身执行的异常外,另一类异常是 Flink 集群的 Master 进行发生异常。目前 Flink 支持启动多个 Master 作为备份,这些 Master 可以通过 ZK 来进行选主,从而保证某一时刻只有一个 Master 在运行。当前活路的 Master 发生异常时,某个备份的 Master 可以接管协调的工作。为了保证 Master 可以准确维护作业的状态,Flink 目前采用了一种最简单的实现方式,即直接重启整个作业。实际上,由于作业本身可能仍在正常运行,因此这种方式存在一定的改进空间。

5. 未来展望

Flink目前仍然在Runtime部分进行不断的迭代和更新。目前来看,Flink未来可能会在以下几个方式继续进行优化和扩展:

  • 更完善的资源管理:从 1.9 开始 Flink 开始了对细粒度资源匹配的支持。基于细粒度的资源匹配,用户可以为 TaskExecutor 和 Task 设置实际提供和使用的 CPU、内存等资源的数量,Flink 可以按照资源的使用情况进行调度。这一机制允许用户更大范围的控制作业的调度,从而为进一步提高资源利用率提供了基础。
  • 统一的 Stream 与 Batch:Flink 目前为流和批分别提供了 DataStream 和 DataSet 两套接口,在一些场景下会导致重复实现逻辑的问题。未来 Flink 会将流和批的接口都统一到 DataStream 之上。
  • 更灵活的调度策略:Flink 从 1.9 开始引入调度插件的支持,从而允许用户来扩展实现自己的调度逻辑。未来 Flink 也会提供更高性能的调度策略的实现。
  • Master Failover 的优化:如上节所述,目前 Flink 在 Master Failover 时需要重启整个作业,而实际上重启作业并不是必须的逻辑。Flink 未来会对 Master failover 进行进一步的优化来避免不必要的作业重启。

▼ Apache Flink 社区推荐 ▼

Apache Flink 及大数据领域顶级盛会 Flink Forward Asia 2019 重磅开启,目前正在征集议题,限量早鸟票优惠ing。了解 Flink Forward Asia 2019 的更多信息,请查看:

https://developerhtbprolaliyunhtbprolcom-s.evpn.library.nenu.edu.cn/special/ffa2019

首届 Apache Flink 极客挑战赛重磅开启,聚焦机器学习与性能优化两大热门领域,40万奖金等你拿,加入挑战请点击:

https://tianchihtbprolaliyunhtbprolcom-s.evpn.library.nenu.edu.cn/markets/tianchi/flink2019

相关实践学习
基于Hologres+Flink搭建GitHub实时数据大屏
通过使用Flink、Hologres构建实时数仓,并通过Hologres对接BI分析工具(以DataV为例),实现海量数据实时分析.
实时计算 Flink 实战课程
如何使用实时计算 Flink 搞定数据处理难题?实时计算 Flink 极客训练营产品、技术专家齐上阵,从开源 Flink功能介绍到实时计算 Flink 优势详解,现场实操,5天即可上手! 欢迎开通实时计算 Flink 版: https://cnhtbprolaliyunhtbprolcom-s.evpn.library.nenu.edu.cn/product/bigdata/sc Flink Forward Asia 介绍: Flink Forward 是由 Apache 官方授权,Apache Flink Community China 支持的会议,通过参会不仅可以了解到 Flink 社区的最新动态和发展计划,还可以了解到国内外一线大厂围绕 Flink 生态的生产实践经验,是 Flink 开发者和使用者不可错过的盛会。 去年经过品牌升级后的 Flink Forward Asia 吸引了超过2000人线下参与,一举成为国内最大的 Apache 顶级项目会议。结合2020年的特殊情况,Flink Forward Asia 2020 将在12月26日以线上峰会的形式与大家见面。
相关文章
|
23天前
|
人工智能 数据处理 API
阿里云、Ververica、Confluent 与 LinkedIn 携手推进流式创新,共筑基于 Apache Flink Agents 的智能体 AI 未来
Apache Flink Agents 是由阿里云、Ververica、Confluent 与 LinkedIn 联合推出的开源子项目,旨在基于 Flink 构建可扩展、事件驱动的生产级 AI 智能体框架,实现数据与智能的实时融合。
217 6
阿里云、Ververica、Confluent 与 LinkedIn 携手推进流式创新,共筑基于 Apache Flink Agents 的智能体 AI 未来
|
存储 Cloud Native 数据处理
从嵌入式状态管理到云原生架构:Apache Flink 的演进与下一代增量计算范式
本文整理自阿里云资深技术专家、Apache Flink PMC 成员梅源在 Flink Forward Asia 新加坡 2025上的分享,深入解析 Flink 状态管理系统的发展历程,从核心设计到 Flink 2.0 存算分离架构,并展望未来基于流批一体的通用增量计算方向。
217 0
从嵌入式状态管理到云原生架构:Apache Flink 的演进与下一代增量计算范式
|
2月前
|
人工智能 运维 Java
Flink Agents:基于Apache Flink的事件驱动AI智能体框架
本文基于Apache Flink PMC成员宋辛童在Community Over Code Asia 2025的演讲,深入解析Flink Agents项目的技术背景、架构设计与应用场景。该项目聚焦事件驱动型AI智能体,结合Flink的实时处理能力,推动AI在工业场景中的工程化落地,涵盖智能运维、直播分析等典型应用,展现其在AI发展第四层次——智能体AI中的重要意义。
906 27
Flink Agents:基于Apache Flink的事件驱动AI智能体框架
|
3月前
|
消息中间件 存储 Kafka
Apache Flink错误处理实战手册:2年生产环境调试经验总结
本文由 Ververica 客户成功经理 Naci Simsek 撰写,基于其在多个行业 Flink 项目中的实战经验,总结了 Apache Flink 生产环境中常见的三大典型问题及其解决方案。内容涵盖 Kafka 连接器迁移导致的状态管理问题、任务槽负载不均问题以及 Kryo 序列化引发的性能陷阱,旨在帮助企业开发者避免常见误区,提升实时流处理系统的稳定性与性能。
301 0
Apache Flink错误处理实战手册:2年生产环境调试经验总结
|
SQL 架构师 API
《Apache Flink 知其然,知其所以然》系列视频课程
# 课程简介 目前在我的公众号新推出了《Apache Flink 知其然,知其所以然》的系列视频课程。在内容上会先对Flink整体架构和所适用的场景做一个基础介绍,让你对Flink有一个整体的认识!然后对核心概念进行详细介绍,让你深入了解流计算中一些核心术语的含义,然后对Flink 各个层面的API,如 SQL/Table&DataStreamAPI/PythonAPI 进行详细的介绍,以及
1507 0
《Apache Flink 知其然,知其所以然》系列视频课程
|
3月前
|
SQL 人工智能 数据挖掘
Apache Flink:从实时数据分析到实时AI
Apache Flink 是实时数据处理领域的核心技术,历经十年发展,已从学术项目成长为实时计算的事实标准。它在现代数据架构中发挥着关键作用,支持实时数据分析、湖仓集成及实时 AI 应用。随着 Flink 2.0 的发布,其在流式湖仓、AI 驱动决策等方面展现出强大潜力,正推动企业迈向智能化、实时化的新阶段。
397 9
Apache Flink:从实时数据分析到实时AI
|
3月前
|
SQL 人工智能 API
Apache Flink 2.1.0: 面向实时 Data + AI 全面升级,开启智能流处理新纪元
Apache Flink 2.1.0 正式发布,标志着实时数据处理引擎向统一 Data + AI 平台迈进。新版本强化了实时 AI 能力,支持通过 Flink SQL 和 Table API 创建及调用 AI 模型,新增 Model DDL、ML_PREDICT 表值函数等功能,实现端到端的实时 AI 工作流。同时增强了 Flink SQL 的流处理能力,引入 Process Table Functions(PTFs)、Variant 数据类型,优化流式 Join 及状态管理,显著提升作业稳定性与资源利用率。
336 0
|
3月前
|
存储 人工智能 数据处理
对话王峰:Apache Flink 在 AI 时代的“剑锋”所向
Flink 2.0 架构升级实现存算分离,迈向彻底云原生化,支持更大规模状态管理、提升资源效率、增强容灾能力。通过流批一体与 AI 场景融合,推动实时计算向智能化演进。生态项目如 Paimon、Fluss 和 Flink CDC 构建湖流一体架构,实现分钟级时效性与低成本平衡。未来,Flink 将深化 AI Agents 框架,引领事件驱动的智能数据处理新方向。
336 6
|
11月前
|
存储 人工智能 大数据
The Past, Present and Future of Apache Flink
本文整理自阿里云开源大数据负责人王峰(莫问)在 Flink Forward Asia 2024 上海站主论坛开场的分享,今年正值 Flink 开源项目诞生的第 10 周年,借此时机,王峰回顾了 Flink 在过去 10 年的发展历程以及 Flink社区当前最新的技术成果,最后展望下一个十年 Flink 路向何方。
741 33
The Past, Present and Future of Apache Flink

相关产品

  • 实时计算 Flink版
  • 推荐镜像

    更多