(beam) branch master updated: Spark Runner: Change to use partitioner in GroupNonMergingWindowsFunctions#groupByKeyInGlobalWindow (#32610)
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new a2710ed16e6 Spark Runner: Change to use partitioner in GroupNonMergingWindowsFunctions#groupByKeyInGlobalWindow (#32610) a2710ed16e6 is described below commit a2710ed16e61f171016ae1cf56afd9eefd0cf245 Author: twosom <72733442+two...@users.noreply.github.com> AuthorDate: Fri Oct 4 16:28:41 2024 +0900 Spark Runner: Change to use partitioner in GroupNonMergingWindowsFunctions#groupByKeyInGlobalWindow (#32610) * change GroupNonMergingWindowsFunctions#groupByKeyInGlobalWindow to use partitioner --- .../GroupNonMergingWindowsFunctions.java | 25 +- .../GroupNonMergingWindowsFunctionsTest.java | 57 ++ 2 files changed, 70 insertions(+), 12 deletions(-) diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/GroupNonMergingWindowsFunctions.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/GroupNonMergingWindowsFunctions.java index 0a8e7d8a159..2461d5cc8d6 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/GroupNonMergingWindowsFunctions.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/GroupNonMergingWindowsFunctions.java @@ -275,17 +275,18 @@ public class GroupNonMergingWindowsFunctions { new Tuple2<>( new ByteArray(CoderHelpers.toByteArray(wv.getValue().getKey(), keyCoder)), CoderHelpers.toByteArray(wv.getValue().getValue(), valueCoder))); -return rawKeyValues -.groupByKey() -.map( -kvs -> -WindowedValue.timestampedValueInGlobalWindow( -KV.of( -CoderHelpers.fromByteArray(kvs._1.getValue(), keyCoder), -Iterables.transform( -kvs._2, -encodedValue -> CoderHelpers.fromByteArray(encodedValue, valueCoder))), -GlobalWindow.INSTANCE.maxTimestamp(), -PaneInfo.ON_TIME_AND_ONLY_FIRING)); + +JavaPairRDD> grouped = +(partitioner == null) ? rawKeyValues.groupByKey() : rawKeyValues.groupByKey(partitioner); +return grouped.map( +kvs -> +WindowedValue.timestampedValueInGlobalWindow( +KV.of( +CoderHelpers.fromByteArray(kvs._1.getValue(), keyCoder), +Iterables.transform( +kvs._2, +encodedValue -> CoderHelpers.fromByteArray(encodedValue, valueCoder))), +GlobalWindow.INSTANCE.maxTimestamp(), +PaneInfo.ON_TIME_AND_ONLY_FIRING)); } } diff --git a/runners/spark/src/test/java/org/apache/beam/runners/spark/translation/GroupNonMergingWindowsFunctionsTest.java b/runners/spark/src/test/java/org/apache/beam/runners/spark/translation/GroupNonMergingWindowsFunctionsTest.java index fd299924af9..ed7bc078564 100644 --- a/runners/spark/src/test/java/org/apache/beam/runners/spark/translation/GroupNonMergingWindowsFunctionsTest.java +++ b/runners/spark/src/test/java/org/apache/beam/runners/spark/translation/GroupNonMergingWindowsFunctionsTest.java @@ -18,6 +18,12 @@ package org.apache.beam.runners.spark.translation; import static org.junit.Assert.assertEquals; +import static org.mockito.Mockito.any; +import static org.mockito.Mockito.mock; +import static org.mockito.Mockito.never; +import static org.mockito.Mockito.times; +import static org.mockito.Mockito.verify; +import static org.mockito.Mockito.when; import java.util.Arrays; import java.util.Iterator; @@ -39,6 +45,9 @@ import org.apache.beam.sdk.util.WindowedValue.FullWindowedValueCoder; import org.apache.beam.sdk.values.KV; import org.apache.beam.sdk.values.WindowingStrategy; import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.primitives.Bytes; +import org.apache.spark.Partitioner; +import org.apache.spark.api.java.JavaPairRDD; +import org.apache.spark.api.java.JavaRDD; import org.joda.time.Duration; import org.joda.time.Instant; import org.junit.Assert; @@ -112,6 +121,54 @@ public class GroupNonMergingWindowsFunctionsTest { } } + @Test + @SuppressWarnings({"rawtypes", "unchecked"}) + public void testGroupByKeyInGlobalWindowWithPartitioner() { +// mocking +Partitioner mockPartitioner = mock(Partitioner.class); +JavaRDD mockRdd = mock(JavaRDD.class); +Coder mockKeyCoder = mock(Coder.class); +Coder mockValueCoder = mock(Coder.class); +JavaPairRDD mockRawKeyValues = mock(JavaPairRDD.class); +JavaPairRDD mockGrouped = mock(JavaPairRDD.class); + +when(mockRdd.mapToPair(any())).thenReturn(mockRawKeyVal
(beam) branch master updated (8ff7f0d75e4 -> 7a4850dcdae)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 8ff7f0d75e4 Bump github.com/docker/docker in /sdks (#32176) add b0f2683cda1 GitHub issue #30257 add 7a4850dcdae Merge pull request #32074: Fixes GitHub issue #30257 No new revisions were added by this update. Summary of changes: .../sdk/transforms/SerializableComparator.java | 21 +++- .../sdk/transforms/SerializableComparatorTest.java | 63 ++ 2 files changed, 83 insertions(+), 1 deletion(-) create mode 100644 sdks/java/core/src/test/java/org/apache/beam/sdk/transforms/SerializableComparatorTest.java
(beam) branch master updated (0790d698251 -> 836e77e755b)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 0790d698251 Merge pull request #31394: #31112 drop flink 1.14 add 5a822adc269 [flink] #31390 emit watermark with empty source new 836e77e755b Merge pull request #31391: [flink] #31390 emit watermark with empty source The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../streaming/io/source/FlinkSourceReaderBase.java | 9 ++-- .../unbounded/FlinkUnboundedSourceReader.java | 8 ++-- .../streaming/io/source/EmptyUnboundedSource.java | 4 +- .../unbounded/FlinkUnboundedSourceReaderTest.java | 54 -- 4 files changed, 63 insertions(+), 12 deletions(-)
(beam) 01/01: Merge pull request #31391: [flink] #31390 emit watermark with empty source
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 836e77e755ba52fb55910308ba5324b01f787079 Merge: 0790d698251 5a822adc269 Author: Jan Lukavský AuthorDate: Sat May 25 19:32:15 2024 +0200 Merge pull request #31391: [flink] #31390 emit watermark with empty source .../streaming/io/source/FlinkSourceReaderBase.java | 9 ++-- .../unbounded/FlinkUnboundedSourceReader.java | 8 ++-- .../streaming/io/source/EmptyUnboundedSource.java | 4 +- .../unbounded/FlinkUnboundedSourceReaderTest.java | 54 -- 4 files changed, 63 insertions(+), 12 deletions(-)
(beam) 01/01: Merge pull request #31394: #31112 drop flink 1.14
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 0790d698251f82809c5f4cecaaef5e74c316a0eb Merge: cf3592bbfa0 c10f5d40456 Author: Jan Lukavský AuthorDate: Sat May 25 17:31:02 2024 +0200 Merge pull request #31394: #31112 drop flink 1.14 .../test-properties.json | 2 +- gradle.properties | 2 +- .../runner-concepts/description.md | 2 +- runners/flink/1.14/build.gradle| 25 -- .../flink/1.14/job-server-container/build.gradle | 26 --- runners/flink/1.14/job-server/build.gradle | 31 .../streaming/AbstractStreamOperatorCompat.java| 91 -- .../streaming/ProcessingTimeCallbackCompat.java| 22 -- .../io/source/compat/FlinkSourceCompat.java| 28 --- .../io/source/compat/SplitEnumeratorCompat.java| 27 --- .../streaming/io/source/compat/package-info.java | 20 - .../beam/runners/flink/MiniClusterCompat.java | 29 --- .../runners/flink/metrics/MetricGroupWrapper.java | 31 .../translation/types/CoderTypeSerializer.java | 0 .../streaming/ProcessingTimeCallbackCompat.java| 22 -- .../beam/runners/flink/MiniClusterCompat.java | 30 --- .../flink/FlinkStreamingTransformTranslators.java | 8 +- .../wrappers/streaming/DoFnOperator.java | 57 -- .../streaming/io/UnboundedSourceWrapper.java | 8 +- .../streaming/io/source/FlinkSourceReaderBase.java | 3 +- .../io/source/FlinkSourceSplitEnumerator.java | 9 +-- .../beam/runners/flink/FlinkSavepointTest.java | 6 +- .../beam/runners/flink/RemoteMiniClusterImpl.java | 0 .../flink/metrics/FlinkMetricContainerTest.java| 3 +- .../runners/flink/streaming/StreamSources.java | 0 .../translation/types/CoderTypeSerializerTest.java | 0 .../streaming/io/UnboundedSourceWrapperTest.java | 4 +- .../io/source/FlinkSourceReaderTestBase.java | 20 ++--- .../streaming/io/source/SourceTestMetrics.java}| 2 +- .../bounded/FlinkBoundedSourceReaderTest.java | 2 +- .../unbounded/FlinkUnboundedSourceReaderTest.java | 2 +- sdks/go/examples/wasm/README.md| 2 +- .../python/apache_beam/options/pipeline_options.py | 2 +- sdks/typescript/src/apache_beam/runners/flink.ts | 2 +- settings.gradle.kts| 4 - .../site/content/en/documentation/runners/flink.md | 9 ++- .../shortcodes/flink_java_pipeline_options.html| 10 +++ .../shortcodes/flink_python_pipeline_options.html | 10 +++ 38 files changed, 115 insertions(+), 436 deletions(-)
(beam) branch master updated (cf3592bbfa0 -> 0790d698251)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from cf3592bbfa0 Fix an incompatibility with hamcrest 2.2 (#31395) add c10f5d40456 [runners-flink] Drop Flnk 1.14 and clean up code (#31112) new 0790d698251 Merge pull request #31394: #31112 drop flink 1.14 The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../test-properties.json | 2 +- gradle.properties | 2 +- .../runner-concepts/description.md | 2 +- runners/flink/1.14/build.gradle| 25 -- .../flink/1.14/job-server-container/build.gradle | 26 --- runners/flink/1.14/job-server/build.gradle | 31 .../streaming/AbstractStreamOperatorCompat.java| 91 -- .../streaming/ProcessingTimeCallbackCompat.java| 22 -- .../io/source/compat/FlinkSourceCompat.java| 28 --- .../io/source/compat/SplitEnumeratorCompat.java| 27 --- .../streaming/io/source/compat/package-info.java | 20 - .../beam/runners/flink/MiniClusterCompat.java | 29 --- .../runners/flink/metrics/MetricGroupWrapper.java | 31 .../translation/types/CoderTypeSerializer.java | 0 .../streaming/ProcessingTimeCallbackCompat.java| 22 -- .../beam/runners/flink/MiniClusterCompat.java | 30 --- .../flink/FlinkStreamingTransformTranslators.java | 8 +- .../wrappers/streaming/DoFnOperator.java | 57 -- .../streaming/io/UnboundedSourceWrapper.java | 8 +- .../streaming/io/source/FlinkSourceReaderBase.java | 3 +- .../io/source/FlinkSourceSplitEnumerator.java | 9 +-- .../beam/runners/flink/FlinkSavepointTest.java | 6 +- .../beam/runners/flink/RemoteMiniClusterImpl.java | 0 .../flink/metrics/FlinkMetricContainerTest.java| 3 +- .../runners/flink/streaming/StreamSources.java | 0 .../translation/types/CoderTypeSerializerTest.java | 0 .../streaming/io/UnboundedSourceWrapperTest.java | 4 +- .../io/source/FlinkSourceReaderTestBase.java | 20 ++--- .../streaming/io/source/SourceTestMetrics.java}| 2 +- .../bounded/FlinkBoundedSourceReaderTest.java | 2 +- .../unbounded/FlinkUnboundedSourceReaderTest.java | 2 +- sdks/go/examples/wasm/README.md| 2 +- .../python/apache_beam/options/pipeline_options.py | 2 +- sdks/typescript/src/apache_beam/runners/flink.ts | 2 +- settings.gradle.kts| 4 - .../site/content/en/documentation/runners/flink.md | 9 ++- .../shortcodes/flink_java_pipeline_options.html| 10 +++ .../shortcodes/flink_python_pipeline_options.html | 10 +++ 38 files changed, 115 insertions(+), 436 deletions(-) delete mode 100644 runners/flink/1.14/build.gradle delete mode 100644 runners/flink/1.14/job-server-container/build.gradle delete mode 100644 runners/flink/1.14/job-server/build.gradle delete mode 100644 runners/flink/1.14/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/AbstractStreamOperatorCompat.java delete mode 100644 runners/flink/1.14/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ProcessingTimeCallbackCompat.java delete mode 100644 runners/flink/1.14/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/io/source/compat/FlinkSourceCompat.java delete mode 100644 runners/flink/1.14/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/io/source/compat/SplitEnumeratorCompat.java delete mode 100644 runners/flink/1.14/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/io/source/compat/package-info.java delete mode 100644 runners/flink/1.14/src/test/java/org/apache/beam/runners/flink/MiniClusterCompat.java delete mode 100644 runners/flink/1.14/src/test/java/org/apache/beam/runners/flink/metrics/MetricGroupWrapper.java rename runners/flink/{1.14 => 1.15}/src/main/java/org/apache/beam/runners/flink/translation/types/CoderTypeSerializer.java (100%) delete mode 100644 runners/flink/1.15/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ProcessingTimeCallbackCompat.java delete mode 100644 runners/flink/1.15/src/test/java/org/apache/beam/runners/flink/MiniClusterCompat.java rename runners/flink/{1.14 => }/src/test/java/org/apache/beam/runners/flink/RemoteMiniClusterImpl.java (100%) rename runners/flink/{1.14 => }/src/test/java/org/apache/beam/runners/flink/streaming/StreamSources.java (100%) rename runners/flink/{1.14 => }/src/test/java/org/apache/beam/runners/flink/translation/types/CoderTypeSerializerTe
(beam) branch master updated (2ee266bec03 -> 736ed4a0a31)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 2ee266bec03 Fixed the extension name in Java SDK Ordered Processing extension (#31382) add 5c8338a38c1 Reapply "[#30789] Add support for Flink 1.18 (#31062)" add a37df32264b [runners-flink] use mvn exec:exec for runQuickstartJavaFlinkLocal new 736ed4a0a31 Merge pull request #31328: Reapply "[#30789] Add support for Flink 1.18 (#31062)" The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../workflows/beam_PostCommit_Java_Tpcds_Flink.yml | 2 +- .../beam_PostCommit_Java_ValidatesRunner_Flink.yml | 2 +- CHANGES.md | 1 + contributor-docs/release-guide.md | 2 +- gradle.properties | 2 +- release/build.gradle.kts | 2 +- .../main/groovy/quickstart-java-flinklocal.groovy | 24 ++ runners/flink/{1.14 => 1.18}/build.gradle | 4 ++-- .../job-server-container/build.gradle | 0 .../flink/{1.16 => 1.18}/job-server/build.gradle | 2 +- sdks/go/examples/stringsplit/stringsplit.go| 2 +- 11 files changed, 25 insertions(+), 18 deletions(-) copy runners/flink/{1.14 => 1.18}/build.gradle (94%) copy runners/flink/{1.14 => 1.18}/job-server-container/build.gradle (100%) copy runners/flink/{1.16 => 1.18}/job-server/build.gradle (95%)
(beam) 01/01: Merge pull request #31328: Reapply "[#30789] Add support for Flink 1.18 (#31062)"
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 736ed4a0a31e8c7a08f8f32b6d8f40bf459fe672 Merge: 2ee266bec03 a37df32264b Author: Jan Lukavský AuthorDate: Thu May 23 19:03:13 2024 +0200 Merge pull request #31328: Reapply "[#30789] Add support for Flink 1.18 (#31062)" .../workflows/beam_PostCommit_Java_Tpcds_Flink.yml | 2 +- .../beam_PostCommit_Java_ValidatesRunner_Flink.yml | 2 +- CHANGES.md | 1 + contributor-docs/release-guide.md | 2 +- gradle.properties | 2 +- release/build.gradle.kts | 2 +- .../main/groovy/quickstart-java-flinklocal.groovy | 24 ++--- runners/flink/1.18/build.gradle| 25 + .../flink/1.18/job-server-container/build.gradle | 26 ++ runners/flink/1.18/job-server/build.gradle | 31 ++ sdks/go/examples/stringsplit/stringsplit.go| 2 +- 11 files changed, 104 insertions(+), 15 deletions(-)
(beam) branch master updated (365c2d92965 -> 0afb52d94c5)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 365c2d92965 Support Kafka Managed IO (#31172) add 5ff55b1caf1 [flink] Add documentation for Flink runner versions updates add 0afb52d94c5 Merge pull request #31217: [flink] Add documentation for Flink runner versions updates No new revisions were added by this update. Summary of changes: .../actions/setup-default-test-properties/test-properties.json| 2 +- sdks/go/examples/wasm/README.md | 2 +- sdks/python/apache_beam/options/pipeline_options.py | 2 +- sdks/typescript/src/apache_beam/runners/flink.ts | 2 +- settings.gradle.kts | 8 website/www/site/content/en/documentation/runners/flink.md| 8 +++- 6 files changed, 19 insertions(+), 5 deletions(-)
(beam) branch master updated: Add support for unaligned checkpoints for flink runner (#31186)
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new bb51380f1b2 Add support for unaligned checkpoints for flink runner (#31186) bb51380f1b2 is described below commit bb51380f1b29a2b69ab82ef795a8895ebd89f87e Author: Marc hurabielle AuthorDate: Wed May 8 00:48:13 2024 +0900 Add support for unaligned checkpoints for flink runner (#31186) --- .../beam/runners/flink/FlinkExecutionEnvironments.java | 7 +++ .../apache/beam/runners/flink/FlinkPipelineOptions.java | 16 2 files changed, 23 insertions(+) diff --git a/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkExecutionEnvironments.java b/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkExecutionEnvironments.java index 7c1bc87ced0..102340329b6 100644 --- a/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkExecutionEnvironments.java +++ b/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkExecutionEnvironments.java @@ -375,6 +375,13 @@ public class FlinkExecutionEnvironments { : ExternalizedCheckpointCleanup.DELETE_ON_CANCELLATION); } + if (options.getUnalignedCheckpointEnabled()) { +flinkStreamEnv.getCheckpointConfig().enableUnalignedCheckpoints(); + } + flinkStreamEnv + .getCheckpointConfig() + .setForceUnalignedCheckpoints(options.getForceUnalignedCheckpointEnabled()); + long minPauseBetweenCheckpoints = options.getMinPauseBetweenCheckpoints(); if (minPauseBetweenCheckpoints != -1) { flinkStreamEnv diff --git a/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkPipelineOptions.java b/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkPipelineOptions.java index 909789bbb12..519afa795bc 100644 --- a/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkPipelineOptions.java +++ b/runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkPipelineOptions.java @@ -127,6 +127,22 @@ public interface FlinkPipelineOptions void setFinishBundleBeforeCheckpointing(boolean finishBundleBeforeCheckpointing); + @Description( + "If set, Unaligned checkpoints contain in-flight data (i.e., data stored in buffers) as part of the " + + "checkpoint state, allowing checkpoint barriers to overtake these buffers. Thus, the checkpoint duration " + + "becomes independent of the current throughput as checkpoint barriers are effectively not embedded into the " + + "stream of data anymore") + @Default.Boolean(false) + boolean getUnalignedCheckpointEnabled(); + + void setUnalignedCheckpointEnabled(boolean unalignedCheckpointEnabled); + + @Description("Forces unaligned checkpoints, particularly allowing them for iterative jobs.") + @Default.Boolean(false) + boolean getForceUnalignedCheckpointEnabled(); + + void setForceUnalignedCheckpointEnabled(boolean forceUnalignedCheckpointEnabled); + @Description( "Shuts down sources which have been idle for the configured time of milliseconds. Once a source has been " + "shut down, checkpointing is not possible anymore. Shutting down the sources eventually leads to pipeline "
(beam) branch master updated (070fc903571 -> 45fe4f96f22)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 070fc903571 Bump google.golang.org/grpc from 1.62.1 to 1.63.2 in /sdks (#30900) add 45fe4f96f22 [#30789] Add support for Flink 1.18 (#31062) No new revisions were added by this update. Summary of changes: .github/workflows/beam_PostCommit_Java_Tpcds_Flink.yml | 2 +- .github/workflows/beam_PostCommit_Java_ValidatesRunner_Flink.yml | 2 +- CHANGES.md | 1 + gradle.properties| 2 +- release/build.gradle.kts | 2 +- runners/flink/{1.14 => 1.18}/build.gradle| 4 ++-- runners/flink/{1.14 => 1.18}/job-server-container/build.gradle | 0 runners/flink/{1.16 => 1.18}/job-server/build.gradle | 2 +- runners/flink/flink_runner.gradle| 2 +- sdks/go/examples/stringsplit/stringsplit.go | 2 +- settings.gradle.kts | 4 11 files changed, 14 insertions(+), 9 deletions(-) copy runners/flink/{1.14 => 1.18}/build.gradle (94%) copy runners/flink/{1.14 => 1.18}/job-server-container/build.gradle (100%) copy runners/flink/{1.16 => 1.18}/job-server/build.gradle (95%)
(beam) branch master updated (5c6bc39a44d -> 3cfc160b512)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 5c6bc39a44d Moving to 2.57.0-SNAPSHOT on master branch. add e4baee698d8 [runners-flink] remove 1.12 and 1.13 runners new 3cfc160b512 Merge pull request #30988: Remove flink 1.12 1.13 The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: gradle.properties | 2 +- release/build.gradle.kts | 2 +- runners/flink/1.12/build.gradle| 25 -- .../flink/1.12/job-server-container/build.gradle | 26 -- runners/flink/1.12/job-server/build.gradle | 31 --- .../streaming/AbstractStreamOperatorCompat.java| 94 -- .../io/source/compat/FlinkSourceCompat.java| 31 --- .../io/source/compat/SplitEnumeratorCompat.java| 27 --- .../beam/runners/flink/RemoteMiniClusterImpl.java | 68 .../runners/flink/metrics/MetricGroupWrapper.java | 31 --- .../runners/flink/streaming/StreamSources.java | 72 - .../streaming/io/source/SourceTestCompat.java | 75 - runners/flink/1.13/build.gradle| 25 -- .../flink/1.13/job-server-container/build.gradle | 26 -- runners/flink/1.13/job-server/build.gradle | 31 --- .../translation/types/CoderTypeSerializer.java | 0 .../streaming/ProcessingTimeCallbackCompat.java| 0 .../io/source/compat/SplitEnumeratorCompat.java| 0 .../streaming/io/source/compat/package-info.java | 0 .../beam/runners/flink/MiniClusterCompat.java | 0 .../translation/types/CoderTypeSerializerTest.java | 0 runners/flink/flink_runner.gradle | 17 ++-- .../streaming/io/source/FlinkSourceReaderBase.java | 2 - sdks/go/examples/stringsplit/stringsplit.go| 2 +- sdks/java/testing/nexmark/build.gradle | 2 +- sdks/java/testing/tpcds/build.gradle | 2 +- settings.gradle.kts| 8 -- .../content/en/documentation/dsls/sql/shell.md | 6 +- .../shortcodes/flink_java_pipeline_options.html| 8 +- .../shortcodes/flink_python_pipeline_options.html | 8 +- 30 files changed, 22 insertions(+), 599 deletions(-) delete mode 100644 runners/flink/1.12/build.gradle delete mode 100644 runners/flink/1.12/job-server-container/build.gradle delete mode 100644 runners/flink/1.12/job-server/build.gradle delete mode 100644 runners/flink/1.12/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/AbstractStreamOperatorCompat.java delete mode 100644 runners/flink/1.12/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/io/source/compat/FlinkSourceCompat.java delete mode 100644 runners/flink/1.12/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/io/source/compat/SplitEnumeratorCompat.java delete mode 100644 runners/flink/1.12/src/test/java/org/apache/beam/runners/flink/RemoteMiniClusterImpl.java delete mode 100644 runners/flink/1.12/src/test/java/org/apache/beam/runners/flink/metrics/MetricGroupWrapper.java delete mode 100644 runners/flink/1.12/src/test/java/org/apache/beam/runners/flink/streaming/StreamSources.java delete mode 100644 runners/flink/1.12/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/io/source/SourceTestCompat.java delete mode 100644 runners/flink/1.13/build.gradle delete mode 100644 runners/flink/1.13/job-server-container/build.gradle delete mode 100644 runners/flink/1.13/job-server/build.gradle rename runners/flink/{1.12 => 1.14}/src/main/java/org/apache/beam/runners/flink/translation/types/CoderTypeSerializer.java (100%) rename runners/flink/{1.12 => 1.14}/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ProcessingTimeCallbackCompat.java (100%) rename runners/flink/{1.13 => 1.14}/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/io/source/compat/SplitEnumeratorCompat.java (100%) rename runners/flink/{1.12 => 1.14}/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/io/source/compat/package-info.java (100%) rename runners/flink/{1.12 => 1.14}/src/test/java/org/apache/beam/runners/flink/MiniClusterCompat.java (100%) rename runners/flink/{1.12 => 1.14}/src/test/java/org/apache/beam/runners/flink/translation/types/CoderTypeSerializerTest.java (100%)
(beam) 01/01: Merge pull request #30988: Remove flink 1.12 1.13
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 3cfc160b51234a5cd8d3c0c34b0b229b3e543baa Merge: 5c6bc39a44d e4baee698d8 Author: Jan Lukavský AuthorDate: Wed Apr 17 20:12:45 2024 +0200 Merge pull request #30988: Remove flink 1.12 1.13 gradle.properties | 2 +- release/build.gradle.kts | 2 +- runners/flink/1.12/build.gradle| 25 -- .../flink/1.12/job-server-container/build.gradle | 26 -- runners/flink/1.12/job-server/build.gradle | 31 --- .../streaming/AbstractStreamOperatorCompat.java| 94 -- .../io/source/compat/FlinkSourceCompat.java| 31 --- .../io/source/compat/SplitEnumeratorCompat.java| 27 --- .../beam/runners/flink/RemoteMiniClusterImpl.java | 68 .../runners/flink/metrics/MetricGroupWrapper.java | 31 --- .../runners/flink/streaming/StreamSources.java | 72 - .../streaming/io/source/SourceTestCompat.java | 75 - runners/flink/1.13/build.gradle| 25 -- .../flink/1.13/job-server-container/build.gradle | 26 -- runners/flink/1.13/job-server/build.gradle | 31 --- .../translation/types/CoderTypeSerializer.java | 0 .../streaming/ProcessingTimeCallbackCompat.java| 0 .../io/source/compat/SplitEnumeratorCompat.java| 0 .../streaming/io/source/compat/package-info.java | 0 .../beam/runners/flink/MiniClusterCompat.java | 0 .../translation/types/CoderTypeSerializerTest.java | 0 runners/flink/flink_runner.gradle | 17 ++-- .../streaming/io/source/FlinkSourceReaderBase.java | 2 - sdks/go/examples/stringsplit/stringsplit.go| 2 +- sdks/java/testing/nexmark/build.gradle | 2 +- sdks/java/testing/tpcds/build.gradle | 2 +- settings.gradle.kts| 8 -- .../content/en/documentation/dsls/sql/shell.md | 6 +- .../shortcodes/flink_java_pipeline_options.html| 8 +- .../shortcodes/flink_python_pipeline_options.html | 8 +- 30 files changed, 22 insertions(+), 599 deletions(-)
(beam) branch master updated (e48a9351d2f -> d70c25328ee)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from e48a9351d2f add Yaml_Xlang_Direct PreCommit to README (#31009) add deee0e9cc8c [flink] add support for Flink 1.17 (#29939) new d70c25328ee Merge pull request #30197: Flink 1.17 The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: ...am_PostCommit_Python_ValidatesRunner_Flink.json | 0 .../beam_PostCommit_Java_ValidatesRunner_Flink.yml | 4 +- gradle.properties | 2 +- runners/flink/1.12/build.gradle| 12 +-- .../translation/types/CoderTypeSerializer.java | 10 ++- .../translation/types/CoderTypeSerializerTest.java | 2 +- runners/flink/1.13/build.gradle| 12 +-- runners/flink/1.14/build.gradle| 13 +--- runners/flink/1.15/build.gradle| 13 +--- runners/flink/1.16/build.gradle| 13 +--- .../java8 => runners/flink/1.17}/build.gradle | 5 +- .../job-server-container/build.gradle | 0 .../flink/{1.12 => 1.17}/job-server/build.gradle | 2 +- .../translation/types/CoderTypeSerializer.java | 31 +--- runners/flink/flink_runner.gradle | 20 - .../types/UnversionedTypeSerializerSnapshot.java | 86 ++ .../UnversionedTypeSerializerSnapshotTest.java | 52 + settings.gradle.kts| 4 + .../shortcodes/flink_java_pipeline_options.html| 15 .../shortcodes/flink_python_pipeline_options.html | 15 20 files changed, 220 insertions(+), 91 deletions(-) copy learning/tour-of-beam/learning-content/io/text-io/text-io-local-write/go-example/myfile.txt => .github/trigger_files/beam_PostCommit_Python_ValidatesRunner_Flink.json (100%) copy runners/flink/{ => 1.12}/src/main/java/org/apache/beam/runners/flink/translation/types/CoderTypeSerializer.java (94%) rename runners/flink/{ => 1.12}/src/test/java/org/apache/beam/runners/flink/translation/types/CoderTypeSerializerTest.java (97%) copy {sdks/java/container/java8 => runners/flink/1.17}/build.gradle (91%) copy runners/flink/{1.12 => 1.17}/job-server-container/build.gradle (100%) copy runners/flink/{1.12 => 1.17}/job-server/build.gradle (95%) rename runners/flink/{ => 1.17}/src/main/java/org/apache/beam/runners/flink/translation/types/CoderTypeSerializer.java (82%) create mode 100644 runners/flink/src/main/java/org/apache/beam/runners/flink/translation/types/UnversionedTypeSerializerSnapshot.java create mode 100644 runners/flink/src/test/java/org/apache/beam/runners/flink/translation/types/UnversionedTypeSerializerSnapshotTest.java
(beam) 01/01: Merge pull request #30197: Flink 1.17
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit d70c25328eec42be281401a958ff9829659174ea Merge: e48a9351d2f deee0e9cc8c Author: Jan Lukavský AuthorDate: Wed Apr 17 16:52:38 2024 +0200 Merge pull request #30197: Flink 1.17 ...am_PostCommit_Python_ValidatesRunner_Flink.json | 0 .../beam_PostCommit_Java_ValidatesRunner_Flink.yml | 4 +- gradle.properties | 2 +- runners/flink/1.12/build.gradle| 12 +-- .../translation/types/CoderTypeSerializer.java | 10 ++- .../translation/types/CoderTypeSerializerTest.java | 2 +- runners/flink/1.13/build.gradle| 12 +-- runners/flink/1.14/build.gradle| 13 +--- runners/flink/1.15/build.gradle| 13 +--- runners/flink/1.16/build.gradle| 13 +--- runners/flink/{1.12 => 1.17}/build.gradle | 14 +--- .../job-server-container}/build.gradle | 15 +--- .../flink/{1.12 => 1.17/job-server}/build.gradle | 20 +++-- .../translation/types/CoderTypeSerializer.java | 31 +--- runners/flink/flink_runner.gradle | 20 - .../types/UnversionedTypeSerializerSnapshot.java | 86 ++ .../UnversionedTypeSerializerSnapshotTest.java | 52 + settings.gradle.kts| 4 + .../shortcodes/flink_java_pipeline_options.html| 15 .../shortcodes/flink_python_pipeline_options.html | 15 20 files changed, 232 insertions(+), 121 deletions(-)
(beam) branch master updated (9fa45df9b06 -> f41f3648c3e)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 9fa45df9b06 Move logic to build a Labeled Metric Name to a util file. (#30796) add 2f3893259fa [runners-flink] Finalize checkpoint marks after successful checkpoint (#29902) add f41f3648c3e Merge pull request #30971: #29902 finalize checkpoints after checkpoint No new revisions were added by this update. Summary of changes: .../streaming/io/source/FlinkSourceReaderBase.java | 11 + .../streaming/io/source/FlinkSourceSplit.java | 21 +++- .../unbounded/FlinkUnboundedSourceReader.java | 56 ++ .../unbounded/FlinkUnboundedSourceReaderTest.java | 11 + 4 files changed, 87 insertions(+), 12 deletions(-)
(beam) branch master updated (4b808b0d4f5 -> e119cd49cf9)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 4b808b0d4f5 Fix build wheels due to setuptools changed naming (#30972) add d680bf0a57d [runners-flink] Fix watermark emission for empty splits (#29816) new e119cd49cf9 Merge pull request #30969: [runners-flink] Fix watermark emission for empty splits (#29816) The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../streaming/io/source/FlinkSourceReaderBase.java | 49 -- .../streaming/io/source/FlinkSourceSplit.java | 8 ++-- .../io/source/FlinkSourceSplitEnumerator.java | 5 ++- .../unbounded/FlinkUnboundedSourceReader.java | 35 +++- .../unbounded/FlinkUnboundedSourceReaderTest.java | 12 ++ 5 files changed, 63 insertions(+), 46 deletions(-)
(beam) 01/01: Merge pull request #30969: [runners-flink] Fix watermark emission for empty splits (#29816)
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit e119cd49cf9f65916434443c3968dbd53d94f086 Merge: 4b808b0d4f5 d680bf0a57d Author: Jan Lukavský AuthorDate: Mon Apr 15 19:35:00 2024 +0200 Merge pull request #30969: [runners-flink] Fix watermark emission for empty splits (#29816) .../streaming/io/source/FlinkSourceReaderBase.java | 49 -- .../streaming/io/source/FlinkSourceSplit.java | 8 ++-- .../io/source/FlinkSourceSplitEnumerator.java | 5 ++- .../unbounded/FlinkUnboundedSourceReader.java | 35 +++- .../unbounded/FlinkUnboundedSourceReaderTest.java | 12 ++ 5 files changed, 63 insertions(+), 46 deletions(-)
(beam) 01/01: Merge pull request #30905: Do not reemit data from impulse (#30903)
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit f071e9e34151276efd6162f81bf3460f5bbad5cc Merge: cd253fd8922 cfe7be6a609 Author: Jan Lukavský AuthorDate: Mon Apr 15 08:56:48 2024 +0200 Merge pull request #30905: Do not reemit data from impulse (#30903) .../flink/FlinkStreamingTransformTranslators.java | 27 +++ .../wrappers/streaming/io/source/FlinkSource.java | 18 -- .../streaming/io/source/FlinkSourceReaderBase.java | 6 ++-- .../streaming/io/source/FlinkSourceSplit.java | 7 +++- .../io/source/FlinkSourceSplitEnumerator.java | 1 + .../io/source/bounded/FlinkBoundedSource.java | 1 + .../source/bounded/FlinkBoundedSourceReader.java | 39 +++--- .../io/source/impulse/BeamImpulseSource.java | 34 --- .../bounded/FlinkBoundedSourceReaderTest.java | 1 + 9 files changed, 52 insertions(+), 82 deletions(-)
(beam) branch master updated (cd253fd8922 -> f071e9e3415)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from cd253fd8922 Revert setuptools upperbound since buggy releases are yanked. (#30958) add cfe7be6a609 [runners-flink] Use ImpulseSourceFunction for streaming Impulse (#30903) new f071e9e3415 Merge pull request #30905: Do not reemit data from impulse (#30903) The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../flink/FlinkStreamingTransformTranslators.java | 27 +++ .../wrappers/streaming/io/source/FlinkSource.java | 18 -- .../streaming/io/source/FlinkSourceReaderBase.java | 6 ++-- .../streaming/io/source/FlinkSourceSplit.java | 7 +++- .../io/source/FlinkSourceSplitEnumerator.java | 1 + .../io/source/bounded/FlinkBoundedSource.java | 1 + .../source/bounded/FlinkBoundedSourceReader.java | 39 +++--- .../io/source/impulse/BeamImpulseSource.java | 34 --- .../bounded/FlinkBoundedSourceReaderTest.java | 1 + 9 files changed, 52 insertions(+), 82 deletions(-)
(beam) 01/01: Merge pull request #30622: [runners-flink] #30621 use groupBy for Reshuffle in batch
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 52c0d5a3f86cccb21f52fe92f8fd25790c06e71c Merge: 0bfe06b61a4 5697c67421a Author: Jan Lukavský AuthorDate: Thu Mar 14 08:41:26 2024 +0100 Merge pull request #30622: [runners-flink] #30621 use groupBy for Reshuffle in batch runners/flink/flink_runner.gradle | 2 -- .../runners/flink/FlinkBatchTransformTranslators.java | 17 + 2 files changed, 9 insertions(+), 10 deletions(-)
(beam) branch master updated (0bfe06b61a4 -> 52c0d5a3f86)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 0bfe06b61a4 Merge pull request #30587 Properly handle timestamp prefixing of unkown window types. add 5697c67421a [runners-flink] #30621 use groupBy for Reshuffle in batch new 52c0d5a3f86 Merge pull request #30622: [runners-flink] #30621 use groupBy for Reshuffle in batch The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: runners/flink/flink_runner.gradle | 2 -- .../runners/flink/FlinkBatchTransformTranslators.java | 17 + 2 files changed, 9 insertions(+), 10 deletions(-)
(beam) branch master updated (fb3f6692dbf -> 907a4514386)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from fb3f6692dbf Fix invalid escape sequence '\#' (#30393) add 436f3b223f0 [runners-core-java] restore original SerializablePipelineOptions add 4982f6ac6e4 [sdks-java-core] add serialVersionUID to PTransform add 4f13a4e1ea2 [runners-core] add jackson new 907a4514386 Merge pull request #30403: [flink] #30402 restore upgradability The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: runners/core-java/build.gradle| 2 ++ .../beam/runners/core}/construction/SerializablePipelineOptions.java | 2 +- .../beam/runners/core/{metrics => construction}/package-info.java | 4 ++-- .../runners/core}/construction/SerializablePipelineOptionsTest.java | 2 +- .../org/apache/beam/runners/flink/FlinkExecutionEnvironments.java | 2 +- .../beam/runners/flink/FlinkStreamingPortablePipelineTranslator.java | 2 +- .../apache/beam/runners/flink/FlinkStreamingTransformTranslators.java | 2 +- .../beam/runners/flink/translation/functions/FlinkDoFnFunction.java | 2 +- .../flink/translation/functions/FlinkExecutableStageFunction.java | 2 +- .../translation/functions/FlinkExecutableStagePruningFunction.java| 2 +- .../translation/functions/FlinkMergingNonShuffleReduceFunction.java | 2 +- .../flink/translation/functions/FlinkMultiOutputPruningFunction.java | 2 +- .../flink/translation/functions/FlinkPartialReduceFunction.java | 2 +- .../beam/runners/flink/translation/functions/FlinkReduceFunction.java | 2 +- .../flink/translation/functions/FlinkStatefulDoFnFunction.java| 2 +- .../beam/runners/flink/translation/types/CoderTypeInformation.java| 2 +- .../beam/runners/flink/translation/types/CoderTypeSerializer.java | 2 +- .../beam/runners/flink/translation/wrappers/SourceInputFormat.java| 2 +- .../runners/flink/translation/wrappers/streaming/DoFnOperator.java| 2 +- .../translation/wrappers/streaming/ExecutableStageDoFnOperator.java | 2 +- .../translation/wrappers/streaming/KvToByteBufferKeySelector.java | 2 +- .../translation/wrappers/streaming/SdfByteBufferKeySelector.java | 2 +- .../flink/translation/wrappers/streaming/WorkItemKeySelector.java | 2 +- .../flink/translation/wrappers/streaming/io/DedupingOperator.java | 2 +- .../translation/wrappers/streaming/io/UnboundedSourceWrapper.java | 2 +- .../flink/translation/wrappers/streaming/io/source/FlinkSource.java | 2 +- .../wrappers/streaming/io/source/bounded/FlinkBoundedSource.java | 2 +- .../wrappers/streaming/io/source/unbounded/FlinkUnboundedSource.java | 2 +- .../wrappers/streaming/stableinput/BufferingDoFnRunner.java | 2 +- .../wrappers/streaming/state/FlinkBroadcastStateInternals.java| 2 +- .../translation/wrappers/streaming/state/FlinkStateInternals.java | 2 +- .../java/org/apache/beam/runners/flink/FlinkPipelineOptionsTest.java | 2 +- .../runners/flink/streaming/FlinkBroadcastStateInternalsTest.java | 2 +- .../apache/beam/runners/flink/streaming/FlinkStateInternalsTest.java | 2 +- .../beam/runners/flink/translation/types/CoderTypeSerializerTest.java | 2 +- .../flink/translation/wrappers/streaming/DoFnOperatorTest.java| 2 +- .../wrappers/streaming/ExecutableStageDoFnOperatorTest.java | 2 +- .../flink/translation/wrappers/streaming/WindowDoFnOperatorTest.java | 2 +- .../wrappers/streaming/stableinput/BufferingDoFnRunnerTest.java | 2 +- .../java/org/apache/beam/runners/jet/JetTransformTranslators.java | 2 +- .../main/java/org/apache/beam/runners/jet/JetTranslationContext.java | 2 +- .../java/org/apache/beam/runners/jet/processors/AbstractParDoP.java | 2 +- .../java/org/apache/beam/runners/jet/processors/BoundedSourceP.java | 2 +- .../src/main/java/org/apache/beam/runners/jet/processors/ParDoP.java | 2 +- .../java/org/apache/beam/runners/jet/processors/StatefulParDoP.java | 2 +- .../java/org/apache/beam/runners/jet/processors/UnboundedSourceP.java | 2 +- .../java/org/apache/beam/runners/jet/processors/WindowGroupP.java | 2 +- .../org/apache/beam/runners/samza/adapter/BoundedSourceSystem.java| 2 +- .../org/apache/beam/runners/samza/adapter/UnboundedSourceSystem.java | 2 +- .../runners/samza/runtime/SplittableParDoProcessKeyedElementsOp.java | 2 +- .../java/org/apache/beam/runners/samza/translation/ConfigBuilder.java | 2 +- .../spark/structuredstreaming/translation/PipelineTranslator.java | 2 +- .../spark/structuredstreaming/translation/SparkSessionFactory.java| 2 +- .../src/main/java/org/apache/beam/runner
(beam) 01/01: Merge pull request #30403: [flink] #30402 restore upgradability
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 907a4514386109245a52630907d252cf5609942d Merge: fb3f6692dbf 4f13a4e1ea2 Author: Jan Lukavský AuthorDate: Mon Feb 26 09:05:23 2024 +0100 Merge pull request #30403: [flink] #30402 restore upgradability runners/core-java/build.gradle | 2 ++ .../construction/SerializablePipelineOptions.java | 2 +- .../runners/core/construction/package-info.java| 24 ++ .../SerializablePipelineOptionsTest.java | 2 +- .../runners/flink/FlinkExecutionEnvironments.java | 2 +- .../FlinkStreamingPortablePipelineTranslator.java | 2 +- .../flink/FlinkStreamingTransformTranslators.java | 2 +- .../translation/functions/FlinkDoFnFunction.java | 2 +- .../functions/FlinkExecutableStageFunction.java| 2 +- .../FlinkExecutableStagePruningFunction.java | 2 +- .../FlinkMergingNonShuffleReduceFunction.java | 2 +- .../functions/FlinkMultiOutputPruningFunction.java | 2 +- .../functions/FlinkPartialReduceFunction.java | 2 +- .../translation/functions/FlinkReduceFunction.java | 2 +- .../functions/FlinkStatefulDoFnFunction.java | 2 +- .../translation/types/CoderTypeInformation.java| 2 +- .../translation/types/CoderTypeSerializer.java | 2 +- .../translation/wrappers/SourceInputFormat.java| 2 +- .../wrappers/streaming/DoFnOperator.java | 2 +- .../streaming/ExecutableStageDoFnOperator.java | 2 +- .../streaming/KvToByteBufferKeySelector.java | 2 +- .../streaming/SdfByteBufferKeySelector.java| 2 +- .../wrappers/streaming/WorkItemKeySelector.java| 2 +- .../wrappers/streaming/io/DedupingOperator.java| 2 +- .../streaming/io/UnboundedSourceWrapper.java | 2 +- .../wrappers/streaming/io/source/FlinkSource.java | 2 +- .../io/source/bounded/FlinkBoundedSource.java | 2 +- .../io/source/unbounded/FlinkUnboundedSource.java | 2 +- .../streaming/stableinput/BufferingDoFnRunner.java | 2 +- .../state/FlinkBroadcastStateInternals.java| 2 +- .../streaming/state/FlinkStateInternals.java | 2 +- .../runners/flink/FlinkPipelineOptionsTest.java| 2 +- .../FlinkBroadcastStateInternalsTest.java | 2 +- .../flink/streaming/FlinkStateInternalsTest.java | 2 +- .../translation/types/CoderTypeSerializerTest.java | 2 +- .../wrappers/streaming/DoFnOperatorTest.java | 2 +- .../streaming/ExecutableStageDoFnOperatorTest.java | 2 +- .../wrappers/streaming/WindowDoFnOperatorTest.java | 2 +- .../stableinput/BufferingDoFnRunnerTest.java | 2 +- .../beam/runners/jet/JetTransformTranslators.java | 2 +- .../beam/runners/jet/JetTranslationContext.java| 2 +- .../runners/jet/processors/AbstractParDoP.java | 2 +- .../runners/jet/processors/BoundedSourceP.java | 2 +- .../apache/beam/runners/jet/processors/ParDoP.java | 2 +- .../runners/jet/processors/StatefulParDoP.java | 2 +- .../runners/jet/processors/UnboundedSourceP.java | 2 +- .../beam/runners/jet/processors/WindowGroupP.java | 2 +- .../runners/samza/adapter/BoundedSourceSystem.java | 2 +- .../samza/adapter/UnboundedSourceSystem.java | 2 +- .../SplittableParDoProcessKeyedElementsOp.java | 2 +- .../runners/samza/translation/ConfigBuilder.java | 2 +- .../translation/PipelineTranslator.java| 2 +- .../translation/SparkSessionFactory.java | 2 +- .../beam/runners/spark/io/SourceDStream.java | 2 +- .../apache/beam/runners/spark/io/SourceRDD.java| 2 +- .../runners/spark/io/SparkUnboundedSource.java | 2 +- .../SparkGroupAlsoByWindowViaWindowSet.java| 2 +- .../runners/spark/stateful/StateSpecFunctions.java | 2 +- .../spark/translation/EvaluationContext.java | 2 +- .../spark/translation/MultiDoFnFunction.java | 2 +- .../runners/spark/translation/SparkCombineFn.java | 2 +- .../translation/SparkExecutableStageFunction.java | 2 +- .../SparkGroupAlsoByWindowViaOutputBufferFn.java | 2 +- .../spark/translation/SparkTranslationContext.java | 2 +- .../streaming/StreamingTransformTranslator.java| 2 +- .../spark/translation/SparkCombineFnTest.java | 2 +- .../SparkExecutableStageFunctionTest.java | 2 +- .../wrappers/Twister2BoundedSource.java| 2 +- .../translators/functions/DoFnFunction.java| 2 +- .../org/apache/beam/sdk/transforms/PTransform.java | 3 +++ 70 files changed, 96 insertions(+), 67 deletions(-)
(beam) 01/01: Merge pull request #30141: Adding a default watermark emit interval for FlinkUnboundedSourceReader
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit cd5f2711c83071b8630f5c3c701b1fda59b1a74a Merge: d5aa44c9ba9 a341eb69abe Author: Jan Lukavský AuthorDate: Tue Feb 6 08:40:15 2024 +0100 Merge pull request #30141: Adding a default watermark emit interval for FlinkUnboundedSourceReader .../unbounded/FlinkUnboundedSourceReader.java | 36 +- 1 file changed, 22 insertions(+), 14 deletions(-)
(beam) branch master updated (d5aa44c9ba9 -> cd5f2711c83)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from d5aa44c9ba9 Handle a user whose account was deleted. (#30192) add a341eb69abe Added default watermark generation interval (apache#30141) new cd5f2711c83 Merge pull request #30141: Adding a default watermark emit interval for FlinkUnboundedSourceReader The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../unbounded/FlinkUnboundedSourceReader.java | 36 +- 1 file changed, 22 insertions(+), 14 deletions(-)
(beam) 01/01: Merge pull request #30163: [sdks-java-core] Test for Reiterator outside loop in CoGbkResult
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 36cab4ffe76530ca5993a61a40e2616621567e71 Merge: 1df9d6bba91 4c5ee349f9b Author: Jan Lukavský AuthorDate: Wed Jan 31 10:03:38 2024 +0100 Merge pull request #30163: [sdks-java-core] Test for Reiterator outside loop in CoGbkResult .../src/main/java/org/apache/beam/sdk/transforms/join/CoGbkResult.java | 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-)
(beam) branch master updated (1df9d6bba91 -> 36cab4ffe76)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 1df9d6bba91 Bumping timeout for beam_PostCommit_Java_DataflowV2 (#30160) add 4c5ee349f9b [sdks-java-core] Test for Reiterator outside loop in CoGbkResult new 36cab4ffe76 Merge pull request #30163: [sdks-java-core] Test for Reiterator outside loop in CoGbkResult The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../src/main/java/org/apache/beam/sdk/transforms/join/CoGbkResult.java | 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-)
(beam) 01/01: Merge pull request #30066: [flink] Fix watermark generation for empty UnboundedSource
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 90e7c9e8f6b59d17fdb0d995db6faec2d72d8680 Merge: cffeff78f62 58650378fe7 Author: Jan Lukavský AuthorDate: Tue Jan 23 21:13:48 2024 +0100 Merge pull request #30066: [flink] Fix watermark generation for empty UnboundedSource .../streaming/io/source/FlinkSourceReaderBase.java | 1 + .../unbounded/FlinkUnboundedSourceReader.java | 13 ++- .../streaming/io/source/EmptyUnboundedSource.java | 101 + .../io/source/FlinkSourceReaderTestBase.java | 8 ++ .../unbounded/FlinkUnboundedSourceReaderTest.java | 33 +++ 5 files changed, 154 insertions(+), 2 deletions(-)
(beam) branch master updated (cffeff78f62 -> 90e7c9e8f6b)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from cffeff78f62 Fix Dicom IO Integration Tests (#30073) add 58650378fe7 [flink] Fix watermark generation for empty UnboundedSource (#30050) new 90e7c9e8f6b Merge pull request #30066: [flink] Fix watermark generation for empty UnboundedSource The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../streaming/io/source/FlinkSourceReaderBase.java | 1 + .../unbounded/FlinkUnboundedSourceReader.java | 13 ++- .../streaming/io/source/EmptyUnboundedSource.java | 101 + .../io/source/FlinkSourceReaderTestBase.java | 8 ++ .../unbounded/FlinkUnboundedSourceReaderTest.java | 33 +++ 5 files changed, 154 insertions(+), 2 deletions(-) create mode 100644 runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/io/source/EmptyUnboundedSource.java
(beam) branch master updated (1061fcf0969 -> a3d011549d7)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 1061fcf0969 Bump github.com/nats-io/nats-server/v2 from 2.10.6 to 2.10.7 in /sdks (#29698) add 75ba21ff5c3 [runners-flink] #29558 emit watermark before source shutdown new a3d011549d7 Merge pull request #29563: [runners-flink] #29558 emit watermark before source shutdown The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../streaming/io/source/FlinkSourceReaderBase.java | 80 --- .../io/source/FlinkSourceSplitEnumerator.java | 1 - .../source/bounded/FlinkBoundedSourceReader.java | 92 +- .../unbounded/FlinkUnboundedSourceReader.java | 51 .../io/source/FlinkSourceReaderTestBase.java | 8 +- .../bounded/FlinkBoundedSourceReaderTest.java | 18 - 6 files changed, 161 insertions(+), 89 deletions(-)
(beam) 01/01: Merge pull request #29563: [runners-flink] #29558 emit watermark before source shutdown
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit a3d011549d78376bcca942aff68347619c660458 Merge: 1061fcf0969 75ba21ff5c3 Author: Jan Lukavský AuthorDate: Mon Dec 11 15:09:53 2023 +0100 Merge pull request #29563: [runners-flink] #29558 emit watermark before source shutdown .../streaming/io/source/FlinkSourceReaderBase.java | 80 --- .../io/source/FlinkSourceSplitEnumerator.java | 1 - .../source/bounded/FlinkBoundedSourceReader.java | 92 +- .../unbounded/FlinkUnboundedSourceReader.java | 51 .../io/source/FlinkSourceReaderTestBase.java | 8 +- .../bounded/FlinkBoundedSourceReaderTest.java | 18 - 6 files changed, 161 insertions(+), 89 deletions(-)
(beam) branch master updated (729c4de416b -> f6189692c9a)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 729c4de416b Fix Dataflow ARM PostCommit failing Java21 (#29444) add f9b9d9af7b6 [dsl-euphoria] #29451 deprecate Euphoria DSL new f6189692c9a Merge pull request #29452: [dsl-euphoria] #29451 deprecate Euphoria DSL The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../extensions/euphoria/core/client/accumulators/Accumulator.java | 3 +++ .../euphoria/core/client/accumulators/AccumulatorProvider.java | 3 +++ .../sdk/extensions/euphoria/core/client/accumulators/Counter.java | 7 ++- .../extensions/euphoria/core/client/accumulators/Histogram.java| 7 ++- .../sdk/extensions/euphoria/core/client/accumulators/Timer.java| 3 +++ .../euphoria/core/client/accumulators/VoidAccumulatorProvider.java | 3 +++ .../extensions/euphoria/core/client/functional/BinaryFunction.java | 2 ++ .../extensions/euphoria/core/client/functional/BinaryFunctor.java | 7 ++- .../euphoria/core/client/functional/CombinableBinaryFunction.java | 7 ++- .../euphoria/core/client/functional/CombinableReduceFunction.java | 3 +++ .../euphoria/core/client/functional/CompositeUnaryFunction.java| 7 ++- .../sdk/extensions/euphoria/core/client/functional/Consumer.java | 7 ++- .../euphoria/core/client/functional/ExtractEventTime.java | 6 +- .../extensions/euphoria/core/client/functional/ReduceFunction.java | 3 +++ .../extensions/euphoria/core/client/functional/ReduceFunctor.java | 3 +++ .../euphoria/core/client/functional/SameArgsBinaryFunction.java| 7 ++- .../sdk/extensions/euphoria/core/client/functional/Supplier.java | 2 ++ .../euphoria/core/client/functional/TernaryFunction.java | 7 ++- .../extensions/euphoria/core/client/functional/UnaryFunction.java | 2 ++ .../euphoria/core/client/functional/UnaryFunctionEnv.java | 2 ++ .../extensions/euphoria/core/client/functional/UnaryFunctor.java | 3 +++ .../extensions/euphoria/core/client/functional/UnaryPredicate.java | 7 ++- .../extensions/euphoria/core/client/functional/VoidFunction.java | 7 ++- .../beam/sdk/extensions/euphoria/core/client/io/Collector.java | 2 ++ .../beam/sdk/extensions/euphoria/core/client/io/Context.java | 7 ++- .../beam/sdk/extensions/euphoria/core/client/io/Environment.java | 7 ++- .../apache/beam/sdk/extensions/euphoria/core/client/lib/Split.java | 3 +++ .../extensions/euphoria/core/client/operator/AssignEventTime.java | 3 +++ .../euphoria/core/client/operator/CompositeOperator.java | 2 ++ .../sdk/extensions/euphoria/core/client/operator/CountByKey.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/Distinct.java | 3 +++ .../beam/sdk/extensions/euphoria/core/client/operator/Filter.java | 3 +++ .../beam/sdk/extensions/euphoria/core/client/operator/FlatMap.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/FullJoin.java | 3 +++ .../beam/sdk/extensions/euphoria/core/client/operator/Join.java| 3 +++ .../sdk/extensions/euphoria/core/client/operator/LeftJoin.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/MapElements.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/ReduceByKey.java | 2 ++ .../sdk/extensions/euphoria/core/client/operator/ReduceWindow.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/RightJoin.java| 3 +++ .../sdk/extensions/euphoria/core/client/operator/SumByKey.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/TopPerKey.java| 3 +++ .../beam/sdk/extensions/euphoria/core/client/operator/Union.java | 3 +++ .../extensions/euphoria/core/client/operator/base/Builders.java| 3 +++ .../euphoria/core/client/operator/base/OptionalMethodBuilder.java | 2 ++ .../extensions/euphoria/core/client/operator/base/WindowAware.java | 2 ++ .../extensions/euphoria/core/client/operator/hint/OutputHint.java | 2 +- .../beam/sdk/extensions/euphoria/core/client/type/TypeAware.java | 3 ++- .../sdk/extensions/euphoria/core/client/type/TypeAwareness.java| 2 ++ .../beam/sdk/extensions/euphoria/core/client/type/TypeUtils.java | 7 ++- .../apache/beam/sdk/extensions/euphoria/core/client/util/Fold.java | 2 ++ .../apache/beam/sdk/extensions/euphoria/core/client/util/Max.java | 7 ++- .../sdk/extensions/euphoria/core/client/util/PCollectionLists.java | 2 ++ .../apache/beam/sdk/extensions/euphoria/core/client/util/Sums.java | 7 ++- .../euphoria/core/translate/BeamAccumulatorProvider.java | 2 ++ .../euphoria/core/translate/BroadcastHashJoinTranslator.java | 3 +++ .../eup
(beam) 01/01: Merge pull request #29452: [dsl-euphoria] #29451 deprecate Euphoria DSL
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit f6189692c9aa5f3fa00d49b32afb170e7dabf29a Merge: 729c4de416b f9b9d9af7b6 Author: Jan Lukavský AuthorDate: Thu Nov 16 16:15:38 2023 +0100 Merge pull request #29452: [dsl-euphoria] #29451 deprecate Euphoria DSL .../extensions/euphoria/core/client/accumulators/Accumulator.java | 3 +++ .../euphoria/core/client/accumulators/AccumulatorProvider.java | 3 +++ .../sdk/extensions/euphoria/core/client/accumulators/Counter.java | 7 ++- .../extensions/euphoria/core/client/accumulators/Histogram.java| 7 ++- .../sdk/extensions/euphoria/core/client/accumulators/Timer.java| 3 +++ .../euphoria/core/client/accumulators/VoidAccumulatorProvider.java | 3 +++ .../extensions/euphoria/core/client/functional/BinaryFunction.java | 2 ++ .../extensions/euphoria/core/client/functional/BinaryFunctor.java | 7 ++- .../euphoria/core/client/functional/CombinableBinaryFunction.java | 7 ++- .../euphoria/core/client/functional/CombinableReduceFunction.java | 3 +++ .../euphoria/core/client/functional/CompositeUnaryFunction.java| 7 ++- .../sdk/extensions/euphoria/core/client/functional/Consumer.java | 7 ++- .../euphoria/core/client/functional/ExtractEventTime.java | 6 +- .../extensions/euphoria/core/client/functional/ReduceFunction.java | 3 +++ .../extensions/euphoria/core/client/functional/ReduceFunctor.java | 3 +++ .../euphoria/core/client/functional/SameArgsBinaryFunction.java| 7 ++- .../sdk/extensions/euphoria/core/client/functional/Supplier.java | 2 ++ .../euphoria/core/client/functional/TernaryFunction.java | 7 ++- .../extensions/euphoria/core/client/functional/UnaryFunction.java | 2 ++ .../euphoria/core/client/functional/UnaryFunctionEnv.java | 2 ++ .../extensions/euphoria/core/client/functional/UnaryFunctor.java | 3 +++ .../extensions/euphoria/core/client/functional/UnaryPredicate.java | 7 ++- .../extensions/euphoria/core/client/functional/VoidFunction.java | 7 ++- .../beam/sdk/extensions/euphoria/core/client/io/Collector.java | 2 ++ .../beam/sdk/extensions/euphoria/core/client/io/Context.java | 7 ++- .../beam/sdk/extensions/euphoria/core/client/io/Environment.java | 7 ++- .../apache/beam/sdk/extensions/euphoria/core/client/lib/Split.java | 3 +++ .../extensions/euphoria/core/client/operator/AssignEventTime.java | 3 +++ .../euphoria/core/client/operator/CompositeOperator.java | 2 ++ .../sdk/extensions/euphoria/core/client/operator/CountByKey.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/Distinct.java | 3 +++ .../beam/sdk/extensions/euphoria/core/client/operator/Filter.java | 3 +++ .../beam/sdk/extensions/euphoria/core/client/operator/FlatMap.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/FullJoin.java | 3 +++ .../beam/sdk/extensions/euphoria/core/client/operator/Join.java| 3 +++ .../sdk/extensions/euphoria/core/client/operator/LeftJoin.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/MapElements.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/ReduceByKey.java | 2 ++ .../sdk/extensions/euphoria/core/client/operator/ReduceWindow.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/RightJoin.java| 3 +++ .../sdk/extensions/euphoria/core/client/operator/SumByKey.java | 3 +++ .../sdk/extensions/euphoria/core/client/operator/TopPerKey.java| 3 +++ .../beam/sdk/extensions/euphoria/core/client/operator/Union.java | 3 +++ .../extensions/euphoria/core/client/operator/base/Builders.java| 3 +++ .../euphoria/core/client/operator/base/OptionalMethodBuilder.java | 2 ++ .../extensions/euphoria/core/client/operator/base/WindowAware.java | 2 ++ .../extensions/euphoria/core/client/operator/hint/OutputHint.java | 2 +- .../beam/sdk/extensions/euphoria/core/client/type/TypeAware.java | 3 ++- .../sdk/extensions/euphoria/core/client/type/TypeAwareness.java| 2 ++ .../beam/sdk/extensions/euphoria/core/client/type/TypeUtils.java | 7 ++- .../apache/beam/sdk/extensions/euphoria/core/client/util/Fold.java | 2 ++ .../apache/beam/sdk/extensions/euphoria/core/client/util/Max.java | 7 ++- .../sdk/extensions/euphoria/core/client/util/PCollectionLists.java | 2 ++ .../apache/beam/sdk/extensions/euphoria/core/client/util/Sums.java | 7 ++- .../euphoria/core/translate/BeamAccumulatorProvider.java | 2 ++ .../euphoria/core/translate/BroadcastHashJoinTranslator.java | 3 +++ .../euphoria/core/translate/CompositeOperatorTranslator.java | 2 ++ .../sdk/extensions/euphoria/core/translate/EuphoriaOptions.java| 7 ++- .../sdk/extensions/euphoria/core/translate/FlatMapTranslator.java | 2 ++ .../sdk/extensions/euphoria/core/translate/JoinTranslator.java | 7
(beam) 01/01: Merge pull request #29162: [runners-spark] Do not set accTimestamp to null in SparkCombineFn (#28256)
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 6a66b722f463d602b85274670344138ec01488c7 Merge: ba714221d5e 524a7bfef40 Author: Jan Lukavský AuthorDate: Tue Oct 31 11:11:26 2023 +0100 Merge pull request #29162: [runners-spark] Do not set accTimestamp to null in SparkCombineFn (#28256) .../runners/spark/translation/SparkCombineFn.java | 18 ++ .../spark/translation/SparkCombineFnTest.java | 29 ++ 2 files changed, 37 insertions(+), 10 deletions(-)
(beam) branch master updated (ba714221d5e -> 6a66b722f46)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from ba714221d5e organize and refactor GrpcWindmillServer. (#29156) add 524a7bfef40 [runners-spark] Do not set accTimestamp to null in SparkCombineFn (#28256) new 6a66b722f46 Merge pull request #29162: [runners-spark] Do not set accTimestamp to null in SparkCombineFn (#28256) The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../runners/spark/translation/SparkCombineFn.java | 18 ++ .../spark/translation/SparkCombineFnTest.java | 29 ++ 2 files changed, 37 insertions(+), 10 deletions(-)
[beam] 01/01: Merge pull request #29102: [flink] Flush buffer during drain operation for requiresStableInput operator
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit bf5ded44e6ee7e9e752f44379df43a3aa453fc7e Merge: 3f05945888e 1a27d2cee0c Author: Jan Lukavský AuthorDate: Tue Oct 24 08:29:50 2023 +0200 Merge pull request #29102: [flink] Flush buffer during drain operation for requiresStableInput operator .../beam/runners/flink/FlinkPipelineOptions.java | 8 ++ .../wrappers/streaming/DoFnOperator.java | 14 .../wrappers/streaming/DoFnOperatorTest.java | 92 ++ .../shortcodes/flink_java_pipeline_options.html| 5 ++ .../shortcodes/flink_python_pipeline_options.html | 5 ++ 5 files changed, 124 insertions(+)
[beam] branch master updated (3f05945888e -> bf5ded44e6e)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 3f05945888e [YAML] Schema-producing Create. (#29030) add 1a27d2cee0c flush buffer during drain operation for requiresStableInput operator (#28554) new bf5ded44e6e Merge pull request #29102: [flink] Flush buffer during drain operation for requiresStableInput operator The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../beam/runners/flink/FlinkPipelineOptions.java | 8 ++ .../wrappers/streaming/DoFnOperator.java | 14 .../wrappers/streaming/DoFnOperatorTest.java | 92 ++ .../shortcodes/flink_java_pipeline_options.html| 5 ++ .../shortcodes/flink_python_pipeline_options.html | 5 ++ 5 files changed, 124 insertions(+)
[beam] branch master updated (451561784c6 -> a0d014ae36f)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 451561784c6 Moving to 2.52.0-SNAPSHOT on master branch. add 77dfa5ae7c8 [runners-flink] #28258 Wait infinitely if the duration value is less than 1ms in FlinkDetachedRunnerResult add a0d014ae36f Merge pull request #28262: [runners-flink] #28258 Wait infinitely if the duration value is less than 1 ms No new revisions were added by this update. Summary of changes: .../java/org/apache/beam/runners/flink/FlinkDetachedRunnerResult.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
[beam] 01/01: Merge pull request #26096: Allow to set operator chaining for flink runner (#26095)
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit dc7f9d11498b5bec567bcb5f29ff27ed2c1fde23 Merge: 6179adfe6d1 df8bf40aa4b Author: Jan Lukavský AuthorDate: Sat Apr 8 13:47:10 2023 +0200 Merge pull request #26096: Allow to set operator chaining for flink runner (#26095) .../org/apache/beam/runners/flink/FlinkExecutionEnvironments.java | 4 .../java/org/apache/beam/runners/flink/FlinkPipelineOptions.java| 6 ++ 2 files changed, 10 insertions(+)
[beam] branch master updated (6179adfe6d1 -> dc7f9d11498)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 6179adfe6d1 Fix some style lints. (#26108) add df8bf40aa4b [Feature Request]: Allow to set operator chaining for flink runner (#26095) new dc7f9d11498 Merge pull request #26096: Allow to set operator chaining for flink runner (#26095) The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../org/apache/beam/runners/flink/FlinkExecutionEnvironments.java | 4 .../java/org/apache/beam/runners/flink/FlinkPipelineOptions.java| 6 ++ 2 files changed, 10 insertions(+)
[beam] branch master updated (2fc56ec663e -> e83a996d437)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 2fc56ec663e Merge pull request #24142: Fix arguments to checkState in BatchViewOverrides add ffdee0b6edb Re-use serializable pipeline options when already available (#24192) new e83a996d437 Merge pull request #24192: Re-use serializable pipeline options when already available. The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../flink/translation/types/CoderTypeInformation.java | 6 +- .../wrappers/streaming/KvToByteBufferKeySelector.java | 2 +- .../wrappers/streaming/SdfByteBufferKeySelector.java| 2 +- .../wrappers/streaming/WorkItemKeySelector.java | 2 +- .../wrappers/streaming/io/UnboundedSourceWrapper.java | 3 +-- .../streaming/state/FlinkBroadcastStateInternals.java | 17 - 6 files changed, 17 insertions(+), 15 deletions(-)
[beam] 01/01: Merge pull request #24192: Re-use serializable pipeline options when already available.
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit e83a996d4374d467d95bcfad7166905622ec615c Merge: 2fc56ec663e ffdee0b6edb Author: Jan Lukavský AuthorDate: Wed Nov 16 15:15:31 2022 +0100 Merge pull request #24192: Re-use serializable pipeline options when already available. .../flink/translation/types/CoderTypeInformation.java | 6 +- .../wrappers/streaming/KvToByteBufferKeySelector.java | 2 +- .../wrappers/streaming/SdfByteBufferKeySelector.java| 2 +- .../wrappers/streaming/WorkItemKeySelector.java | 2 +- .../wrappers/streaming/io/UnboundedSourceWrapper.java | 3 +-- .../streaming/state/FlinkBroadcastStateInternals.java | 17 - 6 files changed, 17 insertions(+), 15 deletions(-)
[beam] 01/01: Merge pull request #17809: [BEAM-14265] Add watermark hold for all timers
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 7bf822a541939ff6874d60d4ef18957bc05128b8 Merge: 080f54a144a 6e450a92da7 Author: Jan Lukavský AuthorDate: Tue Jun 14 14:30:15 2022 +0200 Merge pull request #17809: [BEAM-14265] Add watermark hold for all timers .../beam/runners/core/StatefulDoFnRunner.java | 21 +--- .../wrappers/streaming/DoFnOperator.java | 109 ++--- .../streaming/state/FlinkStateInternals.java | 18 +++- .../wrappers/streaming/DoFnOperatorTest.java | 28 +++--- 4 files changed, 58 insertions(+), 118 deletions(-)
[beam] branch master updated (080f54a144a -> 7bf822a5419)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 080f54a144a [BEAM-13806] Add x-lang BigQuery IO integration test to Go SDK. (#16818) add 6e450a92da7 [BEAM-14265] Add watermark hold for all timers new 7bf822a5419 Merge pull request #17809: [BEAM-14265] Add watermark hold for all timers The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../beam/runners/core/StatefulDoFnRunner.java | 21 +--- .../wrappers/streaming/DoFnOperator.java | 109 ++--- .../streaming/state/FlinkStateInternals.java | 18 +++- .../wrappers/streaming/DoFnOperatorTest.java | 28 +++--- 4 files changed, 58 insertions(+), 118 deletions(-)
[beam] 01/01: Merge pull request #17715: [BEAM-14492] add flinkConfDir to FlinkPipelineOptions
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit c5e521a85f93527b6b3fe20aea505206316ce7ce Merge: fd854635552 8a389892373 Author: Jan Lukavský AuthorDate: Thu May 26 09:13:05 2022 +0200 Merge pull request #17715: [BEAM-14492] add flinkConfDir to FlinkPipelineOptions .../runners/flink/FlinkExecutionEnvironments.java | 26 +++-- .../flink/FlinkPipelineExecutionEnvironment.java | 8 +- .../beam/runners/flink/FlinkPipelineOptions.java | 8 ++ .../flink/FlinkExecutionEnvironmentsTest.java | 118 +++-- .../shortcodes/flink_java_pipeline_options.html| 5 + .../shortcodes/flink_python_pipeline_options.html | 5 + 6 files changed, 72 insertions(+), 98 deletions(-)
[beam] branch master updated (fd854635552 -> c5e521a85f9)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from fd854635552 Merge pull request #17759: Fix 2.38.0 download page. add 8a389892373 [BEAM-14492] add flinkConfDir to FlinkPipelineOptions new c5e521a85f9 Merge pull request #17715: [BEAM-14492] add flinkConfDir to FlinkPipelineOptions The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../runners/flink/FlinkExecutionEnvironments.java | 26 +++-- .../flink/FlinkPipelineExecutionEnvironment.java | 8 +- .../beam/runners/flink/FlinkPipelineOptions.java | 8 ++ .../flink/FlinkExecutionEnvironmentsTest.java | 118 +++-- .../shortcodes/flink_java_pipeline_options.html| 5 + .../shortcodes/flink_python_pipeline_options.html | 5 + 6 files changed, 72 insertions(+), 98 deletions(-)
[beam] branch master updated (e7d3e8c9814 -> 3ac86e46e82)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from e7d3e8c9814 [BEAM-14048] [CdapIO] Add ConfigWrapper for building CDAP PluginConfigs (#17051) add 09e3290fee4 [BEAM-14196] add test verifying output watermark propagation in bundle add 51e882c0309 [BEAM-14196] Fix FlinkRunner mid-bundle output watermark handling new 3ac86e46e82 Merge pull request #17504: [BEAM-14196] add test verifying output watermark propagation in bundle The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../wrappers/streaming/DoFnOperator.java | 46 ++--- .../streaming/ExecutableStageDoFnOperator.java | 12 ++--- .../beam/runners/flink/FlinkSavepointTest.java | 2 +- .../wrappers/streaming/DoFnOperatorTest.java | 35 ++--- .../streaming/ExecutableStageDoFnOperatorTest.java | 3 +- .../wrappers/streaming/WindowDoFnOperatorTest.java | 9 .../org/apache/beam/sdk/transforms/ParDoTest.java | 58 +- 7 files changed, 129 insertions(+), 36 deletions(-)
[beam] 01/01: Merge pull request #17504: [BEAM-14196] add test verifying output watermark propagation in bundle
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 3ac86e46e827a0874f051a2c7035d4c01f0caa56 Merge: e7d3e8c9814 51e882c0309 Author: Jan Lukavský AuthorDate: Thu May 5 10:02:42 2022 +0200 Merge pull request #17504: [BEAM-14196] add test verifying output watermark propagation in bundle .../wrappers/streaming/DoFnOperator.java | 46 ++--- .../streaming/ExecutableStageDoFnOperator.java | 12 ++--- .../beam/runners/flink/FlinkSavepointTest.java | 2 +- .../wrappers/streaming/DoFnOperatorTest.java | 35 ++--- .../streaming/ExecutableStageDoFnOperatorTest.java | 3 +- .../wrappers/streaming/WindowDoFnOperatorTest.java | 9 .../org/apache/beam/sdk/transforms/ParDoTest.java | 58 +- 7 files changed, 129 insertions(+), 36 deletions(-)
[beam] branch master updated (fc735fe -> 16355ca)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from fc735fe Use ThreadLocal for DESERIALIZATION_CONTEXT (#16680) add bb0083f Add reference to Books to Learning Resources in website add 16355ca Merge pull request #16682: Add reference to Books to Learning Resources in website No new revisions were added by this update. Summary of changes: .../content/en/documentation/resources/learning-resources.md | 10 ++ .../www/site/layouts/partials/section-menu/en/get-started.html | 1 + 2 files changed, 11 insertions(+)
[beam] branch master updated (2e5162e -> b236713)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 2e5162e Merge pull request #15387 from ibzib/ignore-recommendation-py add a4a6f7b [BEAM-12704] Failing test for Flink primitive Read add f17b66d [BEAM-12704] Primitive Read working on Flink add b236713 Merge pull request #15370: [BEAM-12704] flink primitive read No new revisions were added by this update. Summary of changes: .../FlinkStreamingPortablePipelineTranslator.java | 187 + .../translation/types/CoderTypeSerializer.java | 11 +- .../translation/wrappers/SourceInputFormat.java| 6 +- .../streaming/ExecutableStageDoFnOperator.java | 5 + .../beam/runners/flink/ReadSourcePortableTest.java | 134 ++- 5 files changed, 297 insertions(+), 46 deletions(-)
[beam] branch master updated (7edca9e -> a02ccfc)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 7edca9e Merge pull request #15013 from pabloem/bqout-successes add fdce248 Fix help of apache_beam.trigger. add a02ccfc Merge pull request #15177: Fix help of apache_beam.trigger. No new revisions were added by this update. Summary of changes: sdks/python/apache_beam/transforms/trigger.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
[beam] branch master updated (54b6ca3 -> 696abe8)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 54b6ca3 [BEAM-9547] Add support for `Series.value_counts()`, add back the ability to group by a Series (#14929) add 4fd9685 [BEAM-12473] fix possible ClassCastException with UDAF add 696abe8 Merge pull request #14986: [BEAM-12473] fix possible ClassCastException with UDAF No new revisions were added by this update. Summary of changes: .../sdk/extensions/sql/TypedCombineFnDelegate.java | 164 + .../beam/sdk/extensions/sql/impl/UdafImpl.java | 22 ++- .../extensions/sql/impl/utils/CalciteUtils.java| 21 ++- .../extensions/sql/TypedCombineFnDelegateTest.java | 85 +++ 4 files changed, 282 insertions(+), 10 deletions(-) create mode 100644 sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/TypedCombineFnDelegate.java create mode 100644 sdks/java/extensions/sql/src/test/java/org/apache/beam/sdk/extensions/sql/TypedCombineFnDelegateTest.java
[beam] 01/01: Merge pull request #14718: [BEAM-12276] Narrow check for output timestamp of event time timers
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 4cc279c364879c0a6501ca9a961bb957b0f343aa Merge: f207ad1 a7b74b0 Author: Jan Lukavský AuthorDate: Thu May 27 10:24:27 2021 +0200 Merge pull request #14718: [BEAM-12276] Narrow check for output timestamp of event time timers .../apache/beam/runners/core/SimpleDoFnRunner.java | 6 +- .../org/apache/beam/sdk/transforms/ParDoTest.java | 76 ++ 2 files changed, 79 insertions(+), 3 deletions(-)
[beam] branch master updated (f207ad1 -> 4cc279c)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from f207ad1 [BEAM-12074] Define docstrings for all Series and StringMethods operations (#14863) add a7b74b0 [BEAM-12276] Narrow check for output timestamp of event time timers new 4cc279c Merge pull request #14718: [BEAM-12276] Narrow check for output timestamp of event time timers The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../apache/beam/runners/core/SimpleDoFnRunner.java | 6 +- .../org/apache/beam/sdk/transforms/ParDoTest.java | 76 ++ 2 files changed, 79 insertions(+), 3 deletions(-)
[beam] branch master updated (9aa9c86 -> 3640ae2)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 9aa9c86 [BEAM-7372][BEAM-9372] cleanup python 2.x and 3.5 codepaths (#13913) add 4dd7789 Simplify LateDataDropping runner. add 3640ae2 Merge pull request #13523: Simplify LateDataDropping runner. No new revisions were added by this update. Summary of changes: .../runners/core/LateDataDroppingDoFnRunner.java | 74 +- 1 file changed, 29 insertions(+), 45 deletions(-)
[beam] branch master updated (04e3343 -> 7aa7e13)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 04e3343 Merge pull request #13721 from KevinGG/BEAM-11588 add 8e1a8b8 [BEAM-10402] Trigger#subTriggers does not return null add 7aa7e13 Merge pull request #13719: [BEAM-10402] Trigger#subTriggers might return null No new revisions were added by this update. Summary of changes: .../main/java/org/apache/beam/sdk/transforms/windowing/Trigger.java| 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-)
[beam] branch master updated (80248d0 -> bea6f88)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 80248d0 Merge pull request #13318: [BEAM-11300] Improve Nexmark performance: add 8de6faa [BEAM-11481] emit output watermark on watermark hold change new bea6f88 Merge pull request #13571: [BEAM-11481] emit output watermark on watermark hold change The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../wrappers/streaming/DoFnOperator.java | 23 ++- .../streaming/KeyedPushedBackElementsHandler.java | 5 +- .../streaming/state/FlinkStateInternals.java | 4 +- .../wrappers/streaming/DoFnOperatorTest.java | 169 - 4 files changed, 194 insertions(+), 7 deletions(-)
[beam] 01/01: Merge pull request #13571: [BEAM-11481] emit output watermark on watermark hold change
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit bea6f88b747dd582b0df6db1c265846f304c6abf Merge: 80248d0 8de6faa Author: Jan Lukavský AuthorDate: Fri Dec 18 18:37:27 2020 +0100 Merge pull request #13571: [BEAM-11481] emit output watermark on watermark hold change .../wrappers/streaming/DoFnOperator.java | 23 ++- .../streaming/KeyedPushedBackElementsHandler.java | 5 +- .../streaming/state/FlinkStateInternals.java | 4 +- .../wrappers/streaming/DoFnOperatorTest.java | 169 - 4 files changed, 194 insertions(+), 7 deletions(-)
[beam] branch master updated (744637d -> 76723d8)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 744637d Merge pull request #13522: [BEAM-11435] Reuse already set timers. add 190261a Log exception caught during UnboundedSource#split add 76723d8 Merge pull request #13520: Log exception caught during UnboundedSource#split No new revisions were added by this update. Summary of changes: sdks/java/core/src/main/java/org/apache/beam/sdk/io/Read.java | 1 + 1 file changed, 1 insertion(+)
[beam] branch master updated (7223fb3 -> ef21dc1)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 7223fb3 Merge pull request #13252 [BEAM-9547] Provide some top level pandas functions. add 02ddf09 [BEAM-11191] fix ClassCastException when clearing watermark state add ef21dc1 Merge pull request #13269: [BEAM-11191] fix ClassCastException when clearing watermark state No new revisions were added by this update. Summary of changes: .../streaming/state/FlinkStateInternals.java | 94 ++ .../flink/streaming/FlinkStateInternalsTest.java | 18 - 2 files changed, 95 insertions(+), 17 deletions(-)
[beam] branch master updated (02bf081 -> b5846c6)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 02bf081 [BEAM-10704] Fix Py3 incompatibility in stager.py. #12579 add 55874cc [BEAM-10691] Use FlinkStateInternals#addWatermarkHoldUsage for timer output timestamp add b5846c6 Merge pull request #12551: [BEAM-10691] Use FlinkStateInternals#addWatermarkHoldUsage for timer output timestamp No new revisions were added by this update. Summary of changes: .../wrappers/streaming/DoFnOperator.java | 33 -- .../streaming/state/FlinkStateInternals.java | 21 +- .../wrappers/streaming/DoFnOperatorTest.java | 2 +- .../wrappers/streaming/WindowDoFnOperatorTest.java | 2 -- 4 files changed, 13 insertions(+), 45 deletions(-)
[beam] branch master updated (a52d6b8 -> 3a3d162)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from a52d6b8 Merge pull request #12379: [BEAM-9702] Add support of AWS SDK v2 for KinesisIO.Read (Java) to CHANGES.md add 96a9db3 [BEAM-8648] Deprecate OutputHints from Euphoria API. add 3a3d162 Merge pull request #12388: [BEAM-8648] Deprecate OutputHints from Euphoria API No new revisions were added by this update. Summary of changes: .../core/client/operator/AssignEventTime.java | 8 ++--- .../euphoria/core/client/operator/CountByKey.java | 3 +- .../euphoria/core/client/operator/Distinct.java| 3 +- .../euphoria/core/client/operator/Filter.java | 3 +- .../euphoria/core/client/operator/FlatMap.java | 3 +- .../euphoria/core/client/operator/Join.java| 10 ++ .../euphoria/core/client/operator/MapElements.java | 3 +- .../euphoria/core/client/operator/ReduceByKey.java | 5 ++- .../core/client/operator/ReduceWindow.java | 3 +- .../euphoria/core/client/operator/SumByKey.java| 3 +- .../euphoria/core/client/operator/TopPerKey.java | 3 +- .../euphoria/core/client/operator/Union.java | 3 +- .../core/client/operator/base/Builders.java| 40 +++--- .../core/client/operator/hint/ComputationHint.java | 7 +++- .../core/client/operator/hint/OutputHint.java | 7 +++- .../core/client/operator/hint/SizeHint.java| 7 +++- 16 files changed, 70 insertions(+), 41 deletions(-)
[beam] 01/01: Merge pull request #12317: [BEAM-10510] fix potential NPE in UnboundedSourceWrapper#close
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 0fc74c09f1b90c1b954b78973e83483daa51fd18 Merge: 8b84720 de7c9f8 Author: Jan Lukavský AuthorDate: Mon Jul 20 17:27:31 2020 +0200 Merge pull request #12317: [BEAM-10510] fix potential NPE in UnboundedSourceWrapper#close .../translation/wrappers/streaming/io/UnboundedSourceWrapper.java | 4 +++- 1 file changed, 3 insertions(+), 1 deletion(-)
[beam] branch master updated (8b84720 -> 0fc74c0)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 8b84720 Merge pull request #12316 from damgad/damgad_fix_upload_wheels_to_gcs add de7c9f8 [BEAM-10510] fix potential NPE in UnboundedSourceWrapper#close new 0fc74c0 Merge pull request #12317: [BEAM-10510] fix potential NPE in UnboundedSourceWrapper#close The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../translation/wrappers/streaming/io/UnboundedSourceWrapper.java | 4 +++- 1 file changed, 3 insertions(+), 1 deletion(-)
[beam] 01/01: Merge pull request #12315: [BEAM-10533] Remove watermark hold from RequiresTimeSortedInput
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit af51670d08fbcf76a6d5fca82d47adf61dc99507 Merge: e6b9ee5 8dfc187 Author: Jan Lukavský AuthorDate: Mon Jul 20 12:38:56 2020 +0200 Merge pull request #12315: [BEAM-10533] Remove watermark hold from RequiresTimeSortedInput .../apache/beam/runners/core/StatefulDoFnRunner.java | 18 +++--- 1 file changed, 11 insertions(+), 7 deletions(-)
[beam] branch master updated (e6b9ee5 -> af51670)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from e6b9ee5 Merge pull request #12145: [BEAM-10136] [BEAM-10135] Add JdbcIO for cross-language with python wrapper add 8dfc187 [BEAM-10533] Remove watermark hold from RequiresTimeSortedInput new af51670 Merge pull request #12315: [BEAM-10533] Remove watermark hold from RequiresTimeSortedInput The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../apache/beam/runners/core/StatefulDoFnRunner.java | 18 +++--- 1 file changed, 11 insertions(+), 7 deletions(-)
[beam] branch master updated (5e15717 -> e5dfa5b)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 5e15717 Pass pipeline options from caller to expansion service (#11574) add a481397 Disable two flaky tests (BEAM-8035, BEAM-9164) new e5dfa5b Merge pull request #11614: Disable two flaky tests (BEAM-8035, BEAM-9164) The 1 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../translation/wrappers/streaming/io/UnboundedSourceWrapperTest.java | 2 ++ .../core/src/test/java/org/apache/beam/sdk/transforms/WatchTest.java| 2 ++ 2 files changed, 4 insertions(+)
[beam] 01/01: Merge pull request #11614: Disable two flaky tests (BEAM-8035, BEAM-9164)
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit e5dfa5bfe182e820be9449be8b849d4769235e54 Merge: 5e15717 a481397 Author: Jan Lukavský AuthorDate: Wed May 6 09:56:27 2020 +0200 Merge pull request #11614: Disable two flaky tests (BEAM-8035, BEAM-9164) .../translation/wrappers/streaming/io/UnboundedSourceWrapperTest.java | 2 ++ .../core/src/test/java/org/apache/beam/sdk/transforms/WatchTest.java| 2 ++ 2 files changed, 4 insertions(+)
[beam] branch master updated (b6b6312 -> a149b6b)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from b6b6312 [BEAM-9317] Fix Dataflow tests to not perform SplittableDoFn expansion at the moment. (#10868) add 85eeb10 [BEAM-9273] Explicitly disable @RequiresTimeSortedInput on unsupported runners add 9e45f9d [BEAM-9273] code review - to be squashed add a149b6b Merge pull request #10816: [BEAM-9273] Explicitly disable @RequiresTimeSortedInput on unsupported runners No new revisions were added by this update. Summary of changes: .../runners/apex/translation/ParDoTranslator.java | 13 +- .../translators/ParDoMultiOutputTranslator.java| 9 + .../dataflow/BatchStatefulParDoOverrides.java | 4 +- .../dataflow/DataflowPipelineTranslator.java | 9 +- .../beam/runners/dataflow/DataflowRunner.java | 44 ++-- .../java/org/apache/beam/runners/jet/Utils.java| 8 +- .../runners/samza/runtime/SamzaDoFnRunners.java| 2 +- .../translation/ParDoBoundMultiTranslator.java | 11 +- .../translation/batch/ParDoTranslatorBatch.java| 15 +- .../sdk/transforms/reflect/DoFnSignatures.java | 50 + .../sdk/transforms/reflect/DoFnSignaturesTest.java | 240 + 11 files changed, 360 insertions(+), 45 deletions(-)
[beam] branch master updated (c1817f4 -> 9710fce)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from c1817f4 Merge pull request #10707 from ibzib/zetasql-error-str add 8654f20 [BEAM-9265] @RequiresTimeSortedInput respects allowedLateness add 9710fce Merge pull request #10795: [BEAM-9265] @RequiresTimeSortedInput respects allowedLateness No new revisions were added by this update. Summary of changes: .../beam/runners/core/StatefulDoFnRunner.java | 29 +++--- .../beam/runners/core/StatefulDoFnRunnerTest.java | 23 ++- .../sdk/testing/UsesRequiresTimeSortedInput.java | 2 +- .../org/apache/beam/sdk/transforms/ParDoTest.java | 34 ++ 4 files changed, 69 insertions(+), 19 deletions(-)
[beam] branch master updated: [BEAM-8550] @RequiresTimeSortedInput: working with legacy flink and spark
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 377f1ac [BEAM-8550] @RequiresTimeSortedInput: working with legacy flink and spark new 041f7af Merge pull request #8774 from je-ik/requires-time-sorted-input-draft: [BEAM-8550] Requires time sorted input 377f1ac is described below commit 377f1ac7ebbc4253299e7efbdb3ad58d0c9e14c5 Author: Jan Lukavsky AuthorDate: Thu Jan 30 13:10:31 2020 +0100 [BEAM-8550] @RequiresTimeSortedInput: working with legacy flink and spark --- .gitignore | 1 + .../pipeline/src/main/proto/beam_runner_api.proto | 4 +- .../translation/operators/ApexParDoOperator.java | 8 +- .../core/construction/ParDoTranslation.java| 8 + .../runners/core/construction/SplittableParDo.java | 5 + .../org/apache/beam/runners/core/DoFnRunners.java | 60 - .../apache/beam/runners/core/SimpleDoFnRunner.java | 4 +- .../beam/runners/core/StatefulDoFnRunner.java | 172 ++--- .../SimplePushbackSideInputDoFnRunnerTest.java | 26 +- .../beam/runners/core/StatefulDoFnRunnerTest.java | 285 ++--- .../apache/beam/runners/direct/ParDoEvaluator.java | 32 ++- .../runners/direct/ParDoMultiOverrideFactory.java | 73 +++--- .../beam/runners/direct/QuiescenceDriver.java | 2 +- .../direct/StatefulParDoEvaluatorFactoryTest.java | 149 +-- .../FlinkBatchPortablePipelineTranslator.java | 11 + .../flink/FlinkBatchTransformTranslators.java | 28 +- .../FlinkStreamingPortablePipelineTranslator.java | 1 - .../flink/FlinkStreamingTransformTranslators.java | 8 - .../utils/FlinkPortableRunnerUtils.java| 58 + .../wrappers/streaming/DoFnOperator.java | 59 +++-- .../streaming/ExecutableStageDoFnOperator.java | 114 + .../wrappers/streaming/SplittableDoFnOperator.java | 6 +- .../wrappers/streaming/WindowDoFnOperator.java | 4 +- .../runners/flink/FlinkPipelineOptionsTest.java| 2 - .../wrappers/streaming/DoFnOperatorTest.java | 21 -- .../streaming/ExecutableStageDoFnOperatorTest.java | 6 +- .../dataflow/PrimitiveParDoSingleFactory.java | 5 + .../runners/samza/runtime/SamzaDoFnRunners.java| 5 +- .../beam/runners/spark/coders/CoderHelpers.java| 47 .../spark/translation/TransformTranslator.java | 175 +++-- .../spark/translation/TransformTranslatorTest.java | 106 .../apache/beam/sdk/runners/AppliedPTransform.java | 9 + .../sdk/testing/UsesRequiresTimeSortedInput.java | 27 ++ .../java/org/apache/beam/sdk/transforms/DoFn.java | 27 ++ .../beam/sdk/transforms/reflect/DoFnSignature.java | 8 + .../sdk/transforms/reflect/DoFnSignatures.java | 2 + .../org/apache/beam/sdk/transforms/ParDoTest.java | 191 +- 37 files changed, 1492 insertions(+), 257 deletions(-) diff --git a/.gitignore b/.gitignore index 5732b9c..f030006 100644 --- a/.gitignore +++ b/.gitignore @@ -8,6 +8,7 @@ # Ignore files generated by the Gradle build process. **/.gradle/**/* **/.gogradle/**/* +**/.nb-gradle/**/* **/gogradle.lock **/build/**/* .test-infra/**/vendor/**/* diff --git a/model/pipeline/src/main/proto/beam_runner_api.proto b/model/pipeline/src/main/proto/beam_runner_api.proto index 57c5295..81e4d2d 100644 --- a/model/pipeline/src/main/proto/beam_runner_api.proto +++ b/model/pipeline/src/main/proto/beam_runner_api.proto @@ -175,7 +175,6 @@ message StandardPTransforms { enum Primitives { // Represents Beam's parallel do operation. // Payload: ParDoPayload. -// TODO(BEAM-3595): Change this to beam:transform:pardo:v1. PAR_DO = 0 [(beam_urn) = "beam:transform:pardo:v1"]; // Represents Beam's flatten operation. @@ -398,6 +397,9 @@ message ParDoPayload { // (Optional) A mapping of local timer family names to timer specifications. map timer_family_specs = 9; + + // Whether this stage requires time sorted input + bool requires_time_sorted_input = 10; } // Parameters that a UDF might require. diff --git a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/operators/ApexParDoOperator.java b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/operators/ApexParDoOperator.java index 4841c6a..8df7997 100644 --- a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/operators/ApexParDoOperator.java +++ b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/operators/ApexParDoOperator.java @@ -511,7 +511,13 @@ public class ApexParDoOperator extends BaseOperator doFnRunner = DoFnRunners.defaultStatefulDoFnRunner( - doFn, doFnRunner, windowingStrategy, cleanupTimer, stateCleaner); + doFn, +
[beam] branch master updated: [BEAM-8450] Allow empty bundles in ParDoLifecycleTest
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new b3cb103 [BEAM-8450] Allow empty bundles in ParDoLifecycleTest new 49ef06e Merge pull request #9848 from je-ik/BEAM-8450: [BEAM-8450] Allow empty bundles in ParDoLifecycleTest b3cb103 is described below commit b3cb103b9a9a5a4c8b39ef96dafd94b6d386d45e Author: Jan Lukavsky AuthorDate: Mon Oct 21 20:18:21 2019 +0200 [BEAM-8450] Allow empty bundles in ParDoLifecycleTest --- .../beam/sdk/transforms/ParDoLifecycleTest.java| 94 +++--- 1 file changed, 66 insertions(+), 28 deletions(-) diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/transforms/ParDoLifecycleTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/transforms/ParDoLifecycleTest.java index 0685644..5f71f86 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/transforms/ParDoLifecycleTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/transforms/ParDoLifecycleTest.java @@ -29,6 +29,10 @@ import static org.hamcrest.Matchers.nullValue; import static org.junit.Assert.fail; import java.io.Serializable; +import java.util.ArrayList; +import java.util.Arrays; +import java.util.Collections; +import java.util.List; import java.util.Map; import java.util.concurrent.ConcurrentHashMap; import java.util.concurrent.CountDownLatch; @@ -36,6 +40,7 @@ import java.util.concurrent.TimeUnit; import java.util.concurrent.atomic.AtomicBoolean; import java.util.concurrent.atomic.AtomicInteger; import java.util.concurrent.atomic.AtomicReference; +import java.util.stream.Collectors; import org.apache.beam.sdk.state.StateSpec; import org.apache.beam.sdk.state.StateSpecs; import org.apache.beam.sdk.state.ValueState; @@ -47,6 +52,7 @@ import org.apache.beam.sdk.values.KV; import org.apache.beam.sdk.values.PCollectionList; import org.apache.beam.sdk.values.TupleTag; import org.apache.beam.sdk.values.TupleTagList; +import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.base.MoreObjects; import org.junit.Before; import org.junit.Rule; import org.junit.Test; @@ -172,7 +178,7 @@ public class ParDoLifecycleTest implements Serializable { p.run(); fail("Pipeline should have failed with an exception"); } catch (Exception e) { - validate(); + validate(CallState.SETUP, CallState.TEARDOWN); } } @@ -185,7 +191,7 @@ public class ParDoLifecycleTest implements Serializable { p.run(); fail("Pipeline should have failed with an exception"); } catch (Exception e) { - validate(); + validate(CallState.SETUP, CallState.START_BUNDLE, CallState.TEARDOWN); } } @@ -198,7 +204,8 @@ public class ParDoLifecycleTest implements Serializable { p.run(); fail("Pipeline should have failed with an exception"); } catch (Exception e) { - validate(); + validate( + CallState.SETUP, CallState.START_BUNDLE, CallState.PROCESS_ELEMENT, CallState.TEARDOWN); } } @@ -211,7 +218,12 @@ public class ParDoLifecycleTest implements Serializable { p.run(); fail("Pipeline should have failed with an exception"); } catch (Exception e) { - validate(); + validate( + CallState.SETUP, + CallState.START_BUNDLE, + CallState.PROCESS_ELEMENT, + CallState.FINISH_BUNDLE, + CallState.TEARDOWN); } } @@ -224,7 +236,7 @@ public class ParDoLifecycleTest implements Serializable { p.run(); fail("Pipeline should have failed with an exception"); } catch (Exception e) { - validate(); + validate(CallState.SETUP, CallState.TEARDOWN); } } @@ -237,7 +249,7 @@ public class ParDoLifecycleTest implements Serializable { p.run(); fail("Pipeline should have failed with an exception"); } catch (Exception e) { - validate(); + validate(CallState.SETUP, CallState.START_BUNDLE, CallState.TEARDOWN); } } @@ -250,11 +262,30 @@ public class ParDoLifecycleTest implements Serializable { p.run(); fail("Pipeline should have failed with an exception"); } catch (Exception e) { - validate(); + validate( + CallState.SETUP, CallState.START_BUNDLE, CallState.PROCESS_ELEMENT, CallState.TEARDOWN); } } - private void validate() { + @Test + @Category({ValidatesRunner.class, UsesStatefulParDo.class, UsesParDoLifecycle.class}) + public void testTeardownCalledAfterExceptionInFinishBundleStateful() { +ExceptionThrowingFn fn = new ExceptionThrowingStatefulFn(MethodForException.FINISH_BUNDLE); +p.apply(Create.of(KV.of("a", 1), KV.of("b", 2), KV.of("a", 3))).apply(ParDo.of(fn)); +try { +
[beam] branch master updated: [BEAM-7520] ignore UsesStrictTimerOrdering for both batch and streaming flink
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 4725b54 [BEAM-7520] ignore UsesStrictTimerOrdering for both batch and streaming flink new 21f9fe1 Merge pull request #9869 from je-ik/BEAM-7520-ignore-timer-ordering-for-flink-pvr-batch:[BEAM-7520] ignore UsesStrictTimerOrdering for Flink 4725b54 is described below commit 4725b54fe457ee44c733a83b0d34adb50ff193ae Author: Jan Lukavsky AuthorDate: Thu Oct 24 16:34:45 2019 +0200 [BEAM-7520] ignore UsesStrictTimerOrdering for both batch and streaming flink --- runners/flink/job-server/flink_job_server.gradle | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/runners/flink/job-server/flink_job_server.gradle b/runners/flink/job-server/flink_job_server.gradle index d6664cc..560f195 100644 --- a/runners/flink/job-server/flink_job_server.gradle +++ b/runners/flink/job-server/flink_job_server.gradle @@ -147,9 +147,9 @@ def portableValidatesRunnerTask(String name, Boolean streaming) { excludeCategories 'org.apache.beam.sdk.testing.UsesParDoLifecycle' excludeCategories 'org.apache.beam.sdk.testing.UsesMapState' excludeCategories 'org.apache.beam.sdk.testing.UsesSetState' + excludeCategories 'org.apache.beam.sdk.testing.UsesStrictTimerOrdering' if (streaming) { excludeCategories 'org.apache.beam.sdk.testing.UsesTestStreamWithProcessingTime' -excludeCategories 'org.apache.beam.sdk.testing.UsesStrictTimerOrdering' } else { excludeCategories 'org.apache.beam.sdk.testing.UsesTestStream' }
[beam] branch master updated (4fa1501 -> 2ef2736)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 4fa1501 [BEAM-7389] Show code snippet outputs as stdout (#9790) new a7c8766 [BEAM-7520] Add ValidatesRunner test for timer ordering new c29d06b [BEAM-7520] Synchronize watermarks with bundle processing and add timer pushback new 0049037 [BEAM-7520] add UsesStrictTimerOrdering category, ignore test for samza and portable flink new 2ef2736 Merge pull request #9190 from je-ik/BEAM-7520: [BEAM-7520] Fix timer firing order in DirectRunner The 23570 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../beam/runners/direct/DirectTimerInternals.java | 12 + .../beam/runners/direct/EvaluationContext.java | 9 +- .../beam/runners/direct/QuiescenceDriver.java | 6 +- .../direct/StatefulParDoEvaluatorFactory.java | 26 +- .../runners/direct/WatermarkCallbackExecutor.java | 23 +- .../beam/runners/direct/WatermarkManager.java | 199 -- .../beam/runners/direct/DirectRunnerTest.java | 230 +++- .../direct/StatefulParDoEvaluatorFactoryTest.java | 8 + .../beam/runners/direct/WatermarkManagerTest.java | 39 ++- runners/flink/job-server/flink_job_server.gradle | 1 + runners/samza/build.gradle | 1 + ...reMessage.java => UsesStrictTimerOrdering.java} | 6 +- .../org/apache/beam/sdk/transforms/ParDoTest.java | 291 + 13 files changed, 816 insertions(+), 35 deletions(-) copy sdks/java/core/src/main/java/org/apache/beam/sdk/testing/{UsesFailureMessage.java => UsesStrictTimerOrdering.java} (77%)
[beam] branch master updated (4fa1501 -> 2ef2736)
This is an automated email from the ASF dual-hosted git repository. janl pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 4fa1501 [BEAM-7389] Show code snippet outputs as stdout (#9790) new a7c8766 [BEAM-7520] Add ValidatesRunner test for timer ordering new c29d06b [BEAM-7520] Synchronize watermarks with bundle processing and add timer pushback new 0049037 [BEAM-7520] add UsesStrictTimerOrdering category, ignore test for samza and portable flink new 2ef2736 Merge pull request #9190 from je-ik/BEAM-7520: [BEAM-7520] Fix timer firing order in DirectRunner The 23570 revisions listed above as "new" are entirely new to this repository and will be described in separate emails. The revisions listed as "add" were already present in the repository and have only been added to this reference. Summary of changes: .../beam/runners/direct/DirectTimerInternals.java | 12 + .../beam/runners/direct/EvaluationContext.java | 9 +- .../beam/runners/direct/QuiescenceDriver.java | 6 +- .../direct/StatefulParDoEvaluatorFactory.java | 26 +- .../runners/direct/WatermarkCallbackExecutor.java | 23 +- .../beam/runners/direct/WatermarkManager.java | 199 -- .../beam/runners/direct/DirectRunnerTest.java | 230 +++- .../direct/StatefulParDoEvaluatorFactoryTest.java | 8 + .../beam/runners/direct/WatermarkManagerTest.java | 39 ++- runners/flink/job-server/flink_job_server.gradle | 1 + runners/samza/build.gradle | 1 + ...reMessage.java => UsesStrictTimerOrdering.java} | 6 +- .../org/apache/beam/sdk/transforms/ParDoTest.java | 291 + 13 files changed, 816 insertions(+), 35 deletions(-) copy sdks/java/core/src/main/java/org/apache/beam/sdk/testing/{UsesFailureMessage.java => UsesStrictTimerOrdering.java} (77%)
[beam] branch master updated: [BEAM-8439] Delay bundle start until pushed back data is emitted
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 49def1e [BEAM-8439] Delay bundle start until pushed back data is emitted new 7f8e4e4 Merge pull request #9841 from je-ik/BEAM-8439: [BEAM-8439] avoid creation of empty bundles 49def1e is described below commit 49def1ef43894c1ed948465e89e1f9112d889d97 Author: Maximilian Michels AuthorDate: Mon Oct 21 17:05:53 2019 +0200 [BEAM-8439] Delay bundle start until pushed back data is emitted --- .../runners/flink/translation/wrappers/streaming/DoFnOperator.java | 6 ++ .../flink/translation/wrappers/streaming/DoFnOperatorTest.java | 5 ++--- .../wrappers/streaming/ExecutableStageDoFnOperatorTest.java | 4 3 files changed, 4 insertions(+), 11 deletions(-) diff --git a/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java b/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java index 4f48287..b4ce64f 100644 --- a/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java +++ b/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java @@ -630,7 +630,6 @@ public class DoFnOperator extends AbstractStreamOperator= BoundedWindow.TIMESTAMP_MAX_VALUE.getMillis()) { @@ -677,7 +676,6 @@ public class DoFnOperator extends AbstractStreamOperator= BoundedWindow.TIMESTAMP_MAX_VALUE.getMillis()) { @@ -698,6 +696,7 @@ public class DoFnOperator extends AbstractStreamOperator> it = pushedBackElementsHandler.getElements().iterator(); while (it.hasNext()) { + checkInvokeStartBundle(); WindowedValue element = it.next(); // we need to set the correct key in case the operator is // a (keyed) window operator @@ -790,8 +789,7 @@ public class DoFnOperator extends AbstractStreamOperator timer) throws Exception { -// We don't have to cal checkInvokeStartBundle() because it's already called in -// processWatermark*(). +checkInvokeStartBundle(); fireTimer(timer); } diff --git a/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperatorTest.java b/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperatorTest.java index 57f7694..2a83944 100644 --- a/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperatorTest.java +++ b/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperatorTest.java @@ -1274,7 +1274,8 @@ public class DoFnOperatorTest { WindowedValue.valueInGlobalWindow("d"), WindowedValue.valueInGlobalWindow("finishBundle"))); -// A final bundle will be created when sending the MAX watermark +// No bundle will be created when sending the MAX watermark +// (unless pushed back items are emitted) newHarness.close(); assertThat( @@ -1282,7 +1283,6 @@ public class DoFnOperatorTest { contains( WindowedValue.valueInGlobalWindow("finishBundle"), WindowedValue.valueInGlobalWindow("d"), -WindowedValue.valueInGlobalWindow("finishBundle"), WindowedValue.valueInGlobalWindow("finishBundle"))); // close() will also call dispose(), but call again to verify no new bundle @@ -1294,7 +1294,6 @@ public class DoFnOperatorTest { contains( WindowedValue.valueInGlobalWindow("finishBundle"), WindowedValue.valueInGlobalWindow("d"), -WindowedValue.valueInGlobalWindow("finishBundle"), WindowedValue.valueInGlobalWindow("finishBundle"))); } diff --git a/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ExecutableStageDoFnOperatorTest.java b/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ExecutableStageDoFnOperatorTest.java index 8134b24..0d7c99f 100644 --- a/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ExecutableStageDoFnOperatorTest.java +++ b/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ExecutableStageDoFnOperatorTest.java @@ -403,10 +403,6 @@ public class ExecutableStageDoFnOperatorTest { verify(stageBundleFactory).getProcessBundleDescriptor(); verify(stageBundleFactory).close(); verify(stageContext).close(); -// DoFnOperator generates a final watermark, which triggers a new bundle.. -verify(stageBundleFactory).getBundle(any(), any(), any()); -
[beam] branch master updated: [BEAM-8439] Delay bundle start until pushed back data is emitted
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new 49def1e [BEAM-8439] Delay bundle start until pushed back data is emitted new 7f8e4e4 Merge pull request #9841 from je-ik/BEAM-8439: [BEAM-8439] avoid creation of empty bundles 49def1e is described below commit 49def1ef43894c1ed948465e89e1f9112d889d97 Author: Maximilian Michels AuthorDate: Mon Oct 21 17:05:53 2019 +0200 [BEAM-8439] Delay bundle start until pushed back data is emitted --- .../runners/flink/translation/wrappers/streaming/DoFnOperator.java | 6 ++ .../flink/translation/wrappers/streaming/DoFnOperatorTest.java | 5 ++--- .../wrappers/streaming/ExecutableStageDoFnOperatorTest.java | 4 3 files changed, 4 insertions(+), 11 deletions(-) diff --git a/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java b/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java index 4f48287..b4ce64f 100644 --- a/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java +++ b/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java @@ -630,7 +630,6 @@ public class DoFnOperator extends AbstractStreamOperator= BoundedWindow.TIMESTAMP_MAX_VALUE.getMillis()) { @@ -677,7 +676,6 @@ public class DoFnOperator extends AbstractStreamOperator= BoundedWindow.TIMESTAMP_MAX_VALUE.getMillis()) { @@ -698,6 +696,7 @@ public class DoFnOperator extends AbstractStreamOperator> it = pushedBackElementsHandler.getElements().iterator(); while (it.hasNext()) { + checkInvokeStartBundle(); WindowedValue element = it.next(); // we need to set the correct key in case the operator is // a (keyed) window operator @@ -790,8 +789,7 @@ public class DoFnOperator extends AbstractStreamOperator timer) throws Exception { -// We don't have to cal checkInvokeStartBundle() because it's already called in -// processWatermark*(). +checkInvokeStartBundle(); fireTimer(timer); } diff --git a/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperatorTest.java b/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperatorTest.java index 57f7694..2a83944 100644 --- a/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperatorTest.java +++ b/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperatorTest.java @@ -1274,7 +1274,8 @@ public class DoFnOperatorTest { WindowedValue.valueInGlobalWindow("d"), WindowedValue.valueInGlobalWindow("finishBundle"))); -// A final bundle will be created when sending the MAX watermark +// No bundle will be created when sending the MAX watermark +// (unless pushed back items are emitted) newHarness.close(); assertThat( @@ -1282,7 +1283,6 @@ public class DoFnOperatorTest { contains( WindowedValue.valueInGlobalWindow("finishBundle"), WindowedValue.valueInGlobalWindow("d"), -WindowedValue.valueInGlobalWindow("finishBundle"), WindowedValue.valueInGlobalWindow("finishBundle"))); // close() will also call dispose(), but call again to verify no new bundle @@ -1294,7 +1294,6 @@ public class DoFnOperatorTest { contains( WindowedValue.valueInGlobalWindow("finishBundle"), WindowedValue.valueInGlobalWindow("d"), -WindowedValue.valueInGlobalWindow("finishBundle"), WindowedValue.valueInGlobalWindow("finishBundle"))); } diff --git a/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ExecutableStageDoFnOperatorTest.java b/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ExecutableStageDoFnOperatorTest.java index 8134b24..0d7c99f 100644 --- a/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ExecutableStageDoFnOperatorTest.java +++ b/runners/flink/src/test/java/org/apache/beam/runners/flink/translation/wrappers/streaming/ExecutableStageDoFnOperatorTest.java @@ -403,10 +403,6 @@ public class ExecutableStageDoFnOperatorTest { verify(stageBundleFactory).getProcessBundleDescriptor(); verify(stageBundleFactory).close(); verify(stageContext).close(); -// DoFnOperator generates a final watermark, which triggers a new bundle.. -verify(stageBundleFactory).getBundle(any(), any(), any()); -
[beam] branch master updated: [BEAM-8092] changed guava's Optional to java.util
This is an automated email from the ASF dual-hosted git repository. janl pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/master by this push: new bfdf6b8 [BEAM-8092] changed guava's Optional to java.util new ff8653d Merge pull request #9431 from je-ik/BEAM-8092: [BEAM-8092] change guava's Optional to java.util in DirectRunner bfdf6b8 is described below commit bfdf6b85df2fbc67f46334842b5f7af3ac56d54d Author: Jan Lukavsky AuthorDate: Thu Aug 15 15:25:54 2019 +0200 [BEAM-8092] changed guava's Optional to java.util --- .../java/org/apache/beam/runners/direct/CommittedResult.java | 2 +- .../runners/direct/CopyOnAccessInMemoryStateInternals.java | 8 .../java/org/apache/beam/runners/direct/EvaluationContext.java | 6 +++--- .../beam/runners/direct/ExecutorServiceParallelExecutor.java | 4 ++-- .../java/org/apache/beam/runners/direct/QuiescenceDriver.java | 6 +++--- .../org/apache/beam/runners/direct/SideInputContainer.java | 4 ++-- .../org/apache/beam/runners/direct/CommittedResultTest.java| 10 +- .../beam/runners/direct/DirectTransformExecutorTest.java | 4 ++-- 8 files changed, 22 insertions(+), 22 deletions(-) diff --git a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/CommittedResult.java b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/CommittedResult.java index 6a14cac..16ff95b 100644 --- a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/CommittedResult.java +++ b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/CommittedResult.java @@ -18,10 +18,10 @@ package org.apache.beam.runners.direct; import com.google.auto.value.AutoValue; +import java.util.Optional; import java.util.Set; import org.apache.beam.sdk.runners.AppliedPTransform; import org.apache.beam.sdk.transforms.View.CreatePCollectionView; -import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.base.Optional; /** A {@link TransformResult} that has been committed. */ @AutoValue diff --git a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/CopyOnAccessInMemoryStateInternals.java b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/CopyOnAccessInMemoryStateInternals.java index 0a64a4b..1153c1f 100644 --- a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/CopyOnAccessInMemoryStateInternals.java +++ b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/CopyOnAccessInMemoryStateInternals.java @@ -22,6 +22,7 @@ import static org.apache.beam.vendor.guava.v26_0_jre.com.google.common.base.Prec import java.util.Collection; import java.util.HashSet; import java.util.Map; +import java.util.Optional; import javax.annotation.Nullable; import org.apache.beam.runners.core.InMemoryStateInternals.InMemoryBag; import org.apache.beam.runners.core.InMemoryStateInternals.InMemoryCombiningState; @@ -51,7 +52,6 @@ import org.apache.beam.sdk.transforms.CombineWithContext.CombineFnWithContext; import org.apache.beam.sdk.transforms.windowing.BoundedWindow; import org.apache.beam.sdk.transforms.windowing.TimestampCombiner; import org.apache.beam.sdk.util.CombineFnUtil; -import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.base.Optional; import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.Iterables; import org.joda.time.Instant; @@ -162,9 +162,9 @@ class CopyOnAccessInMemoryStateInternals implements StateInternals { private Optional earliestWatermarkHold; public CopyOnAccessInMemoryStateTable(StateTable underlying) { - this.underlying = Optional.fromNullable(underlying); + this.underlying = Optional.ofNullable(underlying); binderFactory = new CopyOnBindBinderFactory(this.underlying); - earliestWatermarkHold = Optional.absent(); + earliestWatermarkHold = Optional.empty(); } /** @@ -193,7 +193,7 @@ class CopyOnAccessInMemoryStateInternals implements StateInternals { earliestWatermarkHold = Optional.of(earliestHold); clearEmpty(); binderFactory = new InMemoryStateBinderFactory(); - underlying = Optional.absent(); + underlying = Optional.empty(); } /** diff --git a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/EvaluationContext.java b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/EvaluationContext.java index 5fc2750..c5ebfaf 100644 --- a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/EvaluationContext.java +++ b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/EvaluationContext.java @@ -23,6 +23,7 @@ import java.util.Collection; import java.util.EnumSet; import java.util.List; import java.util.Map; +import java.util.Optional; import java.util.Set; import java.util.concurrent.ConcurrentHashMap; import java.util.concurren