You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by se...@apache.org on 2017/10/24 14:50:45 UTC

[3/3] flink git commit: [hotfix] [docs] CEP docs review to remove weasel words, fix passive voice, typos and formatting

[hotfix] [docs] CEP docs review to remove weasel words, fix passive voice, typos and formatting

Backport from master (1.4)


Project: http://git-wip-us.apache.org/repos/asf/flink/repo
Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/51e526f8
Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/51e526f8
Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/51e526f8

Branch: refs/heads/release-1.3
Commit: 51e526f88e460c9c8b936368714af9f15daf7fa6
Parents: 8c76c33
Author: Chris Ward <ch...@gmail.com>
Authored: Tue Oct 17 16:36:48 2017 +0200
Committer: Stephan Ewen <se...@apache.org>
Committed: Tue Oct 24 16:48:31 2017 +0200

----------------------------------------------------------------------
 docs/dev/libs/cep.md | 319 ++++++++++++++++++++--------------------------
 1 file changed, 137 insertions(+), 182 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/51e526f8/docs/dev/libs/cep.md
----------------------------------------------------------------------
diff --git a/docs/dev/libs/cep.md b/docs/dev/libs/cep.md
index 87680ae..33ba30b 100644
--- a/docs/dev/libs/cep.md
+++ b/docs/dev/libs/cep.md
@@ -23,15 +23,12 @@ specific language governing permissions and limitations
 under the License.
 -->
 
-FlinkCEP is the Complex Event Processing (CEP) library implemented on top of Flink.
-It allows you to easily detect event patterns in an endless stream of events, thus
-giving you the opportunity to quickly get hold of what's really important in your 
-data.
-
-This page describes the API calls available in Flink CEP. We start by presenting the [Pattern API](#the-pattern-api), 
-which allows you to specify the patterns that you want to detect in your stream, before presenting how you can 
-[detect and act upon matching event sequences](#detecting-patterns). At the end, we present the assumptions the CEP 
-library makes when [dealing with lateness](#handling-lateness-in-event-time) in event time and how you can 
+FlinkCEP is the Complex Event Processing (CEP) library implemented on top of Flink. It allows you to detect event patterns in an endless stream of events, giving you the opportunity to get hold of what's important in your
+
+This page describes the API calls available in Flink CEP. We start by presenting the [Pattern API](#the-pattern-api),
+which allows you to specify the patterns that you want to detect in your stream, before presenting how you can
+[detect and act upon matching event sequences](#detecting-patterns). We then present the assumptions the CEP
+library makes when [dealing with lateness](#handling-lateness-in-event-time) in event time and how you can
 [migrate your job](#migrating-from-an-older-flink-version) from an older Flink version to Flink-1.3.
 
 * This will be replaced by the TOC
@@ -39,8 +36,7 @@ library makes when [dealing with lateness](#handling-lateness-in-event-time) in
 
 ## Getting Started
 
-If you want to jump right in, you have to [set up a Flink program]({{ site.baseurl }}/dev/linking_with_flink.html) and 
-add the FlinkCEP dependency to the `pom.xml` of your project.
+If you want to jump right in, [set up a Flink program]({{ site.baseurl }}/dev/linking_with_flink.html) and add the FlinkCEP dependency to the `pom.xml` of your project.
 
 <div class="codetabs" markdown="1">
 <div data-lang="java" markdown="1">
@@ -64,14 +60,13 @@ add the FlinkCEP dependency to the `pom.xml` of your project.
 </div>
 </div>
 
-Note that FlinkCEP is currently not part of the binary distribution.
-See linking with it for cluster execution [here]({{site.baseurl}}/dev/linking.html).
+{% info %} FlinkCEP is not part of the binary distribution. See how to link with it for cluster execution [here]({{site.baseurl}}/dev/linking.html).
 
 Now you can start writing your first CEP program using the Pattern API.
 
 {% warn Attention %} The events in the `DataStream` to which
 you want to apply pattern matching must implement proper `equals()` and `hashCode()` methods
-because these are used for comparing and matching events.
+because FlinkCEP uses them for comparing and matching events..
 
 <div class="codetabs" markdown="1">
 <div data-lang="java" markdown="1">
@@ -130,68 +125,55 @@ val result: DataStream[Alert] = patternStream.select(createAlert(_))
 
 ## The Pattern API
 
-The pattern API allows you to quickly define complex pattern sequences that you want to extract 
-from your input stream.
+The pattern API allows you to define complex pattern sequences that you want to extract from your input stream.
 
-Each such complex pattern sequence consists of multiple simple patterns, i.e. patterns looking for 
-individual events with the same properties. From now on, these simple patterns will be called **patterns**, and
-the final complex pattern sequence we are searching for in the stream, the **pattern sequence**. A pattern sequence
-can be seen as a graph of such patterns, where transitions from one pattern to the next occur based on user-specified
-*conditions*, e.g. `event.getName().equals("start")`. A **match** is a sequence of input events which visits all
+Each complex pattern sequence consists of multiple simple patterns, i.e. patterns looking for individual events with the same properties. From now on, we will call these simple patterns **patterns**, and the final complex pattern sequence we are searching for in the stream, the **pattern sequence**. You can see a pattern sequence as a graph of such patterns, where transitions from one pattern to the next occur based on user-specified *conditions*, e.g. `event.getName().equals("start")`. A **match** is a sequence of input events which visits all
 patterns of the complex pattern graph, through a sequence of valid pattern transitions.
 
-{% warn Attention %} Each pattern must have a unique name, which is used to later
-identify the matched events.
+{% warn Attention %} Each pattern must have a unique name, which you use later to identify the matched events.
 
 {% warn Attention %} Pattern names **CANNOT** contain the character `":"`.
 
-In the remainder of this section we will first describe how to define [Individual Patterns](#individual-patterns), and then cover how you can
-combine individual patterns into [Complex Patterns](#combining-patterns).
+In the rest of this section we will first describe how to define [Individual Patterns](#individual-patterns), and then how you can combine individual patterns into [Complex Patterns](#combining-patterns).
 
 ### Individual Patterns
 
-A **Pattern** can be either a *singleton* pattern, or a *looping* one. Singleton patterns accept a single 
-event, while looping ones can accept more than one. In pattern matching symbols, in the pattern `"a b+ c? d"` (or `"a"`, 
-followed by *one or more* `"b"`'s, optionally followed by a `"c"`, followed by a `"d"`), `a`, `c?`, and `d` are 
-singleton patterns, while `b+` is a looping one. By default, a pattern is a singleton pattern and you can transform 
-it to a looping one by using [Quantifiers](#quantifiers). In addition, each pattern can have one or more
-[Conditions](#conditions) based on which it accepts events.
+A **Pattern** can be either a *singleton* or a *looping* pattern. Singleton patterns accept a single event, while looping patterns can accept more than one. In pattern matching symbols, the pattern `"a b+ c? d"` (or `"a"`, followed by *one or more* `"b"`'s, optionally followed by a `"c"`, followed by a `"d"`), `a`, `c?`, and `d` are singleton patterns, while `b+` is a looping one. By default, a pattern is a singleton pattern and you can transform it to a looping one by using [Quantifiers](#quantifiers). Each pattern can have one or more [Conditions](#conditions) based on which it accepts events.
 
 #### Quantifiers
 
-In FlinkCEP, looping patterns can be specified using these methods: `pattern.oneOrMore()`, for patterns that expect one or
-more occurrences of a given event (e.g. the `b+` mentioned previously); and `pattern.times(#ofTimes)`, for patterns that
-expect a specific number of occurrences of a given type of event, e.g. 4 `a`'s. All patterns, looping or not, can be made 
-optional using the `pattern.optional()` method. For a pattern named `start`, the following are valid quantifiers:
- 
+In FlinkCEP, you can specifiy looping patterns using these methods: `pattern.oneOrMore()`, for patterns that expect one or more occurrences of a given event (e.g. the `b+` mentioned before); and `pattern.times(#ofTimes)`, for patterns that expect a specific number of occurrences of a given type of event, e.g. 4 `a`'s; and `pattern.times(#fromTimes, #toTimes)`, for patterns that expect a specific minimum number of occurrences and a maximum number of occurrences of a given type of event, e.g. 2-4 `a`s.
+
+You can make looping patterns greedy using the `pattern.greedy()` method, but you cannot yet make group patterns greedy. You can make all patterns, looping or not, optional using the `pattern.optional()` method. For a pattern named `start`, the following are valid quantifiers:
+
  <div class="codetabs" markdown="1">
  <div data-lang="java" markdown="1">
  {% highlight java %}
  // expecting 4 occurrences
  start.times(4);
-  
+
  // expecting 0 or 4 occurrences
  start.times(4).optional();
- 
+
  // expecting 1 or more occurrences
  start.oneOrMore();
-   
+
  // expecting 0 or more occurrences
  start.oneOrMore().optional();
  {% endhighlight %}
  </div>
- 
+
  <div data-lang="scala" markdown="1">
  {% highlight scala %}
  // expecting 4 occurrences
  start.times(4)
-   
+
  // expecting 0 or 4 occurrences
  start.times(4).optional()
-  
+
  // expecting 1 or more occurrences
  start.oneOrMore()
-    
+
  // expecting 0 or more occurrences
  start.oneOrMore().optional()
  {% endhighlight %}
@@ -200,30 +182,22 @@ optional using the `pattern.optional()` method. For a pattern named `start`, the
 
 #### Conditions
 
-At every pattern, and in order to go from one pattern to the next, you can specify additional **conditions**. 
-These conditions can be related to:
- 
- 1. a [property of the incoming event](#conditions-on-properties), e.g. its value should be larger than 5, 
- or larger than the average value of the previously accepted events.
+At every pattern, and to go from one pattern to the next, you can specify additional **conditions**. You can relate these conditions to:
+
+1. A [property of the incoming event](#conditions-on-properties), e.g. its value should be larger than 5, or larger than the average value of the previously accepted events.
 
- 2. the [contiguity of the matching events](#conditions-on-contiguity), e.g. detect pattern `a,b,c` without 
- non-matching events between any matching ones.
- 
-The latter refers to "looping" patterns, *i.e.* patterns that can accept more than one event, e.g. the `b+` in `a b+ c`, 
+2. The [contiguity of the matching events](#conditions-on-contiguity), e.g. detect pattern `a,b,c` without non-matching events between any matching ones.
+
+The latter refers to "looping" patterns, *i.e.* patterns that can accept more than one event, e.g. the `b+` in `a b+ c`,
 which searches for one or more `b`'s.
 
 ##### Conditions on Properties
 
-Conditions on the event properties can be specified via the `pattern.where()` or the `pattern.or()` method. These can 
-be either `IterativeCondition`s or `SimpleCondition`s.
+You can specify conditions on the event properties via the `pattern.where()`, `pattern.or()` or the `pattern.until()` method. These can be either `IterativeCondition`s or `SimpleCondition`s.
 
-**Iterative Conditions:** This is the most general type of conditions. This is how you can specify a condition that 
-accepts subsequent events based on properties of the previously accepted events or some statistic over a subset of them. 
+**Iterative Conditions:** This is the most general type of condition. This is how you can specify a condition that accepts subsequent events based on properties of the previously accepted events or a statistic over a subset of them.
 
-Below is the code for an iterative condition that accepts the next event for a pattern named "middle" if its name starts 
-with "foo", and if the sum of the prices of the previously accepted events for that pattern plus the price of the current 
-event do not exceed the value of 5.0. Iterative conditions can be very powerful, especially in combination with looping 
-patterns, e.g. `oneOrMore()`.
+Below is the code for an iterative condition that accepts the next event for a pattern named "middle" if its name starts with "foo", and if the sum of the prices of the previously accepted events for that pattern plus the price of the current event do not exceed the value of 5.0. Iterative conditions can be powerful, especially in combination with looping patterns, e.g. `oneOrMore()`.
 
 <div class="codetabs" markdown="1">
 <div data-lang="java" markdown="1">
@@ -234,7 +208,7 @@ middle.oneOrMore().where(new IterativeCondition<SubEvent>() {
         if (!value.getName().startsWith("foo")) {
             return false;
         }
-        
+
         double sum = value.getPrice();
         for (Event event : ctx.getEventsForPattern("middle")) {
             sum += event.getPrice();
@@ -258,10 +232,10 @@ middle.oneOrMore().where(
 </div>
 
 {% warn Attention %} The call to `context.getEventsForPattern(...)` finds all the
-previously accepted events for a given potential match. The cost of this operation can vary, so when implementing 
+previously accepted events for a given potential match. The cost of this operation can vary, so when implementing
 your condition, try to minimize its use.
 
-**Simple Conditions:** This type of condition extends the aforementioned `IterativeCondition` class and decides 
+**Simple Conditions:** This type of condition extends the aforementioned `IterativeCondition` class and decides
 whether to accept an event or not, based *only* on properties of the event itself.
 
 <div class="codetabs" markdown="1">
@@ -283,8 +257,7 @@ start.where(event => event.getName.startsWith("foo"))
 </div>
 </div>
 
-Finally, we can also restrict the type of the accepted event to some subtype of the initial event type (here `Event`) 
-via the `pattern.subtype(subClass)` method.
+Finally, you can also restrict the type of the accepted event to a subtype of the initial event type (here `Event`) via the `pattern.subtype(subClass)` method.
 
 <div class="codetabs" markdown="1">
 <div data-lang="java" markdown="1">
@@ -305,10 +278,7 @@ start.subtype(classOf[SubEvent]).where(subEvent => ... /* some condition */)
 </div>
 </div>
 
-**Combining Conditions:** As shown, the `subtype` condition can be combined with additional conditions. 
-In fact, this holds for every condition. You can arbitrarily combine conditions by sequentially calling 
-`where()`. The final result will be the logical **AND** of the results of the individual conditions. In 
-order to combine conditions using **OR**, you can use the `or()` method, as shown below.
+**Combining Conditions:** As shown above, you can combine the `subtype` condition with additional conditions. This holds for every condition. You can arbitrarily combine conditions by sequentially calling `where()`. The final result will be the logical **AND** of the results of the individual conditions. To combine conditions using **OR**, you can use the `or()` method, as shown below.
 
 <div class="codetabs" markdown="1">
 <div data-lang="java" markdown="1">
@@ -338,25 +308,24 @@ pattern.where(event => ... /* some condition */).or(event => ... /* or condition
 
 FlinkCEP supports the following forms of contiguity between events:
 
- 1. Strict Contiguity: which expects all matching events to appear strictly the one after the other,
- without any non-matching events in-between.
+1. **Strict Contiguity**: Expects all matching events to appear strictly one after the other, without any non-matching events in-between.
 
- 2. Relaxed Contiguity: which simply ignores non-matching events appearing in-between the matching ones.
- 
- 3. Non-Deterministic Relaxed Contiguity: which further relaxes contiguity, allowing additional matches 
- that ignore some matching events.
+2. **Relaxed Contiguity**: Ignores non-matching events appearing in-between the matching ones.
 
-To illustrate the above with an example, a pattern sequence `"a+ b"` (one or more `"a"`'s followed by a `"b"`) with 
+3. **Non-Deterministic Relaxed Contiguity**: Further relaxes contiguity, allowing additional matches
+   that ignore some matching events.
+
+To illustrate the above with an example, a pattern sequence `"a+ b"` (one or more `"a"`'s followed by a `"b"`) with
 input `"a1", "c", "a2", "b"` will have the following results:
 
- 1. Strict Contiguity: `{a2 b}` -- the `"c"` after `"a1"` causes `"a1"` to be discarded.
+1. **Strict Contiguity**: `{a2 b}` -- the `"c"` after `"a1"` causes `"a1"` to be discarded.
+
+2. **Relaxed Contiguity**: `{a1 b}` and `{a1 a2 b}` -- `c` is ignored.
 
- 2. Relaxed Contiguity: `{a1 b}` and `{a1 a2 b}` -- `c` is simply ignored.
- 
- 3. Non-Deterministic Relaxed Contiguity: `{a1 b}`, `{a2 b}`, and `{a1 a2 b}`.
- 
-For looping patterns (e.g. `oneOrMore()` and `times()`) the default is *relaxed contiguity*. If you want 
-strict contiguity, you have to explicitly specify it by using the `consecutive()` call, and if you want 
+3. **Non-Deterministic Relaxed Contiguity**: `{a1 b}`, `{a2 b}`, and `{a1 a2 b}`.
+
+For looping patterns (e.g. `oneOrMore()` and `times()`) the default is *relaxed contiguity*. If you want
+strict contiguity, you have to explicitly specify it by using the `consecutive()` call, and if you want
 *non-deterministic relaxed contiguity* you can use the `allowCombinations()` call.
 
 {% warn Attention %}
@@ -393,7 +362,7 @@ pattern.where(new IterativeCondition<Event>() {
         <tr>
             <td><strong>or(condition)</strong></td>
             <td>
-                <p>Adds a new condition which is ORed with an existing one. An event can match the pattern only if it 
+                <p>Adds a new condition which is ORed with an existing one. An event can match the pattern only if it
                 passes at least one of the conditions:</p>
 {% highlight java %}
 pattern.where(new IterativeCondition<Event>() {
@@ -413,7 +382,7 @@ pattern.where(new IterativeCondition<Event>() {
        <tr>
            <td><strong>subtype(subClass)</strong></td>
            <td>
-               <p>Defines a subtype condition for the current pattern. An event can only match the pattern if it is 
+               <p>Defines a subtype condition for the current pattern. An event can only match the pattern if it is
                 of this subtype:</p>
 {% highlight java %}
 pattern.subtype(SubEvent.class);
@@ -445,7 +414,7 @@ pattern.times(2);
        <tr>
           <td><strong>optional()</strong></td>
           <td>
-              <p>Specifies that this pattern is optional, i.e. it may not occur at all. This is applicable to all 
+              <p>Specifies that this pattern is optional, i.e. it may not occur at all. This is applicable to all
               aforementioned quantifiers.</p>
 {% highlight java %}
 pattern.oneOrMore().optional();
@@ -458,7 +427,7 @@ pattern.oneOrMore().optional();
               <p>Works in conjunction with <code>oneOrMore()</code> and <code>times()</code> and imposes strict contiguity between the matching
               events, i.e. any non-matching element breaks the match (as in <code>next()</code>).</p>
               <p>If not applied a relaxed contiguity (as in <code>followedBy()</code>) is used.</p>
-            
+
               <p>E.g. a pattern like:</p>
 {% highlight java %}
 Pattern.<Event>begin("start").where(new SimpleCondition<Event>() {
@@ -481,7 +450,7 @@ Pattern.<Event>begin("start").where(new SimpleCondition<Event>() {
 });
 {% endhighlight %}
               <p>Will generate the following matches for an input sequence: C D A1 A2 A3 D A4 B</p>
-            
+
               <p>with consecutive applied: {C A1 B}, {C A1 A2 B}, {C A1 A2 A3 B}</p>
               <p>without consecutive applied: {C A1 B}, {C A1 A2 B}, {C A1 A2 A3 B}, {C A1 A2 A3 A4 B}</p>
           </td>
@@ -492,7 +461,7 @@ Pattern.<Event>begin("start").where(new SimpleCondition<Event>() {
               <p>Works in conjunction with <code>oneOrMore()</code> and <code>times()</code> and imposes non-deterministic relaxed contiguity
               between the matching events (as in <code>followedByAny()</code>).</p>
               <p>If not applied a relaxed contiguity (as in <code>followedBy()</code>) is used.</p>
-                   
+
               <p>E.g. a pattern like:</p>
 {% highlight java %}
 Pattern.<Event>begin("start").where(new SimpleCondition<Event>() {
@@ -515,7 +484,7 @@ Pattern.<Event>begin("start").where(new SimpleCondition<Event>() {
 });
 {% endhighlight %}
                <p>Will generate the following matches for an input sequence: C D A1 A2 A3 D A4 B</p>
-               
+
                <p>with combinations enabled: {C A1 B}, {C A1 A2 B}, {C A1 A3 B}, {C A1 A4 B}, {C A1 A2 A3 B}, {C A1 A2 A4 B}, {C A1 A3 A4 B}, {C A1 A2 A3 A4 B}</p>
                <p>without combinations enabled: {C A1 B}, {C A1 A2 B}, {C A1 A2 A3 B}, {C A1 A2 A3 A4 B}</p>
        </td>
@@ -533,7 +502,7 @@ Pattern.<Event>begin("start").where(new SimpleCondition<Event>() {
         </tr>
 	    </thead>
     <tbody>
-      
+
         <tr>
             <td><strong>where(condition)</strong></td>
             <td>
@@ -547,7 +516,7 @@ pattern.where(event => ... /* some condition */)
         <tr>
             <td><strong>or(condition)</strong></td>
             <td>
-                <p>Adds a new condition which is ORed with an existing one. An event can match the pattern only if it 
+                <p>Adds a new condition which is ORed with an existing one. An event can match the pattern only if it
                 passes at least one of the conditions:</p>
 {% highlight scala %}
 pattern.where(event => ... /* some condition */)
@@ -558,7 +527,7 @@ pattern.where(event => ... /* some condition */)
        <tr>
            <td><strong>subtype(subClass)</strong></td>
            <td>
-               <p>Defines a subtype condition for the current pattern. An event can only match the pattern if it is 
+               <p>Defines a subtype condition for the current pattern. An event can only match the pattern if it is
                of this subtype:</p>
 {% highlight scala %}
 pattern.subtype(classOf[SubEvent])
@@ -580,7 +549,7 @@ pattern.oneOrMore()
                  <td><strong>times(#ofTimes)</strong></td>
                  <td>
                      <p>Specifies that this pattern expects an exact number of occurrences of a matching event.</p>
-                                   <p>By default a relaxed internal contiguity (between subsequent events) is used. 
+                                   <p>By default a relaxed internal contiguity (between subsequent events) is used.
                                    For more info on internal contiguity see <a href="#consecutive_scala">consecutive</a>.</p>
 {% highlight scala %}
 pattern.times(2)
@@ -590,7 +559,7 @@ pattern.times(2)
        <tr>
           <td><strong>optional()</strong></td>
           <td>
-             <p>Specifies that this pattern is optional, i.e. it may not occur at all. This is applicable to all 
+             <p>Specifies that this pattern is optional, i.e. it may not occur at all. This is applicable to all
                            aforementioned quantifiers.</p>
 {% highlight scala %}
 pattern.oneOrMore().optional()
@@ -603,8 +572,8 @@ pattern.oneOrMore().optional()
             <p>Works in conjunction with <code>oneOrMore()</code> and <code>times()</code> and imposes strict contiguity between the matching
                           events, i.e. any non-matching element breaks the match (as in <code>next()</code>).</p>
                           <p>If not applied a relaxed contiguity (as in <code>followedBy()</code>) is used.</p>
-            
-      <p>E.g. a pattern like:</p> 
+
+      <p>E.g. a pattern like:</p>
 {% highlight scala %}
 Pattern.begin("start").where(_.getName().equals("c"))
   .followedBy("middle").where(_.getName().equals("a"))
@@ -613,7 +582,7 @@ Pattern.begin("start").where(_.getName().equals("c"))
 {% endhighlight %}
 
             <p>Will generate the following matches for an input sequence: C D A1 A2 A3 D A4 B</p>
-                        
+
                           <p>with consecutive applied: {C A1 B}, {C A1 A2 B}, {C A1 A2 A3 B}</p>
                           <p>without consecutive applied: {C A1 B}, {C A1 A2 B}, {C A1 A2 A3 B}, {C A1 A2 A3 A4 B}</p>
           </td>
@@ -624,7 +593,7 @@ Pattern.begin("start").where(_.getName().equals("c"))
                 <p>Works in conjunction with <code>oneOrMore()</code> and <code>times()</code> and imposes non-deterministic relaxed contiguity
                      between the matching events (as in <code>followedByAny()</code>).</p>
                      <p>If not applied a relaxed contiguity (as in <code>followedBy()</code>) is used.</p>
-                          
+
       <p>E.g. a pattern like:</p>
 {% highlight scala %}
 Pattern.begin("start").where(_.getName().equals("c"))
@@ -632,9 +601,9 @@ Pattern.begin("start").where(_.getName().equals("c"))
                        .oneOrMore().allowCombinations()
   .followedBy("end1").where(_.getName().equals("b"));
 {% endhighlight %}
-                     
+
                       <p>Will generate the following matches for an input sequence: C D A1 A2 A3 D A4 B</p>
-                          
+
                       <p>with combinations enabled: {C A1 B}, {C A1 A2 B}, {C A1 A3 B}, {C A1 A4 B}, {C A1 A2 A3 B}, {C A1 A2 A4 B}, {C A1 A3 A4 B}, {C A1 A2 A3 A4 B}</p>
                       <p>without combinations enabled: {C A1 B}, {C A1 A2 B}, {C A1 A2 A3 B}, {C A1 A2 A3 A4 B}</p>
               </td>
@@ -647,8 +616,7 @@ Pattern.begin("start").where(_.getName().equals("c"))
 
 ### Combining Patterns
 
-Now that we have seen what an individual pattern can look like, it is time to see how to combine them 
-into a full pattern sequence.
+Now that you've seen what an individual pattern can look like, it is time to see how to combine them into a full pattern sequence.
 
 A pattern sequence has to start with an initial pattern, as shown below:
 
@@ -666,16 +634,16 @@ val start : Pattern[Event, _] = Pattern.begin("start")
 </div>
 </div>
 
-Next, you can append more patterns to your pattern sequence by specifying the desired *contiguity conditions* between 
+Next, you can append more patterns to your pattern sequence by specifying the desired *contiguity conditions* between
 them. In the [previous section](#conditions-on-contiguity) we described the different contiguity modes supported by
-Flink, namely *strict*, *relaxed*, and *non-deterministic relaxed*, and how to apply them in looping patterns. To apply 
-them between consecutive patterns, you can use: 
+Flink, namely *strict*, *relaxed*, and *non-deterministic relaxed*, and how to apply them in looping patterns. To apply
+them between consecutive patterns, you can use:
 
-1. `next()`, for *strict*, 
-2. `followedBy()`, for *relaxed*, and 
+1. `next()`, for *strict*,
+2. `followedBy()`, for *relaxed*, and
 3. `followedByAny()`, for *non-deterministic relaxed* contiguity.
 
-or 
+or
 
 1. `notNext()`, if you do not want an event type to directly follow another
 2. `notFollowedBy()`, if you do not want an event type to be anywhere between two other event types
@@ -728,23 +696,20 @@ val relaxedNot: Pattern[Event, _] = start.notFollowedBy("not").where(...)
 </div>
 </div>
 
-Bear in mind that relaxed contiguity means that only the first succeeding matching event will be matched, while
-with non-deterministic relaxed contiguity, multiple matches will be emitted for the same beginning. As an example,
-a pattern `a b`, given the event sequence `"a", "c", "b1", "b2"`, will give the following results:
+Relaxed contiguity means that only the first succeeding matching event will be matched, while with non-deterministic relaxed contiguity, multiple matches will be emitted for the same beginning. As an example, a pattern `a b`, given the event sequence `"a", "c", "b1", "b2"`, will give the following results:
 
-1. Strict Contiguity between `a` and `b`: `{}` (no match) -- the `"c"` after `"a"` causes `"a"` to be discarded.
+1. Strict Contiguity between `a` and `b`: `{}` (no match), the `"c"` after `"a"` causes `"a"` to be discarded.
 
-2. Relaxed Contiguity between `a` and `b`: `{a b1}` -- as relaxed continuity is viewed as "skip non-matching events 
+2. Relaxed Contiguity between `a` and `b`: `{a b1}`, as relaxed continuity is viewed as "skip non-matching events
 till the next matching one".
- 
-3. Non-Deterministic Relaxed Contiguity between `a` and `b`: `{a b1}`, `{a b2}` -- as this is the most general form.
 
-Finally, it is also possible to define a temporal constraint for the pattern to be valid.
-For example, you can define that a pattern should occur within 10 seconds via the `pattern.within()` method. 
+3. Non-Deterministic Relaxed Contiguity between `a` and `b`: `{a b1}`, `{a b2}`, as this is the most general form.
+
+It's also possible to define a temporal constraint for the pattern to be valid.
+For example, you can define that a pattern should occur within 10 seconds via the `pattern.within()` method.
 Temporal patterns are supported for both [processing and event time]({{site.baseurl}}/dev/event_time.html).
 
-{% warn Attention %} A pattern sequence can only have one temporal constraint. If
-multiple such constraints are defined on different individual patterns, then the smallest one is applied.
+{% warn Attention %} A pattern sequence can only have one temporal constraint. If multiple such constraints are defined on different individual patterns, then the smallest is applied.
 
 <div class="codetabs" markdown="1">
 <div data-lang="java" markdown="1">
@@ -784,7 +749,7 @@ Pattern<Event, ?> start = Pattern.<Event>begin("start");
         <tr>
             <td><strong>next()</strong></td>
             <td>
-                <p>Appends a new pattern. A matching event has to directly succeed the previous matching event 
+                <p>Appends a new pattern. A matching event has to directly succeed the previous matching event
                 (strict contiguity):</p>
 {% highlight java %}
 Pattern<Event, ?> next = start.next("middle");
@@ -794,7 +759,7 @@ Pattern<Event, ?> next = start.next("middle");
         <tr>
             <td><strong>followedBy()</strong></td>
             <td>
-                <p>Appends a new pattern. Other events can occur between a matching event and the previous 
+                <p>Appends a new pattern. Other events can occur between a matching event and the previous
                 matching event (relaxed contiguity):</p>
 {% highlight java %}
 Pattern<Event, ?> followedBy = start.followedBy("middle");
@@ -804,8 +769,8 @@ Pattern<Event, ?> followedBy = start.followedBy("middle");
         <tr>
             <td><strong>followedByAny()</strong></td>
             <td>
-                <p>Appends a new pattern. Other events can occur between a matching event and the previous 
-                matching event, and alternative matches will be presented for every alternative matching event 
+                <p>Appends a new pattern. Other events can occur between a matching event and the previous
+                matching event, and alternative matches will be presented for every alternative matching event
                 (non-deterministic relaxed contiguity):</p>
 {% highlight java %}
 Pattern<Event, ?> followedByAny = start.followedByAny("middle");
@@ -815,7 +780,7 @@ Pattern<Event, ?> followedByAny = start.followedByAny("middle");
         <tr>
                     <td><strong>notNext()</strong></td>
                     <td>
-                        <p>Appends a new negative pattern. A matching (negative) event has to directly succeed the 
+                        <p>Appends a new negative pattern. A matching (negative) event has to directly succeed the
                         previous matching event (strict contiguity) for the partial match to be discarded:</p>
 {% highlight java %}
 Pattern<Event, ?> notNext = start.notNext("not");
@@ -826,7 +791,7 @@ Pattern<Event, ?> notNext = start.notNext("not");
                     <td><strong>notFollowedBy()</strong></td>
                     <td>
                         <p>Appends a new negative pattern. A partial matching event sequence will be discarded even
-                        if other events occur between the matching (negative) event and the previous matching event 
+                        if other events occur between the matching (negative) event and the previous matching event
                         (relaxed contiguity):</p>
 {% highlight java %}
 Pattern<Event, ?> notFollowedBy = start.notFllowedBy("not");
@@ -836,7 +801,7 @@ Pattern<Event, ?> notFollowedBy = start.notFllowedBy("not");
        <tr>
           <td><strong>within(time)</strong></td>
           <td>
-              <p>Defines the maximum time interval for an event sequence to match the pattern. If a non-completed event 
+              <p>Defines the maximum time interval for an event sequence to match the pattern. If a non-completed event
               sequence exceeds this time, it is discarded:</p>
 {% highlight java %}
 pattern.within(Time.seconds(10));
@@ -868,7 +833,7 @@ val start = Pattern.begin[Event]("start")
         <tr>
             <td><strong>next()</strong></td>
             <td>
-                <p>Appends a new pattern. A matching event has to directly succeed the previous matching event 
+                <p>Appends a new pattern. A matching event has to directly succeed the previous matching event
                 (strict contiguity):</p>
 {% highlight scala %}
 val next = start.next("middle")
@@ -878,7 +843,7 @@ val next = start.next("middle")
         <tr>
             <td><strong>followedBy()</strong></td>
             <td>
-                <p>Appends a new pattern. Other events can occur between a matching event and the previous 
+                <p>Appends a new pattern. Other events can occur between a matching event and the previous
                 matching event (relaxed contiguity) :</p>
 {% highlight scala %}
 val followedBy = start.followedBy("middle")
@@ -888,19 +853,19 @@ val followedBy = start.followedBy("middle")
         <tr>
                     <td><strong>followedByAny()</strong></td>
                     <td>
-                        <p>Appends a new pattern. Other events can occur between a matching event and the previous 
-                        matching event, and alternative matches will be presented for every alternative matching event 
+                        <p>Appends a new pattern. Other events can occur between a matching event and the previous
+                        matching event, and alternative matches will be presented for every alternative matching event
                         (non-deterministic relaxed contiguity):</p>
 {% highlight scala %}
 val followedByAny = start.followedByAny("middle");
 {% endhighlight %}
                             </td>
                 </tr>
-                
+
                 <tr>
                                     <td><strong>notNext()</strong></td>
                                     <td>
-                                        <p>Appends a new negative pattern. A matching (negative) event has to directly succeed the 
+                                        <p>Appends a new negative pattern. A matching (negative) event has to directly succeed the
                                         previous matching event (strict contiguity) for the partial match to be discarded:</p>
 {% highlight scala %}
 val notNext = start.notNext("not")
@@ -911,18 +876,18 @@ val notNext = start.notNext("not")
                                     <td><strong>notFollowedBy()</strong></td>
                                     <td>
                                         <p>Appends a new negative pattern. A partial matching event sequence will be discarded even
-                                        if other events occur between the matching (negative) event and the previous matching event 
+                                        if other events occur between the matching (negative) event and the previous matching event
                                         (relaxed contiguity):</p>
 {% highlight scala %}
 val notFollowedBy = start.notFllowedBy("not")
 {% endhighlight %}
                                     </td>
                                 </tr>
-        
+
        <tr>
           <td><strong>within(time)</strong></td>
           <td>
-              <p>Defines the maximum time interval for an event sequence to match the pattern. If a non-completed event 
+              <p>Defines the maximum time interval for an event sequence to match the pattern. If a non-completed event
               sequence exceeds this time, it is discarded:</p>
 {% highlight scala %}
 pattern.within(Time.seconds(10))
@@ -937,9 +902,7 @@ pattern.within(Time.seconds(10))
 
 ## Detecting Patterns
 
-After specifying the pattern sequence you are looking for, it is time to apply it to your input stream to detect 
-potential matches. In order to run a stream of events against your pattern sequence, you have to create a `PatternStream`.
-Given an input stream `input` and a pattern `pattern`, you create the `PatternStream` by calling:
+After specifying the pattern sequence you are looking for, it is time to apply it to your input stream to detect potential matches. To run a stream of events against your pattern sequence, you have to create a `PatternStream`. Given an input stream `input`, a pattern `pattern` and an optional comparator `comparator` used to sort events with the same timestamp in case of EventTime or that arrived at the same moment, you create the `PatternStream` by calling:
 
 <div class="codetabs" markdown="1">
 <div data-lang="java" markdown="1">
@@ -963,8 +926,8 @@ val patternStream: PatternStream[Event] = CEP.pattern(input, pattern)
 
 The input stream can be *keyed* or *non-keyed* depending on your use-case.
 
-{% warn Attention %} Applying your pattern on a non-keyed stream will result in a job with
-parallelism equal to 1.
+{% warn Attention %} Applying your pattern on a non-keyed stream will result in a job with parallelism equal to 1.
+
 
 ### Selecting from Patterns
 
@@ -974,11 +937,9 @@ Once you have obtained a `PatternStream` you can select from detected event sequ
 <div data-lang="java" markdown="1">
 The `select()` method requires a `PatternSelectFunction` implementation.
 A `PatternSelectFunction` has a `select` method which is called for each matching event sequence.
-It receives a match in the form of `Map<String, List<IN>>` where the key is the name of each pattern in your pattern 
-sequence and the value is a list of all accepted events for that pattern (`IN` is the type of your input elements). 
-The events for a given pattern are ordered by timestamp. The reason for returning a list of accepted events for each 
-pattern is that when using looping patterns (e.g. `oneToMany()` and `times()`), more than one event may be accepted for a 
-given pattern. The selection function returns exactly one result.
+It receives a match in the form of `Map<String, List<IN>>` where the key is the name of each pattern in your pattern
+sequence and the value is a list of all accepted events for that pattern (`IN` is the type of your input elements).
+The events for a given pattern are ordered by timestamp. The reason for returning a list of accepted events for each pattern is that when using looping patterns (e.g. `oneToMany()` and `times()`), more than one event may be accepted for a given pattern. The selection function returns exactly one result.
 
 {% highlight java %}
 class MyPatternSelectFunction<IN, OUT> implements PatternSelectFunction<IN, OUT> {
@@ -991,8 +952,8 @@ class MyPatternSelectFunction<IN, OUT> implements PatternSelectFunction<IN, OUT>
 }
 {% endhighlight %}
 
-A `PatternFlatSelectFunction` is similar to the `PatternSelectFunction`, with the only distinction that it can return an 
-arbitrary number of results. In order to do this, the `select` method has an additional `Collector` parameter which is 
+A `PatternFlatSelectFunction` is similar to the `PatternSelectFunction`, with the only distinction that it can return an
+arbitrary number of results. To do this, the `select` method has an additional `Collector` parameter which is
 used to forward your output elements downstream.
 
 {% highlight java %}
@@ -1012,11 +973,10 @@ class MyPatternFlatSelectFunction<IN, OUT> implements PatternFlatSelectFunction<
 
 <div data-lang="scala" markdown="1">
 The `select()` method takes a selection function as argument, which is called for each matching event sequence.
-It receives a match in the form of `Map[String, Iterable[IN]]` where the key is the name of each pattern in your pattern 
-sequence and the value is an Iterable over all accepted events for that pattern (`IN` is the type of your input elements). 
-The events for a given pattern are ordered by timestamp. The reason for returning an iterable of accepted events for each 
-pattern is that when using looping patterns (e.g. `oneToMany()` and `times()`), more than one event may be accepted for a 
-given pattern. The selection function returns exactly one result per call.
+It receives a match in the form of `Map[String, Iterable[IN]]` where the key is the name of each pattern in your pattern
+sequence and the value is an Iterable over all accepted events for that pattern (`IN` is the type of your input elements).
+
+The events for a given pattern are ordered by timestamp. The reason for returning an iterable of accepted events for each pattern is that when using looping patterns (e.g. `oneToMany()` and `times()`), more than one event may be accepted for a given pattern. The selection function returns exactly one result per call.
 
 {% highlight scala %}
 def selectFn(pattern : Map[String, Iterable[IN]]): OUT = {
@@ -1026,8 +986,8 @@ def selectFn(pattern : Map[String, Iterable[IN]]): OUT = {
 }
 {% endhighlight %}
 
-The `flatSelect` method is similar to the `select` method. Their only difference is that the function passed to the 
-`flatSelect` method can return an arbitrary number of results per call. In order to do this, the function for 
+The `flatSelect` method is similar to the `select` method. Their only difference is that the function passed to the
+`flatSelect` method can return an arbitrary number of results per call. In order to do this, the function for
 `flatSelect` has an additional `Collector` parameter which is used to forward your output elements downstream.
 
 {% highlight scala %}
@@ -1044,18 +1004,18 @@ def flatSelectFn(pattern : Map[String, Iterable[IN]], collector : Collector[OUT]
 
 ### Handling Timed Out Partial Patterns
 
-Whenever a pattern has a window length attached via the `within` keyword, it is possible that partial event sequences 
-are discarded because they exceed the window length. In order to react to these timed out partial matches the `select` 
-and `flatSelect` API calls allow a timeout handler to be specified. This timeout handler is called for each timed out 
-partial event sequence. The timeout handler receives all the events that have been matched so far by the pattern, and 
+Whenever a pattern has a window length attached via the `within` keyword, it is possible that partial event sequences
+are discarded because they exceed the window length. To react to these timed out partial matches the `select`
+and `flatSelect` API calls allow you to specify a timeout handler. This timeout handler is called for each timed out
+partial event sequence. The timeout handler receives all the events that have been matched so far by the pattern, and
 the timestamp when the timeout was detected.
 
 <div class="codetabs" markdown="1">
 <div data-lang="java" markdown="1">
-In order to treat partial patterns, the `select` and `flatSelect` API calls offer an overloaded version which takes as 
-the first parameter a `PatternTimeoutFunction`/`PatternFlatTimeoutFunction` and as second parameter the known 
-`PatternSelectFunction`/`PatternFlatSelectFunction`. The return type of the timeout function can be different from the 
-select function. The timeout event and the select event are wrapped in `Either.Left` and `Either.Right` respectively 
+To treat partial patterns, the `select` and `flatSelect` API calls offer an overloaded version which takes as
+the first parameter a `PatternTimeoutFunction`/`PatternFlatTimeoutFunction` and as second parameter the known
+`PatternSelectFunction`/`PatternFlatSelectFunction`. The return type of the timeout function can be different from the
+select function. The timeout event and the select event are wrapped in `Either.Left` and `Either.Right` respectively
 so that the resulting data stream is of type `org.apache.flink.types.Either`.
 
 {% highlight java %}
@@ -1075,7 +1035,7 @@ DataStream<Either<TimeoutEvent, ComplexEvent>> flatResult = patternStream.flatSe
 </div>
 
 <div data-lang="scala" markdown="1">
-In order to treat partial patterns, the `select` API call offers an overloaded version which takes as the first parameter a timeout function and as second parameter a selection function.
+To treat partial patterns, the `select` API call offers an overloaded version which takes as the first parameter a timeout function and as second parameter a selection function.
 The timeout function is called with a map of string-event pairs of the partial match which has timed out and a long indicating when the timeout occurred.
 The string is defined by the name of the pattern to which the event has been matched.
 The timeout function returns exactly one result per call.
@@ -1093,8 +1053,8 @@ DataStream[Either[TimeoutEvent, ComplexEvent]] result = patternStream.select{
 {% endhighlight %}
 
 The `flatSelect` API call offers the same overloaded version which takes as the first parameter a timeout function and as second parameter a selection function.
-In contrast to the `select` functions, the `flatSelect` functions are called with a `Collector`.
-The collector can be used to emit an arbitrary number of events.
+
+In contrast to the `select` functions, the `flatSelect` functions are called with a `Collector`. You can use the collector to emit an arbitrary number of events.
 
 {% highlight scala %}
 val patternStream: PatternStream[Event] = CEP.pattern(input, pattern)
@@ -1113,21 +1073,16 @@ DataStream[Either[TimeoutEvent, ComplexEvent]] result = patternStream.flatSelect
 
 ## Handling Lateness in Event Time
 
-In `CEP` the order in which elements are processed matters. To guarantee that elements are processed in the correct order
-when working in event time, an incoming element is initially put in a buffer where elements are *sorted in ascending 
-order based on their timestamp*, and when a watermark arrives, all the elements in this buffer with timestamps smaller 
-than that of the watermark are processed. This implies that elements between watermarks are processed in event-time order. 
+In `CEP` the order in which elements are processed matters. To guarantee that elements are processed in the correct order when working in event time, an incoming element is initially put in a buffer where elements are *sorted in ascending order based on their timestamp*, and when a watermark arrives, all the elements in this buffer with timestamps smaller than that of the watermark are processed. This implies that elements between watermarks are processed in event-time order.
 
 {% warn Attention %} The library assumes correctness of the watermark when working in event time.
 
-To also guarantee that elements across watermarks are processed in event-time order, Flink's CEP library assumes 
-*correctness of the watermark*, and considers as *late* elements whose timestamp is smaller than that of the last 
-seen watermark. Late elements are not further processed.
+To guarantee that elements across watermarks are processed in event-time order, Flink's CEP library assumes *correctness of the watermark*, and considers as *late* elements whose timestamp is smaller than that of the last seen watermark. Late elements are not further processed.
 
 ## Examples
 
-The following example detects the pattern `start, middle(name = "error") -> end(name = "critical")` on a keyed data 
-stream of `Events`. The events are keyed by their ids and a valid pattern has to occur within 10 seconds.
+The following example detects the pattern `start, middle(name = "error") -> end(name = "critical")` on a keyed data
+stream of `Events`. The events are keyed by their `id`s and a valid pattern has to occur within 10 seconds.
 The whole processing is done with event time.
 
 <div class="codetabs" markdown="1">
@@ -1192,20 +1147,20 @@ val alerts = patternStream.select(createAlert(_)))
 
 ## Migrating from an older Flink version
 
-The CEP library in Flink-1.3 ships with a number of new features which have led to some changes in the API. Here we 
-describe the changes that you need to make to your old CEP jobs, in order to be able to run them with Flink-1.3. After 
-making these changes and recompiling your job, you will be able to resume its execution from a savepoint taken with the 
+The CEP library in Flink-1.3 ships with a number of new features which have led to some changes in the API. Here we
+describe the changes that you need to make to your old CEP jobs, in order to be able to run them with Flink-1.3. After
+making these changes and recompiling your job, you will be able to resume its execution from a savepoint taken with the
 old version of your job, *i.e.* without having to re-process your past data.
 
 The changes required are:
 
-1. Change your conditions (the ones in the `where(...)` clause) to extend the `SimpleCondition` class instead of 
+1. Change your conditions (the ones in the `where(...)` clause) to extend the `SimpleCondition` class instead of
 implementing the `FilterFunction` interface.
 
 2. Change your functions provided as arguments to the `select(...)` and `flatSelect(...)` methods to expect a list of
 events associated with each pattern (`List` in `Java`, `Iterable` in `Scala`). This is because with the addition of
 the looping patterns, multiple input events can match a single (looping) pattern.
 
-3. The `followedBy()` in Flink 1.1 and 1.2 implied `non-deterministic relaxed contiguity` (see 
-[here](#conditions-on-contiguity)). In Flink 1.3 this has changed and `followedBy()` implies `relaxed contiguity`, 
-while `followedByAny()` should be used if `non-deterministic relaxed contiguity` is required.
\ No newline at end of file
+3. The `followedBy()` in Flink 1.1 and 1.2 implied `non-deterministic relaxed contiguity` (see
+[here](#conditions-on-contiguity)). In Flink 1.3 this has changed and `followedBy()` implies `relaxed contiguity`,
+while `followedByAny()` should be used if `non-deterministic relaxed contiguity` is required.