Skip to content

Commit

Permalink
[docs] Note that numberOfExecutionRetries and executionRetryDelay are…
Browse files Browse the repository at this point in the history
… deprecated.

And some other minor fixes and deduplication.
  • Loading branch information
rohitagarwal003 authored and mxm committed Dec 7, 2016
1 parent 91414d9 commit c024b0b
Show file tree
Hide file tree
Showing 3 changed files with 7 additions and 24 deletions.
4 changes: 2 additions & 2 deletions docs/dev/api_concepts.md
Original file line number Diff line number Diff line change
Expand Up @@ -962,9 +962,9 @@ With the closure cleaner disabled, it might happen that an anonymous user functi

- `getMaxParallelism()` / `setMaxParallelism(int parallelism)` Set the default maximum parallelism for the job. This setting determines the maximum degree of parallelism and specifies the upper limit for dynamic scaling.

- `getNumberOfExecutionRetries()` / `setNumberOfExecutionRetries(int numberOfExecutionRetries)` Sets the number of times that failed tasks are re-executed. A value of zero effectively disables fault tolerance. A value of `-1` indicates that the system default value (as defined in the configuration) should be used.
- `getNumberOfExecutionRetries()` / `setNumberOfExecutionRetries(int numberOfExecutionRetries)` Sets the number of times that failed tasks are re-executed. A value of zero effectively disables fault tolerance. A value of `-1` indicates that the system default value (as defined in the configuration) should be used. This is deprecated, use [restart strategies]({{ site.baseurl }}/setup/fault_tolerance.html#restart-strategies) instead.

- `getExecutionRetryDelay()` / `setExecutionRetryDelay(long executionRetryDelay)` Sets the delay in milliseconds that the system waits after a job has failed, before re-executing it. The delay starts after all tasks have been successfully been stopped on the TaskManagers, and once the delay is past, the tasks are re-started. This parameter is useful to delay re-execution in order to let certain time-out related failures surface fully (like broken connections that have not fully timed out), before attempting a re-execution and immediately failing again due to the same problem. This parameter only has an effect if the number of execution re-tries is one or more.
- `getExecutionRetryDelay()` / `setExecutionRetryDelay(long executionRetryDelay)` Sets the delay in milliseconds that the system waits after a job has failed, before re-executing it. The delay starts after all tasks have been successfully been stopped on the TaskManagers, and once the delay is past, the tasks are re-started. This parameter is useful to delay re-execution in order to let certain time-out related failures surface fully (like broken connections that have not fully timed out), before attempting a re-execution and immediately failing again due to the same problem. This parameter only has an effect if the number of execution re-tries is one or more. This is deprecated, use [restart strategies]({{ site.baseurl }}/setup/fault_tolerance.html#restart-strategies) instead.

- `getExecutionMode()` / `setExecutionMode()`. The default execution mode is PIPELINED. Sets the execution mode to execute the program. The execution mode defines whether data exchanges are performed in a batch or on a pipelined manner.

Expand Down
6 changes: 3 additions & 3 deletions docs/dev/datastream_api.md
Original file line number Diff line number Diff line change
Expand Up @@ -342,7 +342,7 @@ allWindowedStream.apply (new AllWindowFunction<Tuple2<String,Integer>, Integer,
<td>
<p>Applies a functional reduce function to the window and returns the reduced value.</p>
{% highlight java %}
windowedStream.reduce (new ReduceFunction<Tuple2<String,Integer>() {
windowedStream.reduce (new ReduceFunction<Tuple2<String,Integer>>() {
public Tuple2<String, Integer> reduce(Tuple2<String, Integer> value1, Tuple2<String, Integer> value2) throws Exception {
return new Tuple2<String,Integer>(value1.f0, value1.f1 + value2.f1);
}
Expand Down Expand Up @@ -388,7 +388,7 @@ windowedStream.maxBy("key");
<tr>
<td><strong>Union</strong><br>DataStream* &rarr; DataStream</td>
<td>
<p>Union of two or more data streams creating a new stream containing all the elements from all the streams. Node: If you union a data stream
<p>Union of two or more data streams creating a new stream containing all the elements from all the streams. Note: If you union a data stream
with itself you will get each element twice in the resulting stream.</p>
{% highlight java %}
dataStream.union(otherStream1, otherStream2, ...);
Expand Down Expand Up @@ -737,7 +737,7 @@ windowedStream.maxBy("key")
<tr>
<td><strong>Union</strong><br>DataStream* &rarr; DataStream</td>
<td>
<p>Union of two or more data streams creating a new stream containing all the elements from all the streams. Node: If you union a data stream
<p>Union of two or more data streams creating a new stream containing all the elements from all the streams. Note: If you union a data stream
with itself you will get each element twice in the resulting stream.</p>
{% highlight scala %}
dataStream.union(otherStream1, otherStream2, ...)
Expand Down
21 changes: 2 additions & 19 deletions docs/setup/fault_tolerance.md
Original file line number Diff line number Diff line change
Expand Up @@ -45,9 +45,6 @@ To enable checkpointing, call `enableCheckpointing(n)` on the `StreamExecutionEn

Other parameters for checkpointing include:

- *Number of retries*: The `setNumberOfExecutionRerties()` method defines how many times the job is restarted after a failure.
When checkpointing is activated, but this value is not explicitly set, the job is restarted infinitely often.

- *exactly-once vs. at-least-once*: You can optionally pass a mode to the `enableCheckpointing(n)` method to choose between the two guarantee levels.
Exactly-once is preferrable for most applications. At-least-once may be relevant for certain super-low-latency (consistently few milliseconds) applications.

Expand Down Expand Up @@ -307,12 +304,12 @@ restart-strategy: fixed-delay
<tbody>
<tr>
<td><it>restart-strategy.fixed-delay.attempts</it></td>
<td>Number of restart attempts</td>
<td>The number of times that Flink retries the execution before the job is declared as failed</td>
<td>1</td>
</tr>
<tr>
<td><it>restart-strategy.fixed-delay.delay</it></td>
<td>Delay between two consecutive restart attempts</td>
<td>Delay between two consecutive restart attempts. Delaying the retry means that after a failed execution, the re-execution does not start immediately, but only after a certain delay. Delaying the retries can be helpful when the program interacts with external systems where for example connections or pending transactions should reach a timeout before re-execution is attempted.</td>
<td><it>akka.ask.timeout</it></td>
</tr>
</tbody>
Expand Down Expand Up @@ -346,20 +343,6 @@ env.setRestartStrategy(RestartStrategies.fixedDelayRestart(
</div>
</div>

#### Restart Attempts

The number of times that Flink retries the execution before the job is declared as failed is configurable via the *restart-strategy.fixed-delay.attempts* parameter.

The default value is **1**.

#### Retry Delays

Execution retries can be configured to be delayed. Delaying the retry means that after a failed execution, the re-execution does not start immediately, but only after a certain delay.

Delaying the retries can be helpful when the program interacts with external systems where for example connections or pending transactions should reach a timeout before re-execution is attempted.

The default value is the value of *akka.ask.timeout*.

{% top %}

### Failure Rate Restart Strategy
Expand Down

0 comments on commit c024b0b

Please sign in to comment.