[源码分析] 从源码入手看 Flink Watermark 之传播过程 --- 上
0x00 摘要
本文将通过源码分析,带领大家熟悉Flink Watermark 之传播过程,顺便也可以对Flink整体逻辑有一个大致把握。
因为篇幅所限,分为上下两篇,本文是上篇,介绍总体传播过程。
0x01 总述
从静态角度讲,watermarks是实现流式计算的核心概念;从动态角度说,watermarks贯穿整个流处理程序。所以为了讲解watermarks的传播,需要对flink的很多模块/概念进行了解,涉及几乎各个阶段。我首先会讲解相关概念,然后会根据一个实例代码从以下几部分来解释:程序逻辑/计算图模型/程序执行。最后是详细Flink源码分析(略冗长,可以选择性阅读)。
0x02 相关概念
流计算被抽象成四个问题,what,where,when,how。
window解决的是where,也就是将无界数据划分成有界数据。
window的数据何时被计算是when?解决这个问题用的方式是watermark和trigger,watermark用来标记窗口的完整性。trigger用来设计窗口数据触发条件。
1. 乱序处理
乱序问题一般是和event time关联的, 对于一个流式处理系统的process time来说,是不存在乱序问题的。所以下面介绍的watermark/allowedLateness也只是在event time作为主时间才生效。
Flink中处理乱序依赖的 watermark+window+trigger,属于全局性的处理;Flink同时对于window而言,还提供了allowedLateness方法,使得更大限度的允许乱序,属于局部性的处理;
即watermark是全局的,不止针对window计算,而allowedLateness让某一个特定window函数能自己控制处理延迟数据的策略,allowedLateness是窗口函数的属性。
2. Watermark(水位线)
watermark是流式系统中主要用于解决流式系统中数据乱序问题的机制,方法是用于标记当前处理到什么水位的数据了,这意味着再早于这个水位的数据过来会被直接丢弃。这使得引擎可以自动跟踪数据中的当前事件时间,并尝试相应地清除旧状态。
Watermarking表示多长时间以前的数据将不再更新,您可以通过指定事件时间列来定义查询的Watermarking,并根据事件时间预测数据的延迟时间。也就是说每次窗口滑动之前会进行Watermarking的计算。当一组数据或新接收的数据事件时间小于Watermarking时,则该数据不会更新,在内存中就不会维护该组数据的状态。
换一种说法,阈值内的滞后数据将被聚合,但是晚于阈值到来的数据(其实际时间比watermark小)将被丢弃。
watermark和数据本身一样作为正常的消息在流中流动。
3. Trigger
Trigger 指明在哪些条件下触发window计算,基于处理数据时的时间以及事件的特定属性。一般trigger的实现是当watermark处于某种时间条件下或者窗口数据达到一定条件,窗口的数据开始计算。
每个窗口分配器都会有一个默认的Trigger。如果默认的Trigger不能满足你的需求,你可以指定一个自定义的trigger()。Flink Trigger接口有如下方法允许trigger对不同的事件做出反应:
* onElement():进入窗口的每个元素都会调用该方法。
* onEventTime():事件时间timer触发的时候被调用。
* onProcessingTime():处理时间timer触发的时候会被调用。
* onMerge():有状态的触发器相关,并在它们相应的窗口合并时合并两个触发器的状态,例如使用会话窗口。
* clear():该方法主要是执行窗口的删除操作。
每次trigger,都是要对新增的数据,相关的window进行重新计算,并输出。输出有complete, append,update三种输出模式:
-
Complete mode:Result Table 全量输出,也就是重新计算过的window结果都输出。意味着这种模式下,每次读了新增的input数据,output的时候会把内存中resulttable中所有window的结果都输出一遍。
-
Append mode (default):只有 Result Table 中新增的行才会被输出,所谓新增是指自上一次 trigger 的时候。因为只是输出新增的行,所以如果老数据有改动就不适合使用这种模式。 更新的window并不输出,否则外存里的key就重了。
-
Update mode:只要更新的 Row 都会被输出,相当于 Append mode 的加强版。而且是对外存中的相同key进行update,而不是append,需要外存是能kv操作的!只会输出新增和更新过的window的结果。
从上面能看出来,流式框架对于window的结果数据是存在一个 result table里的!
4. allowedLateness
Flink中借助watermark以及window和trigger来处理基于event time的乱序问题,那么如何处理“late element”呢?
也许还有人会问,out-of-order element与late element有什么区别?不都是一回事么?答案是一回事,都是为了处理乱序问题而产生的概念。要说区别,可以总结如下:
- 通过watermark机制来处理out-of-order的问题,属于第一层防护,属于全局性的防护,通常说的乱序问题的解决办法,就是指这类;
- 通过窗口上的allowedLateness机制来处理out-of-order的问题,属于第二层防护,属于特定window operator的防护,late element的问题就是指这类。
默认情况下,当watermark通过end-of-window之后,再有之前的数据到达时,这些数据会被删除。为了避免有些迟到的数据被删除,因此产生了allowedLateness的概念。
简单来讲,allowedLateness就是针对event time而言,对于watermark超过end-of-window之后,还允许有一段时间(也是以event time来衡量)来等待之前的数据到达,以便再次处理这些数据。
5. 处理消息过程
- windowoperator接到消息以后,首先存到state,存放的格式为k,v,key的格式是key + window,value是key和window对应的数据。
- 注册一个timer,timer的数据结构为 [key,window,window边界 - 1],将timer放到集合中去。
- 当windowoperator收到watermark以后,取出集合中小于watermark的timer,触发其window。触发的过程中将state里面对应key及window的数据取出来,这里要经过序列化的过程,发送给windowfunction计算。
- 数据发送给windowfunction,实现windowfunction的window数据计算逻辑。
比如某窗口有三个数据:[key A, window A, 0], [key A, window A, 4999], [key A, window A, 5000]
对于固定窗口,当第一个watermark (Watermark 5000)到达时候,[key A, window A, 0], [key A, window A, 4999] 会被计算,当第二个watermark (Watermark 9999)到达时候,[key A, window A, 5000]会被计算。
6. 累加(再次)计算
watermark是全局性的参数,用于管理消息的乱序,watermark超过window的endtime之后,就会触发窗口计算。一般情况下,触发窗口计算之后,窗口就销毁掉了,后面再来的数据也不会再计算。
因为加入了allowedLateness,所以计算会和之前不同了。window这个allowedLateness属性,默认为0,如果allowedLateness > 0,那么在某一个特定watermark到来之前,这个触发过计算的窗口还会继续保留,这个保留主要是窗口里的消息。
这个特定的watermark是什么呢? watermark-allowedLateness>=窗口endtime。这个特定watermark来了之后,窗口就要消失了,后面再来属于这个窗口的消息,就丢掉了。在 "watermark(=窗口endtime)" ~ “watermark(=endtime+allowedLateness)" 这段时间之间,对应窗口可能会多次计算。那么要window的endtime+allowedLateness <= watermark的时候,window才会被清掉。
比如window的endtime是5000,allowedLateness=0,那么如果watermark 5000到来之后,这个window就应该被清除。但是如果allowedLateness = 1000,则需要等water 6000(endtime + allowedLateness)到来之后,这个window才会被清掉。
Flink的allowedLateness可用于TumblingEventTimeWindow、SlidingEventTimeWindow以及EventTimeSessionWindows,这可能使得窗口再次被触发,相当于对前一次窗口的窗口的修正(累加计算或者累加撤回计算);
注意:对于trigger是默认的EventTimeTrigger的情况下,allowedLateness会再次触发窗口的计算,而之前触发的数据,会buffer起来,直到watermark超过end-of-window + allowedLateness的时间,窗口的数据及元数据信息才会被删除。再次计算就是DataFlow模型中的Accumulating的情况。
同时,对于sessionWindow的情况,当late element在allowedLateness范围之内到达时,可能会引起窗口的merge,这样之前窗口的数据会在新窗口中累加计算,这就是DataFlow模型中的AccumulatingAndRetracting的情况。
7. Watermark传播
生产任务的pipeline中通常有多个stage,在源头产生的watermark会在pipeline的多个stage间传递。了解watermark如何在一个pipeline的多个stage间进行传递,可以更好的了解watermark对整个pipeline的影响,以及对pipeline结果延时的影响。我们在pipeline的各stage的边界上对watermark做如下定义:
- 输入watermark(An input watermark):捕捉上游各阶段数据处理进度。对源头算子,input watermark是个特殊的function,对进入的数据产生watermark。对非源头算子,input watermark是上游stage中,所有shard/partition/instance产生的最小的watermark
- 输出watermark(An output watermark):捕捉本stage的数据进度,实质上指本stage中,所有input watermark的最小值,和本stage中所有非late event的数据的event time。比如,该stage中,被缓存起来等待做聚合的数据等。
每个stage内的操作并不是线性递增的。概念上,每个stage的操作都可以被分为几个组件(components),每个组件都会影响pipeline的输出watermark。每个组件的特性与具体的实现方式和包含的算子相关。理论上,这类算子会缓存数据,直到触发某个计算。比如缓存一部分数据并将其存入状态(state)中,直到触发聚合计算,并将计算结果写入下游stage。
watermark可以是以下项的最小值:
- 每个source的watermark(Per-source watermark) - 每个发送数据的stage.
- 每个外部数据源的watermark(Per-external input watermark) - pipeline之外的数据源
- 每个状态组件的watermark(Per-state component watermark) - 每种需要写入的state类型
- 每个输出buffer的watermark(Per-output buffer watermark) - 每个接收stage
这种精度的watermark能够更好的描述系统内部状态。能够更简单的跟踪数据在系统各个buffer中的流转状态,有助于排查数据堵塞问题。
watermark以广播的形式在算子之间传播,当一个算子收到watermark时都要干些什么事情呢?
- 更新算子时间
- 遍历计时器队列触发回调
- 将watermark发送到下游
0x03. Flink 程序结构 & 核心概念
1. 程序结构
Flink程序像常规的程序一样对数据集合进行转换操作,每个程序由下面几部分组成:
- 获取一个执行环境
- 加载/创建初始化数据
- 指定对于数据的transformations操作
- 指定计算的输出结果(打印或者输出到文件)
- 触发程序执行
flink流式计算的核心概念,就是将数据从输入流一个个传递给Operator进行链式处理,最后交给输出流的过程。对数据的每一次处理在逻辑上成为一个operator,并且为了本地化处理的效率起见,operator之间也可以串成一个chain一起处理。
下面这张图表明了flink是如何看待用户的处理流程的:用户操作被抽象化为一系列operator。以source开始,以sink结尾,中间的operator做的操作叫做transform,并且可以把几个操作串在一起执行。
Source ---> Transformation ----> Transformation ----> Sink
以下是一个样例代码,后续的分析会基于此代码。
DataStream<String> text = env.socketTextStream(hostname, port);
DataStream counts = text
.filter(new FilterClass())
.map(new LineSplitter())
.assignTimestampsAndWatermarks(new BoundedOutOfOrdernessGenerator)
.keyBy(0)
.timeWindow(Time.seconds(10))
.sum(2)
counts.print()
System.out.println(env.getExecutionPlan());
2. 核心类/接口
在用户设计程序时候,对应如下核心类/接口:
- DataStream:描述的是一个具有相同数据类型的数据流,底层是通过具体的Transformation来实现,其负责提供各种对流上的数据进行操作转换的API接口。
- Transformation:描述了构建一个DataStream的操作,以及该操作的并行度、输出数据类型等信息,并有一个属性,用来持有StreamOperator的一个具体实例;
上述代码逻辑中,对数据流做了如下操作:filter, map, keyBy, assignTimestampsAndWatermarks, timeWindow, sum。每次转换都生成了一个新的DataStream。
比如实例代码中的timeWindow最后生成了windowedStream。windowedStream之上执行的apply方法会生成了WindowOperator,初始化时包含了trigger以及allowedLateness的值。然后经过transform转换,实际上是执行了DataStream中的transform方法,最后生成了SingleOutputStreamOperator。SingleOutputStreamOperator这个类名字有点误导,实际上它是DataStream的子类。
public <R> SingleOutputStreamOperator<R> apply(WindowFunction<T, R, K, W> function, TypeInformation<R> resultType) {
KeySelector<T, K> keySel = input.getKeySelector(); //根据keyedStream获取key
WindowOperator<K, T, Iterable<T>, R, W> operator;
operator = new WindowOperator<>(windowAssigner, ... ,
new InternalIterableWindowFunction<>(function),
trigger,
allowedLateness,
legacyWindowOpType);
return input.transform(opName, resultType, operator);//根据operator name,窗口函数的类型,以及window operator,执行keyedStream.transaform操作
}
0x04. Flink 执行图模型
Flink 中的执行图可以分成四层:StreamGraph ---> JobGraph ---> ExecutionGraph -> 物理执行图。
- StreamGraph:是对用户逻辑的映射,代表程序的拓扑结构,是根据用户通过 Stream API 编写的代码生成的最初的图。
- JobGraph:StreamGraph经过优化后生成了 JobGraph,提交给 JobManager 的数据结构。主要的优化为,将多个符合条件的节点 chain 在一起作为一个节点,这样可以减少数据在节点之间流动所需要的序列化/反序列化/传输消耗。
- ExecutionGraph:JobManager 根据 JobGraph 生成ExecutionGraph。ExecutionGraph是JobGraph的并行化版本,是调度层最核心的数据结构。
- 物理执行图:JobManager 根据 ExecutionGraph 对 Job 进行调度后,在各个TaskManager 上部署 Task 后形成的“图”,并不是一个具体的数据结构。
我们这里重点看StreamGraph,其相关重点数据结构是:
- StreamNode 是用来描述 operator 的逻辑节点,并具有所有相关的属性,如并发度、入边和出边等。
- StreamEdge 是用来描述两个 StreamNode(operator) 逻辑的链接边。
我们可以直接打印 Execution Plan
System.out.println(env.getExecutionPlan());
其内部调用 StreamExecutionEnvironment.getExecutionPlan 得到 StreamGraph。
public String getExecutionPlan() {
return getStreamGraph(DEFAULT_JOB_NAME, false).getStreamingPlanAsJSON();
}
StreamGraph的转换流是:
* Source --> Filter --> Map --> Timestamps/Watermarks --> Window(SumAggregator) --> Sink
下面是我把 示例代码 打印StreamGraph结果整理出来一个静态架构。可以看出代码中的转换被翻译成了如下执行Unit(在下面图中,其执行序列是由上而下)。
* +-----> Data Source(ID = 1) [ Source Socket Stream ]
* | // env.socketTextStream(hostname, port) 方法中生成了一个 Data Source
* |
* +-----> Operator(ID = 2) [ Filter ]
* |
* |
* +-----> Operator(ID = 3) [ Map ]
* |
* |
* +-----> Operator(ID = 4) [ Timestamps/Watermarks ]
* |
* |
* +-----> Operator(ID = 6) [ Window(SumAggregator) ]
* | // 多个Operator被构建成 Operator Chain
* |
* |
* +-----> Data Sink(ID = 7) [ Sink : Print to Std. Out ]
* // counts.print() 是在数据流最后添加了个 Data Sink,用于承接统计结果
示例代码中,Flink生成StreamGraph的大致处理流程是:
- 首先处理的
Source,生成了Source的StreamNode。 - 处理
Filter,生成了Filter的StreamNode,并生成StreamEdge连接上游Source和Filter。 - 处理
Map,生成了Map的StreamNode,并生成StreamEdge连接上游Filter和Map。 - 处理
assignTimestampsAndWatermarks,生成了Timestamps/Watermarks的StreamNode,并生成StreamEdge连接上游Map和Timestamps/Watermarks。 - 处理
keyBy/timeWindow/sum,生成了Window的StreamNode以及Operator Chain,并生成StreamEdge连接上游Timestamps/Watermarks和Window。 - 最后处理
Sink,创建Sink的StreamNode,并生成StreamEdge与上游Window相连。
0x05. 执行模块生命周期
这里主要核心类是:
-
Function:用户通过继承该接口的不同子类来实现用户自己的数据处理逻辑。如子类SocketTextStreamFunction实现从指定hostname和port来接收数据,并转发字符串的逻辑;
-
Task: 是Flink中执行的基本单位,代表一个 TaskManager 中所起的并行子任务,执行封装的 flink 算子并运行,提供以下服务:消费输入data、生产 IntermediateResultPartition [ flink关于中间结果的抽象 ]、与 JobManager 交互。
-
StreamTask : 是本地执行的基本单位,由TaskManagers部署执行。包含了多个StreamOperator,封装了算子的处理逻辑。
-
StreamOperator:DataStream 上的每一个 Transformation 都对应了一个 StreamOperator,StreamOperator是运行时的具体实现,会决定UDF(User-Defined Funtion)的调用方式。
-
StreamSource 是StreamOperator接口的一个具体实现类,其构造函数入参就是SourceFunction的子类,这里就是SocketTextStreamFunction的实例。
Task 是直接受 TaskManager 管理和调度的,而 Task 又会调用 StreamTask(主要是其各种子类),StreamTask 中封装了算子(StreamOperator)的处理逻辑。StreamSource是用来开启整个流的算子。我们接下来就说说动态逻辑。
我们的示例代码中,所有程序逻辑都是运行在StreamTask(主要是其各种子类)中,filter/map对应了StreamOperator;assignTimestampsAndWatermarks用来生成Watermarks,传递给下游的.keyBy.timeWindow(WindowOperator)。而keyBy/timeWindow/sum又被构建成OperatorChain。所以我们下面就逐一讲解这些概念。
1. Task
Task,它是在线程中执行的Runable对象,每个Task都是由一组Operators Chaining在一起的工作集合,Flink Job的执行过程可看作一张DAG图,Task是DAG图上的顶点(Vertex),顶点之间通过数据传递方式相互链接构成整个Job的Execution Graph。
Task 是直接受 TaskManager 管理和调度的,Flink最后通过RPC方法提交task,实际会调用到TaskExecutor.submitTask方法中。这个方法会创建真正的Task,然后调用task.startTaskThread();开始task的执行。而startTaskThread方法,则会执行executingThread.start,从而调用Task.run方法。
它的最核心的代码如下:
* public class Task implements Runnable...
* The Task represents one execution of a parallel subtask on a TaskManager.
* A Task wraps a Flink operator (which may be a user function) and runs it
*
* -- doRun()
* |
* +----> 从 NetworkEnvironment 中申请 BufferPool
* | 包括 InputGate 的接收 pool 以及 task 的每个 ResultPartition 的输出 pool
* +----> invokable = loadAndInstantiateInvokable(userCodeClassLoader,
* | nameOfInvokableClass) 通过反射创建
* | load and instantiate the task's invokable code
* | invokable即为operator对象实例,例如OneInputStreamTask,SourceStreamTask等
* | OneInputStreamTask继承了StreamTask,这里实际调用的invoke()方法是StreamTask里的
* +----> invokable.invoke()
* | run the invokable,
* |
* |
* OneInputStreamTask<IN,OUT> extends StreamTask<OUT,OneInputStreamOperator<IN, OUT>>
这个nameOfInvokableClass是哪里生成的呢?其实早在生成StreamGraph的时候,这就已经确定了,见StreamGraph.addOperator方法
if (operatorObject instanceof StoppableStreamSource) {
addNode(vertexID, slotSharingGroup, StoppableSourceStreamTask.class, operatorObject, operatorName);
} else if (operatorObject instanceof StreamSource) {
addNode(vertexID, slotSharingGroup, SourceStreamTask.class, operatorObject, operatorName);
} else {
addNode(vertexID, slotSharingGroup, OneInputStreamTask.class, operatorObject, operatorName);
}
这里的OneInputStreamTask.class即为生成的StreamNode的vertexClass。这个值会一直传递
StreamGraph --> JobVertex.invokableClass --> ExecutionJobVertex.TaskInformation.invokableClassName --> Task
2. StreamTask
是本地执行的基本单位,由TaskManagers部署执行,Task会调用 StreamTask。StreamTask包含了headOperator 和 operatorChain,封装了算子的处理逻辑。可以理解为,StreamTask是执行流程框架,OperatorChain(StreamOperator)是负责具体算子逻辑,嵌入到StreamTask的执行流程框架中。
直接从StreamTask的注释中,能看到StreamTask的生命周期。
其中,每个operator的open()方法都被StreamTask的openAllOperators()方法调用。该方法(指openAllOperators)执行所有的operational的初始化,例如使用定时器服务注册定时器。单个task可能正在执行多个operator,消耗其前驱的输出,在这种情况下,该open()方法在最后一个operator中调用,即这个operator的输出也是task本身的输出。这样做使得当第一个operator开始处理任务的输入时,它的所有下游operator都准备好接收其输出。
OperatorChain是在StreamTask的invoke方法中被创建的,在执行的时候,如果一个operator无法被chain起来,那它就只有headOperator,chain里就没有其他operator了。
注意: task中的连续operator是从最后到第一个依次open。
以OneInputStreamTask为例,Task的核心执行代码即为OneInputStreamTask.invoke方法,它会调用StreamTask.invoke方法。
* The life cycle of the task(StreamTask) is set up as follows:
* {@code
* -- setInitialState -> provides state of all operators in the chain
* |
* +----> 重新初始化task的state,并且在如下两种情况下尤为重要:
* | 1. 当任务从故障中恢复并从最后一个成功的checkpoint点重新启动时
* | 2. 从一个保存点恢复时。
* -- invoke()
* |
* +----> Create basic utils (config, etc) and load the chain of operators
* +----> operators.setup() //创建 operatorChain 并设置为 headOperator 的 Output
* --------> openAllOperators()
* +----> task specific init()
* +----> initialize-operator-states()
* +----> open-operators() //执行 operatorChain 中所有 operator 的 open 方法
* +----> run() //runMailboxLoop()方法将一直运行,直到没有更多的输入数据
* --------> mailboxProcessor.runMailboxLoop();
* --------> StreamTask.processInput()
* --------> StreamTask.inputProcessor.processInput()
* --------> 间接调用 operator的processElement()和processWatermark()方法
* +----> close-operators() //执行 operatorChain 中所有 operator 的 close 方法
* +----> dispose-operators()
* +----> common cleanup
* +----> task specific cleanup()
* }
3. OneInputStreamTask
OneInputStreamTask是 StreamTask 的实现类之一,具有代表性。我们示例代码中基本都是由OneInputStreamTask来做具体执行。
看看OneInputStreamTask 是如何生成的?
* 生成StreamNode时候
*
* -- StreamGraph.addOperator()
* |
* +----> addNode(... OneInputStreamTask.class, operatorObject, operatorName);
* | 将 OneInputStreamTask 等 StreamTask 设置到 StreamNode 的节点属性中
*
*
* 在 JobVertex 的节点构造时也会做一次初始化
* |
* +----> jobVertex.setInvokableClass(streamNode.getJobVertexClass());
后续在 TaskDeploymentDescriptor 实例化的时候会获取 jobVertex 中的属性。
再看看OneInputStreamTask 的 init() 和run() 分别都做了什么
* OneInputStreamTask
* class OneInputStreamTask<IN,OUT> extends StreamTask<OUT,OneInputStreamOperator<IN, OUT>> * {@code
* -- init方法
* |
* +----> 获取算子对应的输入序列化器 TypeSerializer
* +----> CheckpointedInputGate inputGate = createCheckpointedInputGate();
* 获取输入数据 InputGate[],InputGate 是 flink 网络传输的核心抽象之一
* 其在内部封装了消息的接收和内存的管理,从 InputGate 可以拿到上游传送过来的数据
* +----> inputProcessor = new StreamOneInputProcessor<>(input,output,operatorChain)
* | 1. StreamInputProcessor,是 StreamTask 内部用来处理 Record 的组件,
* | 里面封装了外部 IO 逻辑【内存不够时将 buffer 吐到磁盘上】以及 时间对齐逻辑【Watermark】
* | 2. output 是 StreamTaskNetworkOutput, input是StreamTaskNetworkInput
* | 这样就把input, output 他俩聚合进StreamOneInputProcessor
* +----> headOperator.getMetricGroup().gauge
* +----> getEnvironment().getMetricGroup().gauge
* 设置一些 metrics 及 累加器
*
*
* -- run方法(就是基类StreamTask.run)
* +----> StreamTask.runMailboxLoop
* | 从 StreamTask.runMailboxLoop 开始,下面是一层层的调用关系
* -----> StreamTask.processInput()
* -----> StreamTask.inputProcessor.processInput()
* -----> StreamOneInputProcessor.processInput
* -----> input.emitNext(output)
* -----> StreamTaskNetworkInput.emitNext()
* | while(true) {从输入source读取一个record, output是 StreamTaskNetworkOutput}
* -----> StreamTaskNetworkInput.processElement() //具体处理record
* | 根据StreamElement的不同类型做不同处理
* | if (recordOrMark.isRecord()) output.emitRecord()
* ------------> StreamTaskNetworkOutput.emitRecord()
* ----------------> operator.processElement(record)
* | if (recordOrMark.isWatermark()) statusWatermarkValve.inputWatermark()
* | if (recordOrMark.isLatencyMarker()) output.emitLatencyMarker()
* | if (recordOrMark.isStreamStatus()) statusWatermarkValve.inputStreamStatus()
4. OperatorChain
flink 中的一个 operator 代表一个最顶级的 api 接口,拿 streaming 来说就是,在 DataStream 上做诸如 map/reduce/keyBy 等操作均会生成一个算子。
Operator Chain是指在生成JobGraph阶段,将Job中的Operators按照一定策略(例如:single output operator可以chain在一起)链接起来并放置在一个Task线程中执行。减少了数据传递/线程切换等环节,降低系统开销的同时增加了资源利用率和Job性能。
chained operators实际上是从下游往上游去反向一个个创建和setup的。假设chained operators为:StreamGroupedReduce - StreamFilter - StreamSink,而实际初始化顺序则相反:StreamSink - StreamFilter - StreamGroupedReduce。
* OperatorChain(
* StreamTask<OUT, OP> containingTask,
* RecordWriterDelegate<SerializationDelegate<StreamRecord<OUT>>> recordWriterDelegate)
* {@code
* -- collect
* |
* +----> pushToOperator(StreamRecord<X> record)
* +---------> operator.processElement(castRecord);
* //这里的operator是chainedOperator,即除了headOperator之外,剩余的operators的chain。
* //这个operator.processElement,会循环调用operator chain所有operator,直到chain end。
* //比如 Operator A 对应的 ChainingOutput collect 调用了对应的算子 A 的 processElement 方法,这里又会调用 B 的 ChainingOutput 的 collect 方法,以此类推。这样便实现了可 chain 算子的本地处理,最终经由网络输出 RecordWriterOutput 发送到下游节点。
5. StreamOperator
StreamTask会调用Operator,所以我们需要看看Operator的生命周期。
逻辑算子Transformation最后会对应到物理算子Operator,这个概念对应的就是StreamOperator。
StreamOperator是根接口。对于 Streaming 来说所有的算子都继承自 StreamOperator。继承了StreamOperator的扩展接口则有OneInputStreamOperator,TwoInputStreamOperator。实现了StreamOperator的抽象类有AbstractStreamOperator以及它的子类AbstractStreamUdfOperator。
其中operator处理输入的数据(elements)可以是以下之一:input element,watermark和checkpoint barriers。他们中的每一个都有一个特殊的单元来处理。element由processElement()方法处理,watermark由processWatermark()处理,checkpoint barriers由异步调用的snapshotState()方法处理,此方法会触发一次checkpoint 。
processElement()方法也是UDF的逻辑被调用的地方,例如MapFunction里的map()方法。
* AbstractUdfStreamOperator, which is the basic class for all operators that execute UDFs.
*
* // initialization phase
* //初始化operator-specific方法,如RuntimeContext和metric collection
* OPERATOR::setup
* UDF::setRuntimeContext
* //setup的调用链是invoke(StreamTask) -> constructor(OperatorChain) -> setup
* //调用setup时,StreamTask已经在各个TaskManager节点上
* //给出一个用来初始state的operator
*
* OPERATOR::initializeState
* //执行所有operator-specific的初始化
* OPERATOR::open
* UDF::open
*
* // processing phase (called on every element/watermark)
* OPERATOR::processElement
* UDF::run //给定一个operator可以有一个用户定义的函数(UDF)
* OPERATOR::processWatermark
*
* // checkpointing phase (called asynchronously on every checkpoint)
* OPERATOR::snapshotState
*
* // termination phase
* OPERATOR::close
* UDF::close
* OPERATOR::dispose
OneInputStreamOperator与TwoInputStreamOperator接口。这两个接口非常类似,本质上就是处理流上存在的三种元素StreamRecord,Watermark和LatencyMarker。一个用作单流输入,一个用作双流输入。
6. StreamSource
StreamSource是用来开启整个流的算子(继承AbstractUdfStreamOperator)。StreamSource因为没有输入,所以没有实现InputStreamOperator的接口。比较特殊的是ChainingStrategy初始化为HEAD。
在StreamSource这个类中,在运行时由SourceStreamTask调用SourceFunction的run方法来启动source。
* class StreamSource<OUT, SRC extends SourceFunction<OUT>>
* extends AbstractUdfStreamOperator<OUT, SRC> implements StreamOperator<OUT>
*
*
* -- run()
* |
* +----> latencyEmitter = new LatencyMarksEmitter
* | 用来产生延迟监控的LatencyMarker
* +----> this.ctx = StreamSourceContexts.getSourceContext
* | 据时间模式(EventTime/IngestionTime/ProcessingTime)生成相应SourceConext
* | 包含了产生element关联的timestamp的方法和生成watermark的方法
* +----> userFunction.run(ctx);
* | 调用SourceFunction的run方法来启动source,进行数据的转发
*
public {
//读到数据后,把数据交给collect方法,collect方法负责把数据交到合适的位置(如发布为br变量,或者交给下个operator,或者通过网络发出去)
private transient SourceFunction.SourceContext<OUT> ctx;
private transient volatile boolean canceledOrStopped = false;
private transient volatile boolean hasSentMaxWatermark = false;
public void run(final Object lockingObject,
final StreamStatusMaintainer streamStatusMaintainer,
final Output<StreamRecord<OUT>> collector,
final OperatorChain<?, ?> operatorChain) throws Exception {
userFunction.run(ctx);
}
}
7. StreamMap
StreamFilter,StreamMap与StreamFlatMap算子在实现的processElement分别调用传入的FilterFunction,MapFunction, FlatMapFunction的udf将element传到下游。这里用StreamMap举例:
public class StreamMap<IN, OUT>
extends AbstractUdfStreamOperator<OUT, MapFunction<IN, OUT>>
implements OneInputStreamOperator<IN, OUT> {
public StreamMap(MapFunction<IN, OUT> mapper) {
super(mapper);
chainingStrategy = ChainingStrategy.ALWAYS;
}
@Override
public void processElement(StreamRecord<IN> element) throws Exception {
output.collect(element.replace(userFunction.map(element.getValue())));
}
}
8. WindowOperator
Flink通过水位线分配器(TimestampsAndPeriodicWatermarksOperator和TimestampsAndPunctuatedWatermarksOperator这两个算子)向事件流中注入水位线。
我们示例代码中,timeWindow()最终对应了WindowStream,窗口算子WindowOperator是窗口机制的底层实现。assignTimestampsAndWatermarks 则对应了TimestampsAndPeriodicWatermarksOperator算子,它把产生的Watermark传递给了WindowOperator。
元素在streaming dataflow引擎中流动到WindowOperator时,会被分为两拨,分别是普通事件和水位线。
-
如果是普通的事件,则会调用processElement方法进行处理,在processElement方法中,首先会利用窗口分配器为当前接收到的元素分配窗口,接着会调用触发器的onElement方法进行逐元素触发。对于时间相关的触发器,通常会注册事件时间或者处理时间定时器,这些定时器会被存储在WindowOperator的处理时间定时器队列和水位线定时器队列中,如果触发的结果是FIRE,则对窗口进行计算。
-
如果是水位线(事件时间场景),则方法processWatermark将会被调用,它将会处理水位线定时器队列中的定时器。如果时间戳满足条件,则利用触发器的onEventTime方法进行处理。
而对于处理时间的场景,WindowOperator将自身实现为一个基于处理时间的触发器,以触发trigger方法来消费处理时间定时器队列中的定时器满足条件则会调用窗口触发器的onProcessingTime,根据触发结果判断是否对窗口进行计算。
* public class WindowOperator<K, IN, ACC, OUT, W extends Window>
* extends AbstractUdfStreamOperator<OUT, InternalWindowFunction<ACC, OUT, K, W>>
* implements OneInputStreamOperator<IN, OUT>, Triggerable<K, W>
*
* -- processElement()
* |
* +----> windowAssigner.assignWindows
* | //通过WindowAssigner为element分配一系列windows
* +----> windowState.add(element.getValue())
* | //把当前的element加入buffer state
* +----> TriggerResult triggerResult = triggerContext.onElement(element)
* | //触发onElment,得到triggerResult
* +----> Trigger.OnMergeContext.onElement()
* +----> trigger.onElement(element.getValue(), element.getTimestamp(), window,...)
* +----> EventTimeTriggers.onElement()
* | //如果当前window.maxTimestamp已经小于CurrentWatermark,直接触发
* | //否则将window.maxTimestamp注册到TimeService中,等待触发
* +----> contents = windowState.get(); emitWindowContents(actualWindow, contents)
* | //对triggerResult做各种处理,如果fire,真正去计算窗口中的elements
* -- processWatermark()
* -----> 最终进入基类AbstractStreamOperator.processWatermark
* -----> AbstractStreamOperator.processWatermark(watermark)
* -----> timeServiceManager.advanceWatermark(mark); 第一步处理watermark
* -----> output.emitWatermark(mark) 第二步将watermark发送到下游
* -----> InternalTimeServiceManager.advanceWatermark
0x06. 处理 Watermark 的简要流程
最后是处理 Watermark 的简要流程(OneInputStreamTask为例)
* -- OneInputStreamTask.invoke()
* |
* +----> StreamTask.init
* | 把StreamTaskNetworkOutput/StreamTaskNetworkInput聚合StreamOneInputProcessor
* +----> StreamTask.runMailboxLoop
* | 从 StreamTask.runMailboxLoop 开始,下面是一层层的调用关系
* -----> StreamTask.processInput()
* -----> StreamTask.inputProcessor.processInput()
* -----> StreamOneInputProcessor.processInput
* -----> input.emitNext(output)
* -----> StreamTaskNetworkInput.emitNext()
* -----> StreamTaskNetworkInput.processElement()
* 下面是处理普通 Record
* -- StreamTaskNetworkInput.processElement()
* |
* | 下面都是一层层的调用关系
* -----> output.emitRecord(recordOrMark.asRecord())
* -----> StreamTaskNetworkOutput.emitRecord()
* -----> operator.processElement(record)
* 进入具体算子 processElement 的处理,比如StreamFlatMap.processElement
* -----> StreamFlatMap.processElement(record)
* -----> userFunction.flatMap()
* -- 下面是处理 Watermark
* -- StreamTaskNetworkInput.processElement()
* |
* | 下面都是一层层的调用关系
* -----> StatusWatermarkValve.inputWatermark()
* -----> StatusWatermarkValve.findAndOutputNewMinWatermarkAcrossAlignedChannels()
* -----> output.emitWatermark()
* -----> StreamTaskNetworkOutput.emitWatermark()
* -----> operator.processWatermark(watermark)
* -----> KeyedProcessOperator.processWatermark(watermark)
* 具体算子processWatermark处理,如WindowOperator/KeyedProcessOperator.processWatermark
* 最终进入基类AbstractStreamOperator.processWatermark
* -----> AbstractStreamOperator.processWatermark(watermark)
* -----> timeServiceManager.advanceWatermark(mark); 第一步处理watermark
* output.emitWatermark(mark) 第二步将watermark发送到下游
* -----> InternalTimeServiceManager.advanceWatermark
* -----> 下面看看第一步处理watermark
* -----> InternalTimerServiceImpl.advanceWatermark
* 逻辑timer时间小于watermark的都应该被触发回调。从eventTimeTimersQueue从小到大取timer,如果小于传入的water mark,那么说明这个window需要触发。注意watermarker是没有key的,所以当一个watermark来的时候是会触发所有timer,而timer的key是不一定的,所以这里一定要设置keyContext,否则就乱了
* -----> triggerTarget.onEventTime(timer);
* triggerTarget是具体operator对象,open时通过InternalTimeServiceManager.getInternalTimerService传递到HeapInternalTimerService
* -----> KeyedProcessOperator.onEeventTime()
* 调用用户实现的keyedProcessFunction.onTimer去做具体事情。对于window来说也是调用onEventTime或者onProcessTime来从key和window對應的状态中的数据发送到windowFunction中去计算并发送到下游节点
* -----> invokeUserFunction(TimeDomain.PROCESSING_TIME, timer);
* -----> userFunction.onTimer(timer.getTimestamp(), onTimerContext, collector);
* -- DataStream 设置定时发送Watermark,是加了个chain的TimestampsAndPeriodicWatermarksOperator
* -- StreamTaskNetworkInput.processElement()
* -----> TimestampsAndPeriodicWatermarksOperator.processElement
* 会调用AssignerWithPeriodicWatermarks.extractTimestamp提取event time
* 然后更新StreamRecord的时间
* -----> WindowOperator.processElement
* 在windowAssigner.assignWindows时以element的timestamp作为assign时间
0xEE 个人信息
★★★★★★关于生活和技术的思考★★★★★★
微信公众账号:罗西的思考
如果您想及时得到个人撰写文章的消息推送,或者想看看个人推荐的技术资料,敬请关注。
0xFF 参考
Flink原理与实现:如何生成ExecutionGraph及物理执行图
Apache Flink源码解析 (四)Stream Operator
Apache Flink 进阶(六):Flink 作业执行深度解析
Streaming System 第三章:Watermarks
Apache Flink源码解析之stream-source
Flink源码系列——Flink中一个简单的数据处理功能的实现过程
聊聊flink的Execution Plan Visualization
Flink源码系列——Flink中一个简单的数据处理功能的实现过程
Flink源码解读系列1——分析一个简单Flink程序的执行过程
Flink timer注册与watermark触发[转载自网易云音乐实时计算平台经典实践知乎专栏]
[Flink – process watermark](cnblogs.com/fxjwind/p/7…)
Flink流计算编程--Flink中allowedLateness详细介绍及思考
「Spark-2.2.0」Structured Streaming - Watermarking操作详解
flink的window计算、watermark、allowedLateness、trigger
Apache Flink源码解析 (四)Stream Operator
Flink入门教程--Task Lifecycle(任务的生命周期简介)