You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@beam.apache.org by mx...@apache.org on 2019/06/15 10:09:44 UTC

[beam] 01/01: Merge pull request #8850: [BEAM-7144] Fix for rescaling problem on Flink >= 1.6

This is an automated email from the ASF dual-hosted git repository.

mxm pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/beam.git

commit 89c5abfe12dea9bc1e45cafe1ccfbb9b7ecd5541
Merge: 4d2eba2 1642655
Author: Maximilian Michels <mx...@apache.org>
AuthorDate: Sat Jun 15 12:09:20 2019 +0200

    Merge pull request #8850: [BEAM-7144] Fix for rescaling problem on Flink >= 1.6

 .../translation/types/CoderTypeSerializer.java     |   2 +-
 .../translation/types/CoderTypeSerializer.java     |   2 +-
 .../wrappers/streaming/DoFnOperator.java           |  13 +++
 .../streaming/ExecutableStageDoFnOperator.java     |  26 +++++
 .../streaming/KeyedPushedBackElementsHandler.java  |  38 +++----
 .../stableinput/KeyedBufferingElementsHandler.java |  45 +++-----
 .../streaming/state/FlinkStateInternals.java       | 118 +++++++++++++++++++++
 .../beam/runners/flink/FlinkSavepointTest.java     |  78 ++++++++------
 8 files changed, 232 insertions(+), 90 deletions(-)

diff --cc runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ExecutableStageDoFnOperator.java
index 230a65e,735e961..0fb08d2
--- a/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ExecutableStageDoFnOperator.java
+++ b/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ExecutableStageDoFnOperator.java
@@@ -82,8 -84,10 +84,11 @@@ import org.apache.beam.sdk.values.KV
  import org.apache.beam.sdk.values.PCollectionView;
  import org.apache.beam.sdk.values.TupleTag;
  import org.apache.beam.sdk.values.WindowingStrategy;
 +import org.apache.beam.vendor.grpc.v1p13p1.com.google.protobuf.ByteString;
  import org.apache.beam.vendor.guava.v20_0.com.google.common.base.Preconditions;
+ import org.apache.beam.vendor.sdk.v2.sdk.extensions.protobuf.ByteStringCoder;
+ import org.apache.flink.api.common.state.ListStateDescriptor;
+ import org.apache.flink.api.common.typeutils.base.StringSerializer;
  import org.apache.flink.api.java.functions.KeySelector;
  import org.apache.flink.runtime.state.KeyedStateBackend;
  import org.apache.flink.streaming.api.operators.InternalTimer;