blob: 2053cf75afb5f30d540e7bd1c7f1909519e5a7ce [file] [log] [blame]
<!DOCTYPE html>
<html lang="en">
<head>
<meta charset="utf-8">
<meta http-equiv="X-UA-Compatible" content="IE=edge">
<meta name="viewport" content="width=device-width, initial-scale=1">
<!-- The above 3 meta tags *must* come first in the head; any other head content must come *after* these tags -->
<title>Apache Flink: Introducing Stream Windows in Apache Flink</title>
<link rel="shortcut icon" href="/favicon.ico" type="image/x-icon">
<link rel="icon" href="/favicon.ico" type="image/x-icon">
<!-- Bootstrap -->
<link rel="stylesheet" href="https://maxcdn.bootstrapcdn.com/bootstrap/3.4.1/css/bootstrap.min.css">
<link rel="stylesheet" href="/css/flink.css">
<link rel="stylesheet" href="/css/syntax.css">
<!-- Blog RSS feed -->
<link href="/blog/feed.xml" rel="alternate" type="application/rss+xml" title="Apache Flink Blog: RSS feed" />
<!-- jQuery (necessary for Bootstrap's JavaScript plugins) -->
<!-- We need to load Jquery in the header for custom google analytics event tracking-->
<script src="/js/jquery.min.js"></script>
<!-- HTML5 shim and Respond.js for IE8 support of HTML5 elements and media queries -->
<!-- WARNING: Respond.js doesn't work if you view the page via file:// -->
<!--[if lt IE 9]>
<script src="https://oss.maxcdn.com/html5shiv/3.7.2/html5shiv.min.js"></script>
<script src="https://oss.maxcdn.com/respond/1.4.2/respond.min.js"></script>
<![endif]-->
</head>
<body>
<!-- Main content. -->
<div class="container">
<div class="row">
<div id="sidebar" class="col-sm-3">
<!-- Top navbar. -->
<nav class="navbar navbar-default">
<!-- The logo. -->
<div class="navbar-header">
<button type="button" class="navbar-toggle collapsed" data-toggle="collapse" data-target="#bs-example-navbar-collapse-1">
<span class="icon-bar"></span>
<span class="icon-bar"></span>
<span class="icon-bar"></span>
</button>
<div class="navbar-logo">
<a href="/">
<img alt="Apache Flink" src="/img/flink-header-logo.svg" width="147px" height="73px">
</a>
</div>
</div><!-- /.navbar-header -->
<!-- The navigation links. -->
<div class="collapse navbar-collapse" id="bs-example-navbar-collapse-1">
<ul class="nav navbar-nav navbar-main">
<!-- First menu section explains visitors what Flink is -->
<!-- What is Stream Processing? -->
<!--
<li><a href="/streamprocessing1.html">What is Stream Processing?</a></li>
-->
<!-- What is Flink? -->
<li><a href="/flink-architecture.html">What is Apache Flink?</a></li>
<!-- What is Stateful Functions? -->
<li><a href="/stateful-functions.html">What is Stateful Functions?</a></li>
<!-- Use cases -->
<li><a href="/usecases.html">Use Cases</a></li>
<!-- Powered by -->
<li><a href="/poweredby.html">Powered By</a></li>
&nbsp;
<!-- Second menu section aims to support Flink users -->
<!-- Downloads -->
<li><a href="/downloads.html">Downloads</a></li>
<!-- Getting Started -->
<li class="dropdown">
<a class="dropdown-toggle" data-toggle="dropdown" href="#">Getting Started<span class="caret"></span></a>
<ul class="dropdown-menu">
<li><a href="https://ci.apache.org/projects/flink/flink-docs-release-1.11/getting-started/index.html" target="_blank">With Flink <small><span class="glyphicon glyphicon-new-window"></span></small></a></li>
<li><a href="https://ci.apache.org/projects/flink/flink-statefun-docs-release-2.1/getting-started/project-setup.html" target="_blank">With Flink Stateful Functions <small><span class="glyphicon glyphicon-new-window"></span></small></a></li>
<li><a href="/training.html">Training Course</a></li>
</ul>
</li>
<!-- Documentation -->
<li class="dropdown">
<a class="dropdown-toggle" data-toggle="dropdown" href="#">Documentation<span class="caret"></span></a>
<ul class="dropdown-menu">
<li><a href="https://ci.apache.org/projects/flink/flink-docs-release-1.11" target="_blank">Flink 1.11 (Latest stable release) <small><span class="glyphicon glyphicon-new-window"></span></small></a></li>
<li><a href="https://ci.apache.org/projects/flink/flink-docs-master" target="_blank">Flink Master (Latest Snapshot) <small><span class="glyphicon glyphicon-new-window"></span></small></a></li>
<li><a href="https://ci.apache.org/projects/flink/flink-statefun-docs-release-2.1" target="_blank">Flink Stateful Functions 2.1 (Latest stable release) <small><span class="glyphicon glyphicon-new-window"></span></small></a></li>
<li><a href="https://ci.apache.org/projects/flink/flink-statefun-docs-master" target="_blank">Flink Stateful Functions Master (Latest Snapshot) <small><span class="glyphicon glyphicon-new-window"></span></small></a></li>
</ul>
</li>
<!-- getting help -->
<li><a href="/gettinghelp.html">Getting Help</a></li>
<!-- Blog -->
<li class="active"><a href="/blog/"><b>Flink Blog</b></a></li>
<!-- Flink-packages -->
<li>
<a href="https://flink-packages.org" target="_blank">flink-packages.org <small><span class="glyphicon glyphicon-new-window"></span></small></a>
</li>
&nbsp;
<!-- Third menu section aim to support community and contributors -->
<!-- Community -->
<li><a href="/community.html">Community &amp; Project Info</a></li>
<!-- Roadmap -->
<li><a href="/roadmap.html">Roadmap</a></li>
<!-- Contribute -->
<li><a href="/contributing/how-to-contribute.html">How to Contribute</a></li>
<!-- GitHub -->
<li>
<a href="https://github.com/apache/flink" target="_blank">Flink on GitHub <small><span class="glyphicon glyphicon-new-window"></span></small></a>
</li>
&nbsp;
<!-- Language Switcher -->
<li>
<!-- link to the Chinese home page when current is blog page -->
<a href="/zh">中文版</a>
</li>
</ul>
<ul class="nav navbar-nav navbar-bottom">
<hr />
<!-- Twitter -->
<li><a href="https://twitter.com/apacheflink" target="_blank">@ApacheFlink <small><span class="glyphicon glyphicon-new-window"></span></small></a></li>
<!-- Visualizer -->
<li class=" hidden-md hidden-sm"><a href="/visualizer/" target="_blank">Plan Visualizer <small><span class="glyphicon glyphicon-new-window"></span></small></a></li>
<hr />
<li><a href="https://apache.org" target="_blank">Apache Software Foundation <small><span class="glyphicon glyphicon-new-window"></span></small></a></li>
<li>
<style>
.smalllinks:link {
display: inline-block !important; background: none; padding-top: 0px; padding-bottom: 0px; padding-right: 0px; min-width: 75px;
}
</style>
<a class="smalllinks" href="https://www.apache.org/licenses/" target="_blank">License</a> <small><span class="glyphicon glyphicon-new-window"></span></small>
<a class="smalllinks" href="https://www.apache.org/security/" target="_blank">Security</a> <small><span class="glyphicon glyphicon-new-window"></span></small>
<a class="smalllinks" href="https://www.apache.org/foundation/sponsorship.html" target="_blank">Donate</a> <small><span class="glyphicon glyphicon-new-window"></span></small>
<a class="smalllinks" href="https://www.apache.org/foundation/thanks.html" target="_blank">Thanks</a> <small><span class="glyphicon glyphicon-new-window"></span></small>
</li>
</ul>
</div><!-- /.navbar-collapse -->
</nav>
</div>
<div class="col-sm-9">
<div class="row-fluid">
<div class="col-sm-12">
<div class="row">
<h1>Introducing Stream Windows in Apache Flink</h1>
<p><i></i></p>
<article>
<p>04 Dec 2015 by Fabian Hueske (<a href="https://twitter.com/fhueske">@fhueske</a>)</p>
<p>The data analysis space is witnessing an evolution from batch to stream processing for many use cases. Although batch can be handled as a special case of stream processing, analyzing never-ending streaming data often requires a shift in the mindset and comes with its own terminology (for example, “windowing” and “at-least-once”/”exactly-once” processing). This shift and the new terminology can be quite confusing for people being new to the space of stream processing. Apache Flink is a production-ready stream processor with an easy-to-use yet very expressive API to define advanced stream analysis programs. Flink’s API features very flexible window definitions on data streams which let it stand out among other open source stream processors.</p>
<p>In this blog post, we discuss the concept of windows for stream processing, present Flink’s built-in windows, and explain its support for custom windowing semantics.</p>
<h2 id="what-are-windows-and-what-are-they-good-for">What are windows and what are they good for?</h2>
<p>Consider the example of a traffic sensor that counts every 15 seconds the number of vehicles passing a certain location. The resulting stream could look like:</p>
<center>
<img src="/img/blog/window-intro/window-stream.png" style="width:75%;margin:15px" />
</center>
<p>If you would like to know, how many vehicles passed that location, you would simply sum the individual counts. However, the nature of a sensor stream is that it continuously produces data. Such a stream never ends and it is not possible to compute a final sum that can be returned. Instead, it is possible to compute rolling sums, i.e., return for each input event an updated sum record. This would yield a new stream of partial sums.</p>
<center>
<img src="/img/blog/window-intro/window-rolling-sum.png" style="width:75%;margin:15px" />
</center>
<p>However, a stream of partial sums might not be what we are looking for, because it constantly updates the count and even more important, some information such as variation over time is lost. Hence, we might want to rephrase our question and ask for the number of cars that pass the location every minute. This requires us to group the elements of the stream into finite sets, each set corresponding to sixty seconds. This operation is called a <em>tumbling windows</em> operation.</p>
<center>
<img src="/img/blog/window-intro/window-tumbling-window.png" style="width:75%;margin:15px" />
</center>
<p>Tumbling windows discretize a stream into non-overlapping windows. For certain applications it is important that windows are not disjunct because an application might require smoothed aggregates. For example, we can compute every thirty seconds the number of cars passed in the last minute. Such windows are called <em>sliding windows</em>.</p>
<center>
<img src="/img/blog/window-intro/window-sliding-window.png" style="width:75%;margin:15px" />
</center>
<p>Defining windows on a data stream as discussed before is a non-parallel operation. This is because each element of a stream must be processed by the same window operator that decides which windows the element should be added to. Windows on a full stream are called <em>AllWindows</em> in Flink. For many applications, a data stream needs to be grouped into multiple logical streams on each of which a window operator can be applied. Think for example about a stream of vehicle counts from multiple traffic sensors (instead of only one sensor as in our previous example), where each sensor monitors a different location. By grouping the stream by sensor id, we can compute windowed traffic statistics for each location in parallel. In Flink, we call such partitioned windows simply <em>Windows</em>, as they are the common case for distributed streams. The following figure shows tumbling windows that collect two elements over a stream of <code>(sensorId, count)</code> pair elements.</p>
<center>
<img src="/img/blog/window-intro/windows-keyed.png" style="width:75%;margin:15px" />
</center>
<p>Generally speaking, a window defines a finite set of elements on an unbounded stream. This set can be based on time (as in our previous examples), element counts, a combination of counts and time, or some custom logic to assign elements to windows. Flink’s DataStream API provides concise operators for the most common window operations as well as a generic windowing mechanism that allows users to define very custom windowing logic. In the following we present Flink’s time and count windows before discussing its windowing mechanism in detail.</p>
<h2 id="time-windows">Time Windows</h2>
<p>As their name suggests, time windows group stream elements by time. For example, a tumbling time window of one minute collects elements for one minute and applies a function on all elements in the window after one minute passed.</p>
<p>Defining tumbling and sliding time windows in Apache Flink is very easy:</p>
<div class="highlight"><pre><code class="language-scala"><span class="c1">// Stream of (sensorId, carCnt)</span>
<span class="k">val</span> <span class="n">vehicleCnts</span><span class="k">:</span> <span class="kt">DataStream</span><span class="o">[(</span><span class="kt">Int</span>, <span class="kt">Int</span><span class="o">)]</span> <span class="k">=</span> <span class="o">...</span>
<span class="k">val</span> <span class="n">tumblingCnts</span><span class="k">:</span> <span class="kt">DataStream</span><span class="o">[(</span><span class="kt">Int</span>, <span class="kt">Int</span><span class="o">)]</span> <span class="k">=</span> <span class="n">vehicleCnts</span>
<span class="c1">// key stream by sensorId</span>
<span class="o">.</span><span class="n">keyBy</span><span class="o">(</span><span class="mi">0</span><span class="o">)</span>
<span class="c1">// tumbling time window of 1 minute length</span>
<span class="o">.</span><span class="n">timeWindow</span><span class="o">(</span><span class="nc">Time</span><span class="o">.</span><span class="n">minutes</span><span class="o">(</span><span class="mi">1</span><span class="o">))</span>
<span class="c1">// compute sum over carCnt</span>
<span class="o">.</span><span class="n">sum</span><span class="o">(</span><span class="mi">1</span><span class="o">)</span>
<span class="k">val</span> <span class="n">slidingCnts</span><span class="k">:</span> <span class="kt">DataStream</span><span class="o">[(</span><span class="kt">Int</span>, <span class="kt">Int</span><span class="o">)]</span> <span class="k">=</span> <span class="n">vehicleCnts</span>
<span class="o">.</span><span class="n">keyBy</span><span class="o">(</span><span class="mi">0</span><span class="o">)</span>
<span class="c1">// sliding time window of 1 minute length and 30 secs trigger interval</span>
<span class="o">.</span><span class="n">timeWindow</span><span class="o">(</span><span class="nc">Time</span><span class="o">.</span><span class="n">minutes</span><span class="o">(</span><span class="mi">1</span><span class="o">),</span> <span class="nc">Time</span><span class="o">.</span><span class="n">seconds</span><span class="o">(</span><span class="mi">30</span><span class="o">))</span>
<span class="o">.</span><span class="n">sum</span><span class="o">(</span><span class="mi">1</span><span class="o">)</span></code></pre></div>
<p>There is one aspect that we haven’t discussed yet, namely the exact meaning of “<em>collects elements for one minute</em>” which boils down to the question, “<em>How does the stream processor interpret time?</em>”.</p>
<p>Apache Flink features three different notions of time, namely <em>processing time</em>, <em>event time</em>, and <em>ingestion time</em>.</p>
<ol>
<li>In <strong>processing time</strong>, windows are defined with respect to the wall clock of the machine that builds and processes a window, i.e., a one minute processing time window collects elements for exactly one minute.</li>
<li>In <strong>event time</strong>, windows are defined with respect to timestamps that are attached to each event record. This is common for many types of events, such as log entries, sensor data, etc, where the timestamp usually represents the time at which the event occurred. Event time has several benefits over processing time. First of all, it decouples the program semantics from the actual serving speed of the source and the processing performance of system. Hence you can process historic data, which is served at maximum speed, and continuously produced data with the same program. It also prevents semantically incorrect results in case of backpressure or delays due to failure recovery. Second, event time windows compute correct results, even if events arrive out-of-order of their timestamp which is common if a data stream gathers events from distributed sources.</li>
<li><strong>Ingestion time</strong> is a hybrid of processing and event time. It assigns wall clock timestamps to records as soon as they arrive in the system (at the source) and continues processing with event time semantics based on the attached timestamps.</li>
</ol>
<h2 id="count-windows">Count Windows</h2>
<p>Apache Flink also features count windows. A tumbling count window of 100 will collect 100 events in a window and evaluate the window when the 100th element has been added.</p>
<p>In Flink’s DataStream API, tumbling and sliding count windows are defined as follows:</p>
<div class="highlight"><pre><code class="language-scala"><span class="c1">// Stream of (sensorId, carCnt)</span>
<span class="k">val</span> <span class="n">vehicleCnts</span><span class="k">:</span> <span class="kt">DataStream</span><span class="o">[(</span><span class="kt">Int</span>, <span class="kt">Int</span><span class="o">)]</span> <span class="k">=</span> <span class="o">...</span>
<span class="k">val</span> <span class="n">tumblingCnts</span><span class="k">:</span> <span class="kt">DataStream</span><span class="o">[(</span><span class="kt">Int</span>, <span class="kt">Int</span><span class="o">)]</span> <span class="k">=</span> <span class="n">vehicleCnts</span>
<span class="c1">// key stream by sensorId</span>
<span class="o">.</span><span class="n">keyBy</span><span class="o">(</span><span class="mi">0</span><span class="o">)</span>
<span class="c1">// tumbling count window of 100 elements size</span>
<span class="o">.</span><span class="n">countWindow</span><span class="o">(</span><span class="mi">100</span><span class="o">)</span>
<span class="c1">// compute the carCnt sum </span>
<span class="o">.</span><span class="n">sum</span><span class="o">(</span><span class="mi">1</span><span class="o">)</span>
<span class="k">val</span> <span class="n">slidingCnts</span><span class="k">:</span> <span class="kt">DataStream</span><span class="o">[(</span><span class="kt">Int</span>, <span class="kt">Int</span><span class="o">)]</span> <span class="k">=</span> <span class="n">vehicleCnts</span>
<span class="o">.</span><span class="n">keyBy</span><span class="o">(</span><span class="mi">0</span><span class="o">)</span>
<span class="c1">// sliding count window of 100 elements size and 10 elements trigger interval</span>
<span class="o">.</span><span class="n">countWindow</span><span class="o">(</span><span class="mi">100</span><span class="o">,</span> <span class="mi">10</span><span class="o">)</span>
<span class="o">.</span><span class="n">sum</span><span class="o">(</span><span class="mi">1</span><span class="o">)</span></code></pre></div>
<h2 id="dissecting-flinks-windowing-mechanics">Dissecting Flink’s windowing mechanics</h2>
<p>Flink’s built-in time and count windows cover a wide range of common window use cases. However, there are of course applications that require custom windowing logic that cannot be addressed by Flink’s built-in windows. In order to support also applications that need very specific windowing semantics, the DataStream API exposes interfaces for the internals of its windowing mechanics. These interfaces give very fine-grained control about the way that windows are built and evaluated.</p>
<p>The following figure depicts Flink’s windowing mechanism and introduces the components being involved.</p>
<center>
<img src="/img/blog/window-intro/window-mechanics.png" style="width:90%;margin:15px" />
</center>
<p>Elements that arrive at a window operator are handed to a <code>WindowAssigner</code>. The WindowAssigner assigns elements to one or more windows, possibly creating new windows. A <code>Window</code> itself is just an identifier for a list of elements and may provide some optional meta information, such as begin and end time in case of a <code>TimeWindow</code>. Note that an element can be added to multiple windows, which also means that multiple windows can exist at the same time.</p>
<p>Each window owns a <code>Trigger</code> that decides when the window is evaluated or purged. The trigger is called for each element that is inserted into the window and when a previously registered timer times out. On each event, a trigger can decide to fire (i.e., evaluate), purge (remove the window and discard its content), or fire and then purge the window. A trigger that just fires evaluates the window and keeps it as it is, i.e., all elements remain in the window and are evaluated again when the triggers fires the next time. A window can be evaluated several times and exists until it is purged. Note that a window consumes memory until it is purged.</p>
<p>When a Trigger fires, the list of window elements can be given to an optional <code>Evictor</code>. The evictor can iterate through the list and decide to cut off some elements from the start of the list, i.e., remove some of the elements that entered the window first. The remaining elements are given to an evaluation function. If no Evictor was defined, the Trigger hands all the window elements directly to the evaluation function.</p>
<p>The evaluation function receives the elements of a window (possibly filtered by an Evictor) and computes one or more result elements for the window. The DataStream API accepts different types of evaluation functions, including predefined aggregation functions such as <code>sum()</code>, <code>min()</code>, <code>max()</code>, as well as a <code>ReduceFunction</code>, <code>FoldFunction</code>, or <code>WindowFunction</code>. A WindowFunction is the most generic evaluation function and receives the window object (i.e, the meta data of the window), the list of window elements, and the window key (in case of a keyed window) as parameters.</p>
<p>These are the components that constitute Flink’s windowing mechanics. We now show step-by-step how to implement custom windowing logic with the DataStream API. We start with a stream of type <code>DataStream[IN]</code> and key it using a key selector function that extracts a key of type <code>KEY</code> to obtain a <code>KeyedStream[IN, KEY]</code>.</p>
<div class="highlight"><pre><code class="language-scala"><span class="k">val</span> <span class="n">input</span><span class="k">:</span> <span class="kt">DataStream</span><span class="o">[</span><span class="kt">IN</span><span class="o">]</span> <span class="k">=</span> <span class="o">...</span>
<span class="c1">// created a keyed stream using a key selector function</span>
<span class="k">val</span> <span class="n">keyed</span><span class="k">:</span> <span class="kt">KeyedStream</span><span class="o">[</span><span class="kt">IN</span>, <span class="kt">KEY</span><span class="o">]</span> <span class="k">=</span> <span class="n">input</span>
<span class="o">.</span><span class="n">keyBy</span><span class="o">(</span><span class="n">myKeySel</span><span class="k">:</span> <span class="o">(</span><span class="kt">IN</span><span class="o">)</span> <span class="o">=&gt;</span> <span class="nc">KEY</span><span class="o">)</span></code></pre></div>
<p>We apply a <code>WindowAssigner[IN, WINDOW]</code> that creates windows of type <code>WINDOW</code> resulting in a <code>WindowedStream[IN, KEY, WINDOW]</code>. In addition, a <code>WindowAssigner</code> also provides a default <code>Trigger</code> implementation.</p>
<div class="highlight"><pre><code class="language-scala"><span class="c1">// create windowed stream using a WindowAssigner</span>
<span class="k">var</span> <span class="n">windowed</span><span class="k">:</span> <span class="kt">WindowedStream</span><span class="o">[</span><span class="kt">IN</span>, <span class="kt">KEY</span>, <span class="kt">WINDOW</span><span class="o">]</span> <span class="k">=</span> <span class="n">keyed</span>
<span class="o">.</span><span class="n">window</span><span class="o">(</span><span class="n">myAssigner</span><span class="k">:</span> <span class="kt">WindowAssigner</span><span class="o">[</span><span class="kt">IN</span>, <span class="kt">WINDOW</span><span class="o">])</span></code></pre></div>
<p>We can explicitly specify a <code>Trigger</code> to overwrite the default <code>Trigger</code> provided by the <code>WindowAssigner</code>. Note that specifying a triggers does not add an additional trigger condition but replaces the current trigger.</p>
<div class="highlight"><pre><code class="language-scala"><span class="c1">// override the default trigger of the WindowAssigner</span>
<span class="n">windowed</span> <span class="k">=</span> <span class="n">windowed</span>
<span class="o">.</span><span class="n">trigger</span><span class="o">(</span><span class="n">myTrigger</span><span class="k">:</span> <span class="kt">Trigger</span><span class="o">[</span><span class="kt">IN</span>, <span class="kt">WINDOW</span><span class="o">])</span></code></pre></div>
<p>We may want to specify an optional <code>Evictor</code> as follows.</p>
<div class="highlight"><pre><code class="language-scala"><span class="c1">// specify an optional evictor</span>
<span class="n">windowed</span> <span class="k">=</span> <span class="n">windowed</span>
<span class="o">.</span><span class="n">evictor</span><span class="o">(</span><span class="n">myEvictor</span><span class="k">:</span> <span class="kt">Evictor</span><span class="o">[</span><span class="kt">IN</span>, <span class="kt">WINDOW</span><span class="o">])</span></code></pre></div>
<p>Finally, we apply a <code>WindowFunction</code> that returns elements of type <code>OUT</code> to obtain a <code>DataStream[OUT]</code>.</p>
<div class="highlight"><pre><code class="language-scala"><span class="c1">// apply window function to windowed stream</span>
<span class="k">val</span> <span class="n">output</span><span class="k">:</span> <span class="kt">DataStream</span><span class="o">[</span><span class="kt">OUT</span><span class="o">]</span> <span class="k">=</span> <span class="n">windowed</span>
<span class="o">.</span><span class="n">apply</span><span class="o">(</span><span class="n">myWinFunc</span><span class="k">:</span> <span class="kt">WindowFunction</span><span class="o">[</span><span class="kt">IN</span>, <span class="kt">OUT</span>, <span class="kt">KEY</span>, <span class="kt">WINDOW</span><span class="o">])</span></code></pre></div>
<p>With Flink’s internal windowing mechanics and its exposure through the DataStream API it is possible to implement very custom windowing logic such as session windows or windows that emit early results if the values exceed a certain threshold.</p>
<h2 id="conclusion">Conclusion</h2>
<p>Support for various types of windows over continuous data streams is a must-have for modern stream processors. Apache Flink is a stream processor with a very strong feature set, including a very flexible mechanism to build and evaluate windows over continuous data streams. Flink provides pre-defined window operators for common uses cases as well as a toolbox that allows to define very custom windowing logic. The Flink community will add more pre-defined window operators as we learn the requirements from our users.</p>
</article>
</div>
<div class="row">
<div id="disqus_thread"></div>
<script type="text/javascript">
/* * * CONFIGURATION VARIABLES: EDIT BEFORE PASTING INTO YOUR WEBPAGE * * */
var disqus_shortname = 'stratosphere-eu'; // required: replace example with your forum shortname
/* * * DON'T EDIT BELOW THIS LINE * * */
(function() {
var dsq = document.createElement('script'); dsq.type = 'text/javascript'; dsq.async = true;
dsq.src = '//' + disqus_shortname + '.disqus.com/embed.js';
(document.getElementsByTagName('head')[0] || document.getElementsByTagName('body')[0]).appendChild(dsq);
})();
</script>
</div>
</div>
</div>
</div>
</div>
<hr />
<div class="row">
<div class="footer text-center col-sm-12">
<p>Copyright © 2014-2019 <a href="http://apache.org">The Apache Software Foundation</a>. All Rights Reserved.</p>
<p>Apache Flink, Flink®, Apache®, the squirrel logo, and the Apache feather logo are either registered trademarks or trademarks of The Apache Software Foundation.</p>
<p><a href="/privacy-policy.html">Privacy Policy</a> &middot; <a href="/blog/feed.xml">RSS feed</a></p>
</div>
</div>
</div><!-- /.container -->
<!-- Include all compiled plugins (below), or include individual files as needed -->
<script src="https://maxcdn.bootstrapcdn.com/bootstrap/3.3.4/js/bootstrap.min.js"></script>
<script src="https://cdnjs.cloudflare.com/ajax/libs/jquery.matchHeight/0.7.0/jquery.matchHeight-min.js"></script>
<script src="/js/codetabs.js"></script>
<script src="/js/stickysidebar.js"></script>
<!-- Google Analytics -->
<script>
(function(i,s,o,g,r,a,m){i['GoogleAnalyticsObject']=r;i[r]=i[r]||function(){
(i[r].q=i[r].q||[]).push(arguments)},i[r].l=1*new Date();a=s.createElement(o),
m=s.getElementsByTagName(o)[0];a.async=1;a.src=g;m.parentNode.insertBefore(a,m)
})(window,document,'script','//www.google-analytics.com/analytics.js','ga');
ga('create', 'UA-52545728-1', 'auto');
ga('send', 'pageview');
</script>
</body>
</html>