提交 345de772 编写于 作者: M Mikhail Lipkovich 提交者: Aljoscha Krettek

[FLINK-7567] [scala] Remove keepPartitioning parameter from DataStream.iterate()

上级 42cc3a2a
......@@ -87,7 +87,8 @@ public class FeedbackTransformation<T> extends StreamTransformation<T> {
throw new UnsupportedOperationException(
"Parallelism of the feedback stream must match the parallelism of the original" +
" stream. Parallelism of original stream: " + this.getParallelism() +
"; parallelism of feedback stream: " + transform.getParallelism());
"; parallelism of feedback stream: " + transform.getParallelism() +
". Parallelism can be modified using DataStream#setParallelism() method");
}
feedbackEdges.add(transform);
......
......@@ -228,6 +228,10 @@ under the License.
<excludes combine.children="append">
<!-- Exclude generated classes from api compatibility checks -->
<exclude>*\$\$anon\$*</exclude>
<!-- Ignore method which was created automatically by Scala for default value calculation.
Can be removed once https://github.com/siom79/japicmp/issues/176 will be fixed -->
<exclude>org.apache.flink.streaming.api.scala.DataStream#iterate\$default\$3()</exclude>
</excludes>
</parameter>
</configuration>
......
......@@ -499,23 +499,22 @@ class DataStream[T](stream: JavaStream[T]) {
* stepfunction: initialStream => (feedback, output)
*
* A common pattern is to use output splitting to create feedback and output DataStream.
* Please refer to the .split(...) method of the DataStream
* Please refer to the [[split]] method of the DataStream
*
* By default a DataStream with iteration will never terminate, but the user
* can use the maxWaitTime parameter to set a max waiting time for the iteration head.
* If no data received in the set time the stream terminates.
*
* By default the feedback partitioning is set to match the input, to override this set
* the keepPartitioning flag to true
*
* Parallelism of the feedback stream must match the parallelism of the original stream.
* Please refer to the [[setParallelism]] method for parallelism modification
*/
@PublicEvolving
def iterate[R](stepFunction: DataStream[T] => (DataStream[T], DataStream[R]),
maxWaitTimeMillis:Long = 0,
keepPartitioning: Boolean = false) : DataStream[R] = {
maxWaitTimeMillis:Long = 0) : DataStream[R] = {
val iterativeStream = stream.iterate(maxWaitTimeMillis)
val (feedback, output) = stepFunction(new DataStream[T](iterativeStream))
iterativeStream.closeWith(feedback.javaStream)
output
}
......
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册