[beam] branch master updated (6261a007a9e -> 612bfc4b486)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 6261a007a9e Log user code exceptions in Java (#28514) add 612bfc4b486 Log user code exceptions in Python (#28515) No new revisions were added by this update. Summary of changes: sdks/python/apache_beam/runners/common.py | 4 + .../apache_beam/runners/worker/log_handler_test.py | 151 + 2 files changed, 155 insertions(+)
[beam] branch master updated (272da41a1cc -> 6261a007a9e)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 272da41a1cc Updating access modifiers for utility methods (#28511) add 6261a007a9e Log user code exceptions in Java (#28514) No new revisions were added by this update. Summary of changes: .../java/org/apache/beam/fn/harness/FnHarness.java | 1 - .../fn/harness/control/BeamFnControlClient.java| 4 +- .../fn/harness/control/ExecutionStateSampler.java | 23 .../harness/data/PCollectionConsumerRegistry.java | 47 +--- .../fn/harness/logging/BeamFnLoggingClient.java| 23 ++-- .../beam/fn/harness/logging/BeamFnLoggingMDC.java | 20 +++ .../harness/control/ExecutionStateSamplerTest.java | 25 .../data/PCollectionConsumerRegistryTest.java | 134 + .../harness/logging/BeamFnLoggingClientTest.java | 20 +++ 9 files changed, 263 insertions(+), 34 deletions(-)
[beam] branch master updated: Add flag to control automatic exception sampling in Java (disabled) (#28403)
This is an automated email from the ASF dual-hosted git repository. udim 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 5df59a943d1 Add flag to control automatic exception sampling in Java (disabled) (#28403) 5df59a943d1 is described below commit 5df59a943d18723942cd0531c12bef256ef90e20 Author: Sam Rohde AuthorDate: Thu Sep 14 14:21:24 2023 -0700 Add flag to control automatic exception sampling in Java (disabled) (#28403) * Add flag to control automatic exception sampling in Java (disabled) * Fix NPE in FnHarness with DataSampler * trigger tests * trigger tests * trigger tests * trigger tests - Co-authored-by: Sam Rohde --- .../java/org/apache/beam/fn/harness/FnHarness.java | 17 ++-- .../apache/beam/fn/harness/debug/DataSampler.java | 63 +++- .../beam/fn/harness/debug/OutputSampler.java | 9 +- .../beam/fn/harness/debug/DataSamplerTest.java | 106 - .../beam/fn/harness/debug/OutputSamplerTest.java | 47 ++--- 5 files changed, 216 insertions(+), 26 deletions(-) diff --git a/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/FnHarness.java b/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/FnHarness.java index 3f018c376f0..448c8d42df7 100644 --- a/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/FnHarness.java +++ b/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/FnHarness.java @@ -97,7 +97,6 @@ public class FnHarness { private static final String PIPELINE_OPTIONS_FILE = "PIPELINE_OPTIONS_FILE"; private static final String PIPELINE_OPTIONS = "PIPELINE_OPTIONS"; private static final String RUNNER_CAPABILITIES = "RUNNER_CAPABILITIES"; - private static final String ENABLE_DATA_SAMPLING_EXPERIMENT = "enable_data_sampling"; private static final Logger LOG = LoggerFactory.getLogger(FnHarness.class); private static Endpoints.ApiServiceDescriptor getApiServiceDescriptor(String descriptor) @@ -248,7 +247,8 @@ public class FnHarness { options.as(ExecutorOptions.class).getScheduledExecutorService(); ExecutionStateSampler executionStateSampler = new ExecutionStateSampler(options, System::currentTimeMillis); -final DataSampler dataSampler = new DataSampler(); + +final @Nullable DataSampler dataSampler = DataSampler.create(options); // The logging client variable is not used per se, but during its lifetime (until close()) it // intercepts logging and sends it to the logging service. @@ -276,10 +276,6 @@ public class FnHarness { FinalizeBundleHandler finalizeBundleHandler = new FinalizeBundleHandler(executorService); - // Create the sampler, if the experiment is enabled. - boolean shouldSample = - ExperimentalOptions.hasExperiment(options, ENABLE_DATA_SAMPLING_EXPERIMENT); - // Retrieves the ProcessBundleDescriptor from cache. Requests the PBD from the Runner if it // doesn't exist. Additionally, runs any graph modifications. Function getProcessBundleDescriptor = @@ -314,7 +310,7 @@ public class FnHarness { metricsShortIds, executionStateSampler, processWideCache, - shouldSample ? dataSampler : null); + dataSampler); logging.setProcessBundleHandler(processBundleHandler); BeamFnStatusClient beamFnStatusClient = null; @@ -363,7 +359,12 @@ public class FnHarness { InstructionRequest.RequestCase.HARNESS_MONITORING_INFOS, processWideHandler::harnessMonitoringInfos); handlers.put( - InstructionRequest.RequestCase.SAMPLE_DATA, dataSampler::handleDataSampleRequest); + InstructionRequest.RequestCase.SAMPLE_DATA, + request -> + dataSampler == null + ? BeamFnApi.InstructionResponse.newBuilder() + .setSampleData(BeamFnApi.SampleDataResponse.newBuilder()) + : dataSampler.handleDataSampleRequest(request)); JvmInitializers.runBeforeProcessing(options); diff --git a/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/debug/DataSampler.java b/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/debug/DataSampler.java index b03c453475b..29011b82a4d 100644 --- a/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/debug/DataSampler.java +++ b/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/debug/DataSampler.java @@ -23,9 +23,12 @@ import java.io.IOException; import java.util.List; import java.util.Map; import java.util.concurrent.ConcurrentHashMap; +import javax.annotation.Nullable; import org.apache.beam.model.fnexecution.v1.BeamFnApi; import org.apache.beam.model.fnexecution.v1.BeamFn
[beam] branch master updated: Add flag to control automatic exception sampling in Python (disabled) (#28418)
This is an automated email from the ASF dual-hosted git repository. udim 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 1b42dedd82d Add flag to control automatic exception sampling in Python (disabled) (#28418) 1b42dedd82d is described below commit 1b42dedd82d921912f8b0a319260658a09c1b60c Author: Sam Rohde AuthorDate: Wed Sep 13 15:29:10 2023 -0700 Add flag to control automatic exception sampling in Python (disabled) (#28418) * Always-on exception sampling to Python (disable) * Fix only_sample_exceptions logic * trigger tests * fix flaky test - Co-authored-by: Sam Rohde --- .../apache_beam/runners/worker/data_sampler.py | 29 +- .../runners/worker/data_sampler_test.py| 102 - .../apache_beam/runners/worker/operations.py | 6 +- .../apache_beam/runners/worker/sdk_worker_main.py | 4 +- 4 files changed, 134 insertions(+), 7 deletions(-) diff --git a/sdks/python/apache_beam/runners/worker/data_sampler.py b/sdks/python/apache_beam/runners/worker/data_sampler.py index 303648738f3..5ca307ca1b3 100644 --- a/sdks/python/apache_beam/runners/worker/data_sampler.py +++ b/sdks/python/apache_beam/runners/worker/data_sampler.py @@ -40,6 +40,8 @@ from typing import Union from apache_beam.coders.coder_impl import CoderImpl from apache_beam.coders.coder_impl import WindowedValueCoderImpl from apache_beam.coders.coders import Coder +from apache_beam.options.pipeline_options import DebugOptions +from apache_beam.options.pipeline_options import PipelineOptions from apache_beam.portability.api import beam_fn_api_pb2 from apache_beam.utils.windowed_value import WindowedValue @@ -216,6 +218,7 @@ class DataSampler: self, max_samples: int = 10, sample_every_sec: float = 30, + sample_only_exceptions: bool = False, clock=None) -> None: # Key is PCollection id. Is guarded by the _samplers_lock. self._samplers: Dict[str, OutputSampler] = {} @@ -223,10 +226,34 @@ class DataSampler: # runner queries for samples. self._samplers_lock: threading.Lock = threading.Lock() self._max_samples = max_samples -self._sample_every_sec = sample_every_sec +self._sample_every_sec = 0.0 if sample_only_exceptions else sample_every_sec self._samplers_by_output: Dict[str, List[OutputSampler]] = {} self._clock = clock + _ENABLE_DATA_SAMPLING = 'enable_data_sampling' + _ENABLE_ALWAYS_ON_EXCEPTION_SAMPLING = 'enable_always_on_exception_sampling' + _DISABLE_ALWAYS_ON_EXCEPTION_SAMPLING = 'disable_always_on_exception_sampling' + + @staticmethod + def create(sdk_pipeline_options: PipelineOptions, **kwargs): +experiments = sdk_pipeline_options.view_as(DebugOptions).experiments or [] + +# When true, enables only the sampling of exceptions. +always_on_exception_sampling = ( +DataSampler._ENABLE_ALWAYS_ON_EXCEPTION_SAMPLING in experiments and +DataSampler._DISABLE_ALWAYS_ON_EXCEPTION_SAMPLING not in experiments) + +# When true, enables the sampling of all PCollections and exceptions. +enable_data_sampling = DataSampler._ENABLE_DATA_SAMPLING in experiments + +if enable_data_sampling or always_on_exception_sampling: + sample_only_exceptions = ( + always_on_exception_sampling and not enable_data_sampling) + return DataSampler( + sample_only_exceptions=sample_only_exceptions, **kwargs) +else: + return None + def stop(self) -> None: """Stops all sampling, does not clear samplers in case there are outstanding samples. diff --git a/sdks/python/apache_beam/runners/worker/data_sampler_test.py b/sdks/python/apache_beam/runners/worker/data_sampler_test.py index 8d063fdb49d..8c47315b7a9 100644 --- a/sdks/python/apache_beam/runners/worker/data_sampler_test.py +++ b/sdks/python/apache_beam/runners/worker/data_sampler_test.py @@ -27,6 +27,7 @@ from typing import Optional from apache_beam.coders import FastPrimitivesCoder from apache_beam.coders import WindowedValueCoder +from apache_beam.options.pipeline_options import PipelineOptions from apache_beam.portability.api import beam_fn_api_pb2 from apache_beam.runners.worker.data_sampler import DataSampler from apache_beam.runners.worker.data_sampler import OutputSampler @@ -56,7 +57,9 @@ class DataSamplerTest(unittest.TestCase): return descriptor def setUp(self): -self.data_sampler = DataSampler(sample_every_sec=0.1) +self.data_sampler = DataSampler.create( +PipelineOptions(experiments=[DataSampler._ENABLE_DATA_SAMPLING]), +sample_every_sec=0.1) def tearDown(self): self.data_sampler.stop() @@ -341,6 +344,103 @@ class DataSamplerTest(unittest.TestCase): samples = self.data_sampler.wait_for_samples([MAIN_PCOLLECTION_ID]) s
[beam] branch master updated (d15ea43275e -> 3304e1258b3)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from d15ea43275e Revert "Add to blog" add 3304e1258b3 Update Python data sampling algorithm (#28273) No new revisions were added by this update. Summary of changes: sdks/python/apache_beam/runners/worker/data_sampler.py | 10 +- 1 file changed, 9 insertions(+), 1 deletion(-)
[beam] branch master updated (f0c0685dbe0 -> 1eda6dbfcf8)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from f0c0685dbe0 Update ttypescript version. (#28133) add c8f3b51cec1 Fix ConcurrentModificationException in OutputSampler new 1eda6dbfcf8 Merge pull request #28150: Fix ConcurrentModificationException in OutputSampler 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: .../java/org/apache/beam/fn/harness/debug/OutputSampler.java | 12 .../org/apache/beam/fn/harness/debug/OutputSamplerTest.java | 8 ++-- 2 files changed, 14 insertions(+), 6 deletions(-)
[beam] 01/01: Merge pull request #28150: Fix ConcurrentModificationException in OutputSampler
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 1eda6dbfcf87734b5f9c3ff5f7c76604f97e6548 Merge: f0c0685dbe0 c8f3b51cec1 Author: Udi Meiri AuthorDate: Mon Aug 28 18:36:51 2023 -0700 Merge pull request #28150: Fix ConcurrentModificationException in OutputSampler .../java/org/apache/beam/fn/harness/debug/OutputSampler.java | 12 .../org/apache/beam/fn/harness/debug/OutputSamplerTest.java | 8 ++-- 2 files changed, 14 insertions(+), 6 deletions(-)
[beam] branch master updated: Implement Exception Sampling in the Python SDK (#27280)
This is an automated email from the ASF dual-hosted git repository. udim 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 23fe94b3af2 Implement Exception Sampling in the Python SDK (#27280) 23fe94b3af2 is described below commit 23fe94b3af28d1a7d55f994dbddc9669a6567cb1 Author: Sam Rohde AuthorDate: Wed Jun 28 16:48:13 2023 -0700 Implement Exception Sampling in the Python SDK (#27280) * Python exception sampling implementation * add to cython def * add more cython defs * address comments * fix circular imports * linter * fix tests * remove print * Add traceback to exception * fix tests - Co-authored-by: Sam Rohde --- sdks/python/apache_beam/runners/common.pxd | 2 + sdks/python/apache_beam/runners/common.py | 35 ++- .../apache_beam/runners/worker/bundle_processor.py | 20 +- .../runners/worker/bundle_processor_test.py| 142 +++-- .../apache_beam/runners/worker/data_sampler.py | 139 +++-- .../runners/worker/data_sampler_test.py| 330 + .../apache_beam/runners/worker/operations.pxd | 4 + .../apache_beam/runners/worker/operations.py | 48 +-- .../apache_beam/runners/worker/sdk_worker.py | 10 +- .../apache_beam/runners/worker/sdk_worker_test.py | 17 +- 10 files changed, 535 insertions(+), 212 deletions(-) diff --git a/sdks/python/apache_beam/runners/common.pxd b/sdks/python/apache_beam/runners/common.pxd index d1745970d26..9fb44af6377 100644 --- a/sdks/python/apache_beam/runners/common.pxd +++ b/sdks/python/apache_beam/runners/common.pxd @@ -121,6 +121,8 @@ cdef class DoFnRunner: cdef list side_inputs cdef DoFnInvoker do_fn_invoker cdef public object bundle_finalizer_param + cdef str transform_id + cdef object execution_context cpdef process(self, WindowedValue windowed_value) diff --git a/sdks/python/apache_beam/runners/common.py b/sdks/python/apache_beam/runners/common.py index 75eb85f2110..e91199787e5 100644 --- a/sdks/python/apache_beam/runners/common.py +++ b/sdks/python/apache_beam/runners/common.py @@ -66,6 +66,7 @@ from apache_beam.utils.windowed_value import WindowedBatch from apache_beam.utils.windowed_value import WindowedValue if TYPE_CHECKING: + from apache_beam.runners.worker.bundle_processor import ExecutionContext from apache_beam.transforms import sideinputs from apache_beam.transforms.core import TimerSpec from apache_beam.io.iobase import RestrictionProgress @@ -1338,7 +1339,8 @@ class DoFnRunner: state=None, scoped_metrics_container=None, operation_name=None, - user_state_context=None # type: Optional[userstate.UserStateContext] + transform_id=None, + user_state_context=None, # type: Optional[userstate.UserStateContext] ): """Initializes a DoFnRunner. @@ -1354,6 +1356,7 @@ class DoFnRunner: state: handle for accessing DoFn state scoped_metrics_container: DEPRECATED operation_name: The system name assigned by the runner for this operation. + transform_id: The PTransform Id in the pipeline proto for this DoFn. user_state_context: The UserStateContext instance for the current Stateful DoFn. """ @@ -1361,8 +1364,10 @@ class DoFnRunner: side_inputs = list(side_inputs) self.step_name = step_name +self.transform_id = transform_id self.context = DoFnContext(step_name, state=state) self.bundle_finalizer_param = DoFn.BundleFinalizerParam() +self.execution_context = None # type: Optional[ExecutionContext] do_fn_signature = DoFnSignature(fn) @@ -1417,9 +1422,25 @@ class DoFnRunner: try: return self.do_fn_invoker.invoke_process(windowed_value) except BaseException as exn: - self._reraise_augmented(exn) + self._reraise_augmented(exn, windowed_value) return [] + def _maybe_sample_exception( + self, exn: BaseException, windowed_value: WindowedValue) -> None: + +if self.execution_context is None: + return + +output_sampler = self.execution_context.output_sampler +if output_sampler is None: + return + +output_sampler.sample_exception( +windowed_value, +exn, +self.transform_id, +self.execution_context.instruction_id) + def process_batch(self, windowed_batch): # type: (WindowedBatch) -> None try: @@ -1487,7 +1508,7 @@ class DoFnRunner: # type: () -> None self.bundle_finalizer_param.finalize_bundle() - def _reraise_augmented(self, exn): + def _reraise_augmented(self, exn, windowed_value=None): if getattr(exn, '_tagged_with_step', False)
[beam] 01/01: Merge pull request #26705: Add Exception metadata to the data sampling protocol
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit d6bde217607c9008401dd2a5d87cc5b7c9e5913c Merge: 38c7644be58 5170103e86f Author: Udi Meiri AuthorDate: Fri Jun 23 13:51:59 2023 -0700 Merge pull request #26705: Add Exception metadata to the data sampling protocol .../beam/model/fn_execution/v1/beam_fn_api.proto | 49 +- 1 file changed, 20 insertions(+), 29 deletions(-)
[beam] branch master updated (38c7644be58 -> d6bde217607)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 38c7644be58 Merge pull request #25430: Using capabilities instead of the container name to set use_single_core_per_container add 5170103e86f Add Exception metadata to the data sampling protocol new d6bde217607 Merge pull request #26705: Add Exception metadata to the data sampling protocol 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/model/fn_execution/v1/beam_fn_api.proto | 49 +- 1 file changed, 20 insertions(+), 29 deletions(-)
[beam] 01/01: Merge pull request #27132: QuotaEvents for BigQuery
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit fc74d829653a17503ba7e98d9c2e722dfb649802 Merge: 0822ab49697 ba17f369e93 Author: Udi Meiri AuthorDate: Wed Jun 14 18:13:49 2023 -0700 Merge pull request #27132: QuotaEvents for BigQuery sdks/java/io/google-cloud-platform/build.gradle| 1 + .../beam/sdk/io/gcp/bigquery/BigQueryHelpers.java | 19 .../sdk/io/gcp/bigquery/BigQueryServicesImpl.java | 50 -- 3 files changed, 58 insertions(+), 12 deletions(-)
[beam] branch master updated (0822ab49697 -> fc74d829653)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git from 0822ab49697 Initial implementation of QuotaEvent (#27078) add ba17f369e93 QuotaEvents for BigQuery new fc74d829653 Merge pull request #27132: QuotaEvents for BigQuery 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: sdks/java/io/google-cloud-platform/build.gradle| 1 + .../beam/sdk/io/gcp/bigquery/BigQueryHelpers.java | 19 .../sdk/io/gcp/bigquery/BigQueryServicesImpl.java | 50 -- 3 files changed, 58 insertions(+), 12 deletions(-)
[beam] branch master updated: Initial implementation of QuotaEvent (#27078)
This is an automated email from the ASF dual-hosted git repository. udim 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 0822ab49697 Initial implementation of QuotaEvent (#27078) 0822ab49697 is described below commit 0822ab49697d3cb0f3f0b069e64c9571b848b7e2 Author: Udi Meiri AuthorDate: Wed Jun 14 11:43:24 2023 -0700 Initial implementation of QuotaEvent (#27078) * Initial implementation of QuotaEvent * Fix style * Fix typo * Add benchmark * Added clarification about close() --- .../jmh/logging/BeamFnLoggingClientBenchmark.java | 18 .../fn/harness/logging/BeamFnLoggingClient.java| 4 +- .../apache/beam/fn/harness/logging/QuotaEvent.java | 108 + 3 files changed, 129 insertions(+), 1 deletion(-) diff --git a/sdks/java/harness/jmh/src/main/java/org/apache/beam/fn/harness/jmh/logging/BeamFnLoggingClientBenchmark.java b/sdks/java/harness/jmh/src/main/java/org/apache/beam/fn/harness/jmh/logging/BeamFnLoggingClientBenchmark.java index 3732bdfb189..d7c18059dc2 100644 --- a/sdks/java/harness/jmh/src/main/java/org/apache/beam/fn/harness/jmh/logging/BeamFnLoggingClientBenchmark.java +++ b/sdks/java/harness/jmh/src/main/java/org/apache/beam/fn/harness/jmh/logging/BeamFnLoggingClientBenchmark.java @@ -24,6 +24,7 @@ import java.util.concurrent.TimeUnit; import java.util.concurrent.atomic.AtomicInteger; import org.apache.beam.fn.harness.logging.BeamFnLoggingClient; import org.apache.beam.fn.harness.logging.BeamFnLoggingMDC; +import org.apache.beam.fn.harness.logging.QuotaEvent; import org.apache.beam.model.fnexecution.v1.BeamFnApi; import org.apache.beam.model.fnexecution.v1.BeamFnLoggingGrpc; import org.apache.beam.model.pipeline.v1.Endpoints.ApiServiceDescriptor; @@ -206,6 +207,23 @@ public class BeamFnLoggingClientBenchmark { } } + @Benchmark + @Threads(16) // Use several threads since we expect contention during logging + public void testLoggingWithQuotaEvent( + ManyExpectedCallsLoggingClientAndService client, ManageExecutionState executionState) + throws Exception { +try (Closeable state = + executionState.executionStateTracker.enterState(executionState.simpleExecutionState)) { + try (AutoCloseable ac = + new QuotaEvent.Builder() + .withOperation("test") + .withFullResourceName("//test.googleapis.com/abc/123") + .create()) { +LOG.warn("log me"); + } +} + } + @Benchmark @Threads(16) // Use several threads since we expect contention during logging public void testSkippedLogging(ZeroExpectedCallsLoggingClientAndService client) { diff --git a/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/logging/BeamFnLoggingClient.java b/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/logging/BeamFnLoggingClient.java index f2ec67c2ac5..68ad6727c4d 100644 --- a/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/logging/BeamFnLoggingClient.java +++ b/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/logging/BeamFnLoggingClient.java @@ -126,7 +126,9 @@ public class BeamFnLoggingClient implements AutoCloseable { logRecordHandler.setLevel(Level.ALL); logRecordHandler.setFormatter(DEFAULT_FORMATTER); logRecordHandler.executeOn(options.as(ExecutorOptions.class).getScheduledExecutorService()); - logRecordHandler.setLogMdc(options.as(SdkHarnessOptions.class).getLogMdc()); +boolean logMdc = options.as(SdkHarnessOptions.class).getLogMdc(); +logRecordHandler.setLogMdc(logMdc); +QuotaEvent.setEnabled(logMdc); outboundObserver = (CallStreamObserver) stub.logging(inboundObserver); rootLogger.addHandler(logRecordHandler); } diff --git a/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/logging/QuotaEvent.java b/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/logging/QuotaEvent.java new file mode 100644 index 000..2ebb8b9ee33 --- /dev/null +++ b/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/logging/QuotaEvent.java @@ -0,0 +1,108 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or impl
[beam] branch master updated (d84541d -> 6c04c3c)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from d84541d [BEAM-13946] Add get_dummies(), a non-deferred column operation on categorical columns (#16615) add 45170d0 Fix final allowskew error to properly handle a large allowedSkew add 6c04c3c Merge pull request #16885: Fix final allowskew error to properly handle a large allowedSkew No new revisions were added by this update. Summary of changes: .../src/main/java/org/apache/beam/fn/harness/FnApiDoFnRunner.java | 4 +++- 1 file changed, 3 insertions(+), 1 deletion(-)
[beam] branch master updated (0e057fd -> 4f33325)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 0e057fd Merge pull request #16798: [BEAM-13159] Update embedded-redis dependency add 0dd125a Fixing the log line to properly handle a large allowed skew. add 4f33325 Merge pull request #16317: Fix allowedSkew log line to properly print large skews No new revisions were added by this update. Summary of changes: .../apache/beam/runners/core/SimpleDoFnRunner.java | 17 ++--- .../org/apache/beam/sdk/transforms/ParDoTest.java | 41 -- .../apache/beam/fn/harness/FnApiDoFnRunner.java| 13 +-- 3 files changed, 60 insertions(+), 11 deletions(-)
[beam] annotated tag sdks/v2.33.0 updated (ae8431f -> 0f311ea)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to annotated tag sdks/v2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git. *** WARNING: tag sdks/v2.33.0 was modified! *** from ae8431f (commit) to 0f311ea (tag) tagging c5c2561e037c8209e7c9cdd1c275ba084f84bf89 (tag) length 142 bytes by Udi Meiri on Thu Nov 4 16:29:23 2021 -0700 - Log - Tag for Go SDK -BEGIN PGP SIGNATURE- iQIzBAABCgAdFiEEiWHz7455ZohAZ4fPWHsEnDbar+YFAmGEbOAACgkQWHsEnDba r+bvqxAAoTSIjb4krd8o9b1fjSMzV0GpUrdQcWEspd5IU7Yz02AB3uXugfwVLihT 4b7zz5UnDJhqQIMUBzhjNnqDPJ+ITFmvS/VxGuiV2OzGQ/R3bx2hp15XU6+FmCkO Eksvc3M2dSV1DiOlD5uSCSe+cAIK4BLj0myZ2vfKI7yU1f93ljGqhjpTc8XyUfMB 4rjPDH7QVIZJ+UU8y39MAxn1n/82+f/+IzbhJA8XOxhURhfluP+aHYGYxinNfNdS t3j5RN4vEpzVGFjMhMZqFBQVup1lK2d/oDZTpK/0UGwZBz8JQUG7gpcMwLfT35Bb 2mWJ2kbMrsMyDyfR2piTUKKzF9YIQU3ntloQ/gPqTFp4amKByM8BzVV3wLgAp5B9 Q4NI4V/nDy88joa4aZ/v4K6bYIDHEiQbI7ZQYfwpbwpMw1AZSKaXMSXizd78aJhv sIYYur8HEVL2SKLkeiGYz1KWVKaQLNOHZOMiXAg6FkknU5vR2rW3wBjuqD387zkq VFWyTqlYzOJDxWyrC/j68Qzml+y9TlDs8fQtIa9P2+Yed3xEl4lF0d8u4dLHoiH9 vukA8/nUzqsg1fc9QeAD2W0fvy/l0lZPxsgJtTnsCSMW2nKeMvD+bzSNP1ljuUpD S7Rhs2cpr+rrbzVccTLAuph3Ar0P4oLpPPQV7YqPefvIoVTEQes= =2vVm -END PGP SIGNATURE- --- No new revisions were added by this update. Summary of changes:
[beam] branch master updated (735db24 -> beac28d)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 735db24 [BEAM-13140] Fixes bug in WriteRename class add 0d172c3 Performance improvement to PubSubLiteIO to not use a streaming committer to send one commit message. add beac28d Merge pull request #15880: Performance improvement to PubSubLiteIO to not use a streaming committer to send one commit message No new revisions were added by this update. Summary of changes: ...nitialOffsetReader.java => BlockingCommitter.java} | 8 ++-- .../internal/PerSubscriptionPartitionSdf.java | 14 -- .../gcp/pubsublite/internal/SubscribeTransform.java | 3 +-- .../gcp/pubsublite/internal/SubscriberAssembler.java | 19 ++- .../internal/PerSubscriptionPartitionSdfTest.java | 16 +++- 5 files changed, 20 insertions(+), 40 deletions(-) copy sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/internal/{InitialOffsetReader.java => BlockingCommitter.java} (84%)
[beam] branch master updated: Release guide: minor updates
This is an automated email from the ASF dual-hosted git repository. udim 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 3757d2a Release guide: minor updates new f8660d3 Merge pull request #15691 from udim/release-guide2 3757d2a is described below commit 3757d2abe011ac3ad7cb5d6dffc54a9838f59c03 Author: Udi Meiri AuthorDate: Thu Oct 7 18:50:39 2021 -0700 Release guide: minor updates Notes about clearing out old docker images and unlocking the gpg signing key. --- website/www/site/content/en/contribute/release-guide.md | 7 +++ 1 file changed, 7 insertions(+) diff --git a/website/www/site/content/en/contribute/release-guide.md b/website/www/site/content/en/contribute/release-guide.md index ee979b8..a308df5 100644 --- a/website/www/site/content/en/contribute/release-guide.md +++ b/website/www/site/content/en/contribute/release-guide.md @@ -551,6 +551,10 @@ See the source of the script for more details, or to run commands manually in ca ### Run build_release_candidate.sh to create a release candidate +Before you start, run this command to make sure you'll be using the latest docker images: + + docker system prune -a + * **Script:** [build_release_candidate.sh](https://github.com/apache/beam/blob/master/release/src/main/scripts/build_release_candidate.sh) * **Usage** @@ -1147,6 +1151,9 @@ Merge all of the website pull requests Create and push a new signed tag for the released version by copying the tag for the final release candidate, as follows: ``` +# Optional: unlock the signing key by signing an arbitrary file. +gpg --output ~/doc.sig --sign ~/.bashrc + VERSION_TAG="v${RELEASE}" git tag -s "$VERSION_TAG" "$RC_TAG" git push https://github.com/apache/beam "$VERSION_TAG"
[beam] branch master updated (38305d9 -> c8e7a4c)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 38305d9 Merge pull request #15502 from meowcakes/register_filesystems_for_artifact_retrieval_service add c8e7a4c Release 2.33 documentation update (#15543) No new revisions were added by this update. Summary of changes: CHANGES.md | 23 ++-- website/www/site/config.toml | 2 +- website/www/site/content/en/blog/beam-2.33.0.md| 150 + .../www/site/content/en/get-started/downloads.md | 8 ++ website/www/site/static/.htaccess | 2 +- 5 files changed, 172 insertions(+), 13 deletions(-) create mode 100644 website/www/site/content/en/blog/beam-2.33.0.md
[beam] annotated tag v2.33.0 updated (ae8431f -> a3d8463)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to annotated tag v2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git. *** WARNING: tag v2.33.0 was modified! *** from ae8431f (commit) to a3d8463 (tag) tagging c5c2561e037c8209e7c9cdd1c275ba084f84bf89 (tag) length 142 bytes by Udi Meiri on Thu Oct 7 17:12:35 2021 -0700 - Log - foo -BEGIN PGP SIGNATURE- iQIzBAABCgAdFiEEiWHz7455ZohAZ4fPWHsEnDbar+YFAmFfjPMACgkQWHsEnDba r+bgrA//f7pMAgyhDutFAXdnAJEIlDpvLgHWRRO93ZMFdyKKxhlFN/JhDzumnjXI REfj2C4ep8O52zPXoOh9DnFPSdrgSI168aUb7iMqJbIKIjGbbCoh8WGqwVnrBYgc yf5C9eriYN3Ak6PXOiUnGNyi7+vUH48sJStJNPghcgdETTs6NeDCGUOnLgnwcbgN /yIqICION3C+aCPYXAty6NrF7ZOL7Hld+B8mhmKHV7h3U2QB616gEninfCZ2o+Qn NBxUCW8Vk/lrfhKUBe5KszlCyrCcxJ47Kb7i1jgnpeXW5PSk7XqyPtx+Tq8X6Ngz S3vQof4DfEP7i2MTXc+B6OcCmgD2WnUZeE9vM9bqprZAbGhmmcOgGhxEwdZ9yHlG 1UH2QBcgRNj3fCgdFehVIi8Jb9yK+zbulN5NGrsL9CClj7qOYM6QGraKmBmxOIB3 K7LBEh681fPiuFmTxols/eqv0duVPBi1F0yWaaI0moDzOV+K9+NbWl/P0U/m+k0n 3ckf0k2aqAW2398ReDDlK1mfj2rSNGMC40hbYopx73lnb3Ur4kw9TwBgyldyeJ26 cxhbHTdvITWP1Xj65MRkmWl6XqKeYQ0H5o/dN8g89DB53Gq6p7NILPpWTT7vc45/ WF14U+8VOIu/2kgVqHjuBONAGdgm//fiHTx6QUfHGtJJ1hPb2Us= =z5f7 -END PGP SIGNATURE- --- No new revisions were added by this update. Summary of changes:
svn commit: r50248 - /dev/beam/2.33.0/python/
Author: udim Date: Mon Oct 4 18:56:08 2021 New Revision: 50248 Log: Staging Python artifacts for Apache Beam 2.33.0 RC2 Modified: dev/beam/2.33.0/python/apache-beam-2.33.0.zip dev/beam/2.33.0/python/apache-beam-2.33.0.zip.asc dev/beam/2.33.0/python/apache-beam-2.33.0.zip.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-macosx_10_9_x86_64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-macosx_10_9_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-macosx_10_9_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_i686.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_i686.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_i686.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_x86_64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_i686.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_i686.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_i686.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_x86_64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2014_aarch64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2014_aarch64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2014_aarch64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win32.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win32.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win32.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win_amd64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win_amd64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win_amd64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-macosx_10_9_x86_64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-macosx_10_9_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-macosx_10_9_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_i686.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_i686.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_i686.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_x86_64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_i686.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_i686.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_i686.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_x86_64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2014_aarch64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2014_aarch64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2014_aarch64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win32.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win32.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win32.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win_amd64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win_amd64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win_amd64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-macosx_10_9_x86_64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-macosx_10_9_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-macosx_10_9_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-manylinux1_i686.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-manylinux1_i686.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-manylinux1_i686.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-manylinux1_x86_64.whl dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-manylinux1_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-manylinux1_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp38
svn commit: r50247 - in /dev/beam/2.33.0: apache-beam-2.33.0-source-release.zip apache-beam-2.33.0-source-release.zip.asc apache-beam-2.33.0-source-release.zip.sha512
Author: udim Date: Mon Oct 4 18:53:34 2021 New Revision: 50247 Log: Staging Java artifacts for Apache Beam 2.33.0 RC2 Added: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip (with props) Modified: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 Added: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip == Binary file - no diff available. Propchange: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip -- svn:mime-type = application/octet-stream Modified: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc == --- dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc (original) +++ dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc Mon Oct 4 18:53:34 2021 @@ -1,16 +1,16 @@ -BEGIN PGP SIGNATURE- -iQIzBAABCgAdFiEEiWHz7455ZohAZ4fPWHsEnDbar+YFAmFXhEwACgkQWHsEnDba -r+bK2A//QFA80fAfm+Yh57D4y/E9ciCxdnjTZslkub4rx7mu3QEvMcduxISxYRws -PetUn8y7g63TFfaEGW6F11VWLBUYPomGb8RC6y8rZ784Z4oUtW0WzZ6p5H6i1rIB -zb49TP3UsSqX7Mx6l8YMIaZ83hxZ6QIngDwDx4TgZRF64AH9+/XrMV8A2CuLUMMk -5aZy5bXLEq8KYTngVDHYwZT+W6q79ZyadmCcTdHR1C2XBIXtMHB2Rjl0EAxqxmxa -9D9M5HJodkHu0z8sxTZ6aCYNAb82r+8QRSY8jDfs+ocecW8R98+vUyJBEk9AUtJg -OJEvWrtSwwdNSRcBi9KUI9IsTOj2YFyRLTOc+3XeRB2kGyI66P4CT+r29GgXhQPQ -YhVicsRGG34JgV9p3VFEIDWxxtcrgw+LucHWfojyJ3PAAUd8hH19nOlecQ5pDkm9 -XxG4buSVw83L8mOsOEqCsNyFLrzqjG2lAiBaLnvOph+rMmk5mlEjZS+Q9niee/zV -O1MTTdsb0MdDApsKHyVhNezRShr+Lv58M8kJBC1aXrwnSohfnDeQEsd9VdfASTJg -meCK5SRe+m6qmD9jwXU4qllhdyH4VCB+DkIjOSZPwdhixvx0IxBRPcWiN3K1hWU1 -PNgDQG2CXq/V06tqF1v1ardd07iOvfRUU7pnJrM6dEEq2zx6E/w= -=aCxn +iQIzBAABCgAdFiEEiWHz7455ZohAZ4fPWHsEnDbar+YFAmFbTYAACgkQWHsEnDba +r+bSgQ/+Nw/4svyPifYiuFk5FQzBce1M26eSa9KJiPr5fJ0j2xa06Fe4JjX4M5b7 +RcImnxRRsb9CfkMMEq2xpVeH98qiw23NYaOmrjKa4zxSOb2NJQjZA7I2pINUQGS8 +QkRruPQhZQhqn2VSDEd+EyhZ24+PRYIrt3K4LfOjNvje6y/0dFegTjrZFmy/32/N +gbxoLomj7kj1G/2Cc1NHNsTU16xcESz1HX5+09g9do6Oerk0IIkoADJTbAKCfyRa +sShZR+qhB3faja2UfSRgbYSDK1E9IZ5cfT+vKHi7QRKFkrAsANRRHtitYrH9XzFE +LFmOdkEHEDKuxsMeOKMLXPODI7Fi1ISzU+4lyng07pxr9nkg9Ouf7l2GNGBihlxV +LcqnypTwb2Tjylv6S1DRCoVQLstSAZON2BaEUU2krgi/awBKzfVXgstSzfLSvIjb +u+YEUXMLcFk7n9f+60zJ1PS4HtOEH9+jWGBbtVXxn94/o8lzKRTU/LwfVWV91AOy +0/WXxwgiD5Y4OB+yEl1LN4bLXfogYkIj5+9OEGQ7ZmcAJbaZzb2AwlCB/P5Nyf9F +aDZeP93VC95b4OtSTvnCp6tJsaOh2WPC2cMtL4KciQ+4nSvpzQx0EXK+pk3afKkt +rW4dYbKYQOIbrlRsqcne7Jaed3ajar9r4Dyv9cbyoTzGbNvtb5A= +=XP5e -END PGP SIGNATURE- Modified: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 == --- dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 (original) +++ dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 Mon Oct 4 18:53:34 2021 @@ -1 +1 @@ -292fa81d5e697cb62cda44d3f25083cec5682da2f7fe2755a750f1d3ccf2f05340a54353b41c346b23212a92f0af3e0346477d7a2411a28a196a2bbb794adcf3 apache-beam-2.33.0-source-release.zip +b016bec8e6714e1002d53b1b2b133b0835c3f169a1e3579a3677dcd35a9d9680fed7c2bd5a90534e96867ff359b3f5387af91efb1264ec0b067c2b526b50f22a apache-beam-2.33.0-source-release.zip
svn commit: r50221 - in /dev/beam/2.33.0: apache-beam-2.33.0-source-release.zip apache-beam-2.33.0-source-release.zip.asc apache-beam-2.33.0-source-release.zip.sha512
Author: udim Date: Fri Oct 1 21:58:35 2021 New Revision: 50221 Log: Staging Java artifacts for Apache Beam 2.33.0 RC2 Added: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip (with props) Modified: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 Added: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip == Binary file - no diff available. Propchange: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip -- svn:mime-type = application/octet-stream Modified: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc == --- dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc (original) +++ dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc Fri Oct 1 21:58:35 2021 @@ -1,16 +1,16 @@ -BEGIN PGP SIGNATURE- -iQIzBAABCgAdFiEEiWHz7455ZohAZ4fPWHsEnDbar+YFAmFDcKUACgkQWHsEnDba -r+Y0pg//fdCH6mCIzh/y/ufzAJhWN0wnuB8sZFr7PQVdfv04eKufD01g3hcU9s77 -V/f0K4kMUDaheD07UxL9OaqxhbzQsMxRXIdZ+vbx++H9v/2pyqGoi5b+akmogXvZ -JB0Ixio5QWks9ghaV9btkx+JencptT/Cj2YWsOMf4Dwa0+ppa62n7Mjg7WPRyjoc -RagxDfEsrJ5Sur8Ve8L8Q6HnnCPOX2VG4//wGsbX429pOQu0Ej037oGjIne2imx2 -chF1LOpNCAPZDJHPQbCG5vHVgBI/IDbl7IQC7yiRaY/sH34G6572ug8dosnQLlLM -nPIOZ7OQ260Bk7w4YB/BeQ5aPAbhCxuW7N7/xZSYBtuyTgC8b6Bd//WU3dvWRvuC -SDvLPf9FZMKZrRxApXW19gubjWze478u3UPRe+sVwkuUQWuviYlPbs3CwUeI3SWV -vdXrVeddWb64vnss35NT6A0ggDTqOzoK/+kNg3mBu1nSQqnY8eGF6HnjzWlYx/3G -DK1f4/t4oGOXlzncPfCa7L8phoX94fyvZ8EY8/GE5kvS/kqmzRXxsQPCAk1uggWA -O5c2zbOzD8fBFNYD/F9edcbvx2T1p9kLuAQqQe+df2o5+aA9/kTnI3V9NzJ8ZFL9 -GR9bAhqHGEV79X57OYgpm3PUfZhxvnvLCww8NRfmWwhetb7C9RE= -=4MBB +iQIzBAABCgAdFiEEiWHz7455ZohAZ4fPWHsEnDbar+YFAmFXhEwACgkQWHsEnDba +r+bK2A//QFA80fAfm+Yh57D4y/E9ciCxdnjTZslkub4rx7mu3QEvMcduxISxYRws +PetUn8y7g63TFfaEGW6F11VWLBUYPomGb8RC6y8rZ784Z4oUtW0WzZ6p5H6i1rIB +zb49TP3UsSqX7Mx6l8YMIaZ83hxZ6QIngDwDx4TgZRF64AH9+/XrMV8A2CuLUMMk +5aZy5bXLEq8KYTngVDHYwZT+W6q79ZyadmCcTdHR1C2XBIXtMHB2Rjl0EAxqxmxa +9D9M5HJodkHu0z8sxTZ6aCYNAb82r+8QRSY8jDfs+ocecW8R98+vUyJBEk9AUtJg +OJEvWrtSwwdNSRcBi9KUI9IsTOj2YFyRLTOc+3XeRB2kGyI66P4CT+r29GgXhQPQ +YhVicsRGG34JgV9p3VFEIDWxxtcrgw+LucHWfojyJ3PAAUd8hH19nOlecQ5pDkm9 +XxG4buSVw83L8mOsOEqCsNyFLrzqjG2lAiBaLnvOph+rMmk5mlEjZS+Q9niee/zV +O1MTTdsb0MdDApsKHyVhNezRShr+Lv58M8kJBC1aXrwnSohfnDeQEsd9VdfASTJg +meCK5SRe+m6qmD9jwXU4qllhdyH4VCB+DkIjOSZPwdhixvx0IxBRPcWiN3K1hWU1 +PNgDQG2CXq/V06tqF1v1ardd07iOvfRUU7pnJrM6dEEq2zx6E/w= +=aCxn -END PGP SIGNATURE- Modified: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 == --- dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 (original) +++ dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 Fri Oct 1 21:58:35 2021 @@ -1 +1 @@ -fd7a91f75b4b48b6aeac8b3edc830c21b8a8a866a045132beb4af62a640fdf706eb20cc4d82b2ab2412abd18b1a1ef75da13256d543dd8efb67ffb545e575702 apache-beam-2.33.0-source-release.zip +292fa81d5e697cb62cda44d3f25083cec5682da2f7fe2755a750f1d3ccf2f05340a54353b41c346b23212a92f0af3e0346477d7a2411a28a196a2bbb794adcf3 apache-beam-2.33.0-source-release.zip
[beam] annotated tag v2.33.0-RC2 updated (ae8431f -> c5c2561)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to annotated tag v2.33.0-RC2 in repository https://gitbox.apache.org/repos/asf/beam.git. *** WARNING: tag v2.33.0-RC2 was modified! *** from ae8431f (commit) to c5c2561 (tag) tagging ae8431f309a2c304b9d3c29f928a1a6dbba8b4f8 (commit) replaces jupyterlab-sidepanel-v1.0.0 by Udi Meiri on Fri Oct 1 13:46:57 2021 -0700 - Log - v2.33.0-RC2 --- No new revisions were added by this update. Summary of changes:
[beam] 01/01: Merge pull request #15629 from apache/revert-15608-dpcollins-cherry-pick
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch release-2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git commit d916c1f55e57a61b54135d0922ad8660735bd287 Merge: 1816cf4 bb5ecd1 Author: Udi Meiri AuthorDate: Fri Oct 1 11:33:47 2021 -0700 Merge pull request #15629 from apache/revert-15608-dpcollins-cherry-pick .../org/apache/beam/gradle/BeamModulePlugin.groovy | 2 +- .../beam/sdk/io/gcp/pubsub/PubsubMessages.java | 42 ++-- .../sdk/io/gcp/pubsublite/CloudPubsubChecks.java | 51 .../io/gcp/pubsublite/CloudPubsubTransforms.java | 104 .../pubsublite/ManagedBacklogReaderFactory.java| 33 --- .../ManagedBacklogReaderFactoryImpl.java | 68 - .../sdk/io/gcp/pubsublite/OffsetByteRange.java | 38 --- .../io/gcp/pubsublite/OffsetByteRangeCoder.java| 63 - .../io/gcp/pubsublite/OffsetByteRangeTracker.java | 66 ++--- .../io/gcp/pubsublite/PerServerPublisherCache.java | 4 - .../pubsublite/PerSubscriptionPartitionSdf.java| 57 ++--- .../beam/sdk/io/gcp/pubsublite/PublisherCache.java | 62 ++--- .../beam/sdk/io/gcp/pubsublite/Publishers.java | 72 ++ .../beam/sdk/io/gcp/pubsublite/PubsubLiteIO.java | 2 +- .../beam/sdk/io/gcp/pubsublite/PubsubLiteSink.java | 10 +- .../sdk/io/gcp/pubsublite/SubscribeTransform.java | 44 ++-- .../sdk/io/gcp/pubsublite/SubscriberOptions.java | 26 +- .../SubscriptionPartitionProcessorFactory.java | 3 +- .../SubscriptionPartitionProcessorImpl.java| 23 +- .../gcp/pubsublite/TopicBacklogReaderSettings.java | 6 +- .../sdk/io/gcp/pubsublite/TrackerWithProgress.java | 24 -- .../gcp/pubsublite/OffsetByteRangeTrackerTest.java | 34 +-- .../PerSubscriptionPartitionSdfTest.java | 70 +- .../beam/sdk/io/gcp/pubsublite/ReadWriteIT.java| 280 - .../SubscriptionPartitionProcessorImplTest.java| 49 ++-- 25 files changed, 297 insertions(+), 936 deletions(-)
[beam] branch release-2.33.0 updated (1816cf4 -> d916c1f)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch release-2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git. from 1816cf4 Merge pull request #15622 from clairemcginty/claire/cherrypick-beam-12628 add bb5ecd1 Revert "Cherrypick #15418 and #15515" new d916c1f Merge pull request #15629 from apache/revert-15608-dpcollins-cherry-pick 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/gradle/BeamModulePlugin.groovy | 2 +- .../beam/sdk/io/gcp/pubsub/PubsubMessages.java | 42 ++-- ...dUuidsTransform.java => CloudPubsubChecks.java} | 44 ++-- .../io/gcp/pubsublite/CloudPubsubTransforms.java | 104 .../pubsublite/ManagedBacklogReaderFactory.java| 33 --- .../ManagedBacklogReaderFactoryImpl.java | 68 - .../sdk/io/gcp/pubsublite/OffsetByteRange.java | 38 --- .../io/gcp/pubsublite/OffsetByteRangeCoder.java| 63 - .../io/gcp/pubsublite/OffsetByteRangeTracker.java | 66 ++--- .../io/gcp/pubsublite/PerServerPublisherCache.java | 4 - .../pubsublite/PerSubscriptionPartitionSdf.java| 57 ++--- .../beam/sdk/io/gcp/pubsublite/PublisherCache.java | 62 ++--- .../beam/sdk/io/gcp/pubsublite/Publishers.java | 72 ++ .../beam/sdk/io/gcp/pubsublite/PubsubLiteIO.java | 2 +- .../beam/sdk/io/gcp/pubsublite/PubsubLiteSink.java | 10 +- .../sdk/io/gcp/pubsublite/SubscribeTransform.java | 44 ++-- .../sdk/io/gcp/pubsublite/SubscriberOptions.java | 26 +- .../SubscriptionPartitionProcessorFactory.java | 3 +- .../SubscriptionPartitionProcessorImpl.java| 23 +- .../gcp/pubsublite/TopicBacklogReaderSettings.java | 6 +- .../sdk/io/gcp/pubsublite/TrackerWithProgress.java | 24 -- .../gcp/pubsublite/OffsetByteRangeTrackerTest.java | 34 +-- .../PerSubscriptionPartitionSdfTest.java | 70 +- .../beam/sdk/io/gcp/pubsublite/ReadWriteIT.java| 280 - .../SubscriptionPartitionProcessorImplTest.java| 49 ++-- 25 files changed, 269 insertions(+), 957 deletions(-) copy sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/{AddUuidsTransform.java => CloudPubsubChecks.java} (50%) delete mode 100644 sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/CloudPubsubTransforms.java delete mode 100644 sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/ManagedBacklogReaderFactory.java delete mode 100644 sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/ManagedBacklogReaderFactoryImpl.java delete mode 100644 sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/OffsetByteRange.java delete mode 100644 sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/OffsetByteRangeCoder.java delete mode 100644 sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/TrackerWithProgress.java delete mode 100644 sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/pubsublite/ReadWriteIT.java
[beam] branch release-2.33.0 updated: [BEAM-12628] make useReflectApi default to true
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch release-2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.33.0 by this push: new 9735609 [BEAM-12628] make useReflectApi default to true new 1816cf4 Merge pull request #15622 from clairemcginty/claire/cherrypick-beam-12628 9735609 is described below commit 9735609e62a9ddee4003f100a1e449204479e5a2 Author: Claire McGinty AuthorDate: Tue Sep 28 13:23:09 2021 -0400 [BEAM-12628] make useReflectApi default to true --- .../java/org/apache/beam/sdk/coders/AvroCoder.java | 24 +++--- .../org/apache/beam/sdk/coders/AvroCoderTest.java | 7 --- 2 files changed, 21 insertions(+), 10 deletions(-) diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/AvroCoder.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/AvroCoder.java index b236783..8d43fdd 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/AvroCoder.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/AvroCoder.java @@ -117,9 +117,19 @@ public class AvroCoder extends CustomCoder { * @param the element type */ public static AvroCoder of(TypeDescriptor type) { +return of(type, true); + } + + /** + * Returns an {@code AvroCoder} instance for the provided element type, respecting whether to use + * Avro's Reflect* or Specific* suite for encoding and decoding. + * + * @param the element type + */ + public static AvroCoder of(TypeDescriptor type, boolean useReflectApi) { @SuppressWarnings("unchecked") Class clazz = (Class) type.getRawType(); -return of(clazz); +return of(clazz, useReflectApi); } /** @@ -128,7 +138,7 @@ public class AvroCoder extends CustomCoder { * @param the element type */ public static AvroCoder of(Class clazz) { -return of(clazz, false); +return of(clazz, true); } /** @@ -140,8 +150,8 @@ public class AvroCoder extends CustomCoder { } /** - * Returns an {@code AvroCoder} instance for the given class using Avro's Reflection API for - * encoding and decoding. + * Returns an {@code AvroCoder} instance for the given class, respecting whether to use Avro's + * Reflect* or Specific* suite for encoding and decoding. * * @param the element type */ @@ -158,12 +168,12 @@ public class AvroCoder extends CustomCoder { * @param the element type */ public static AvroCoder of(Class type, Schema schema) { -return of(type, schema, false); +return of(type, schema, true); } /** - * Returns an {@code AvroCoder} instance for the given class and schema using Avro's Reflection - * API for encoding and decoding. + * Returns an {@code AvroCoder} instance for the given class and schema, respecting whether to use + * Avro's Reflect* or Specific* suite for encoding and decoding. * * @param the element type */ diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/coders/AvroCoderTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/coders/AvroCoderTest.java index d7886c3..9443aad 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/coders/AvroCoderTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/coders/AvroCoderTest.java @@ -323,7 +323,8 @@ public class AvroCoderTest { @Test public void testSpecificRecordEncoding() throws Exception { -AvroCoder coder = AvroCoder.of(TestAvro.class, AVRO_SPECIFIC_RECORD.getSchema()); +AvroCoder coder = +AvroCoder.of(TestAvro.class, AVRO_SPECIFIC_RECORD.getSchema(), false); assertTrue(SpecificRecord.class.isAssignableFrom(coder.getType())); CoderProperties.coderDecodeEncodeEqual(coder, AVRO_SPECIFIC_RECORD); @@ -415,8 +416,8 @@ public class AvroCoderTest { } @Test - public void testAvroReflectCoderIsSerializable() throws Exception { -AvroCoder coder = AvroCoder.of(Pojo.class, true); + public void testAvroSpecificCoderIsSerializable() throws Exception { +AvroCoder coder = AvroCoder.of(Pojo.class, false); // Check that the coder is serializable using the regular JSON approach. SerializableUtils.ensureSerializable(coder);
[beam] 01/01: Revert "Cherrypick #15418 and #15515"
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch revert-15608-dpcollins-cherry-pick in repository https://gitbox.apache.org/repos/asf/beam.git commit bb5ecd1d8b7dd2068fc0814c5d7c1a8dfb884cc1 Author: Udi Meiri AuthorDate: Thu Sep 30 14:51:26 2021 -0700 Revert "Cherrypick #15418 and #15515" --- .../org/apache/beam/gradle/BeamModulePlugin.groovy | 2 +- .../beam/sdk/io/gcp/pubsub/PubsubMessages.java | 42 ++-- .../sdk/io/gcp/pubsublite/CloudPubsubChecks.java | 51 .../io/gcp/pubsublite/CloudPubsubTransforms.java | 104 .../pubsublite/ManagedBacklogReaderFactory.java| 33 --- .../ManagedBacklogReaderFactoryImpl.java | 68 - .../sdk/io/gcp/pubsublite/OffsetByteRange.java | 38 --- .../io/gcp/pubsublite/OffsetByteRangeCoder.java| 63 - .../io/gcp/pubsublite/OffsetByteRangeTracker.java | 66 ++--- .../io/gcp/pubsublite/PerServerPublisherCache.java | 4 - .../pubsublite/PerSubscriptionPartitionSdf.java| 57 ++--- .../beam/sdk/io/gcp/pubsublite/PublisherCache.java | 62 ++--- .../beam/sdk/io/gcp/pubsublite/Publishers.java | 72 ++ .../beam/sdk/io/gcp/pubsublite/PubsubLiteIO.java | 2 +- .../beam/sdk/io/gcp/pubsublite/PubsubLiteSink.java | 10 +- .../sdk/io/gcp/pubsublite/SubscribeTransform.java | 44 ++-- .../sdk/io/gcp/pubsublite/SubscriberOptions.java | 26 +- .../SubscriptionPartitionProcessorFactory.java | 3 +- .../SubscriptionPartitionProcessorImpl.java| 23 +- .../gcp/pubsublite/TopicBacklogReaderSettings.java | 6 +- .../sdk/io/gcp/pubsublite/TrackerWithProgress.java | 24 -- .../gcp/pubsublite/OffsetByteRangeTrackerTest.java | 34 +-- .../PerSubscriptionPartitionSdfTest.java | 70 +- .../beam/sdk/io/gcp/pubsublite/ReadWriteIT.java| 280 - .../SubscriptionPartitionProcessorImplTest.java| 49 ++-- 25 files changed, 297 insertions(+), 936 deletions(-) diff --git a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy index 480a2d2..59ca672 100644 --- a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy +++ b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy @@ -446,7 +446,7 @@ class BeamModulePlugin implements Plugin { def errorprone_version = "2.3.4" def google_clients_version = "1.31.0" def google_cloud_bigdataoss_version = "2.2.2" -def google_cloud_pubsublite_version = "1.0.4" +def google_cloud_pubsublite_version = "0.13.2" def google_code_gson_version = "2.8.6" def google_oauth_clients_version = "1.31.0" // Try to keep grpc_version consistent with gRPC version in google_cloud_platform_libraries_bom diff --git a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsub/PubsubMessages.java b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsub/PubsubMessages.java index bf6a288..b0cc681 100644 --- a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsub/PubsubMessages.java +++ b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsub/PubsubMessages.java @@ -24,36 +24,25 @@ import org.apache.beam.sdk.transforms.SerializableFunction; import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.ImmutableMap; /** Common util functions for converting between PubsubMessage proto and {@link PubsubMessage}. */ -public final class PubsubMessages { - private PubsubMessages() {} - - public static com.google.pubsub.v1.PubsubMessage toProto(PubsubMessage input) { -Map attributes = input.getAttributeMap(); -com.google.pubsub.v1.PubsubMessage.Builder message = -com.google.pubsub.v1.PubsubMessage.newBuilder() -.setData(ByteString.copyFrom(input.getPayload())); -// TODO(BEAM-8085) this should not be null -if (attributes != null) { - message.putAllAttributes(attributes); -} -String messageId = input.getMessageId(); -if (messageId != null) { - message.setMessageId(messageId); -} -return message.build(); - } - - public static PubsubMessage fromProto(com.google.pubsub.v1.PubsubMessage input) { -return new PubsubMessage( -input.getData().toByteArray(), input.getAttributesMap(), input.getMessageId()); - } - +public class PubsubMessages { // Convert the PubsubMessage to a PubsubMessage proto, then return its serialized representation. public static class ParsePayloadAsPubsubMessageProto implements SerializableFunction { @Override public byte[] apply(PubsubMessage input) { - return toProto(input).toByteArray(); + Map attributes = input.getAttributeMap(); + com.google.pubsub.v1.PubsubMessage.Builder message = +
[beam] branch revert-15608-dpcollins-cherry-pick created (now bb5ecd1)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch revert-15608-dpcollins-cherry-pick in repository https://gitbox.apache.org/repos/asf/beam.git. at bb5ecd1 Revert "Cherrypick #15418 and #15515" This branch includes the following new commits: new bb5ecd1 Revert "Cherrypick #15418 and #15515" 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.
[beam] branch release-2.33.0 updated: [BEAM-9487] Revert PR15340 for Beam 2.33
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch release-2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.33.0 by this push: new 1a7271d [BEAM-9487] Revert PR15340 for Beam 2.33 new 3b74c92 Merge pull request #15609 from zhoufek/reversion 1a7271d is described below commit 1a7271d99efb2ca9718d0e894d7db5e8b8c4d486 Author: zhoufek AuthorDate: Tue Sep 28 14:28:19 2021 -0400 [BEAM-9487] Revert PR15340 for Beam 2.33 --- CHANGES.md | 5 ++-- .../python/apache_beam/options/pipeline_options.py | 3 ++- sdks/python/apache_beam/transforms/core.py | 31 -- 3 files changed, 21 insertions(+), 18 deletions(-) diff --git a/CHANGES.md b/CHANGES.md index f2076b8..089ee9a 100644 --- a/CHANGES.md +++ b/CHANGES.md @@ -73,8 +73,6 @@ ## Breaking Changes -* Python GBK by defualt will fail on unbounded PCollections that have global windowing and a default trigger. The `--allow_unsafe_triggers` flag can be used to override this. ([BEAM-9487](https://issues.apache.org/jira/browse/BEAM-9487)). -* Python GBK will fail if it detects an unsafe trigger unless the `--allow_unsafe_triggers` flag is set. ([BEAM-9487](https://issues.apache.org/jira/browse/BEAM-9487)). * Go SDK pipelines require new import paths to use this release due to migration to Go Modules. * `go.mod` files will need to change to require `github.com/apache/beam/sdks/v2`. * Code depending on beam imports need to include v2 on the module path. @@ -83,7 +81,8 @@ ## Deprecations -* X behavior is deprecated and will be removed in X versions ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). +* Python GBK will stop supporting unbounded PCollections that have global windowing and a default trigger in Beam 2.34. This can be overriden with `--allow_unsafe_triggers`. ([BEAM-9487](https://issues.apache.org/jira/browse/BEAM-9487)). +* Python GBK will start requiring safe triggers or the `--allow_unsafe_triggers` flag starting with Beam 2.34. ([BEAM-9487](https://issues.apache.org/jira/browse/BEAM-9487)). ## Known Issues diff --git a/sdks/python/apache_beam/options/pipeline_options.py b/sdks/python/apache_beam/options/pipeline_options.py index bba56ef..622c9f0 100644 --- a/sdks/python/apache_beam/options/pipeline_options.py +++ b/sdks/python/apache_beam/options/pipeline_options.py @@ -534,7 +534,8 @@ class TypeOptions(PipelineOptions): 'compatibility. See BEAM-11719.') parser.add_argument( '--allow_unsafe_triggers', -default=False, +# TODO(BEAM-9487): Set to False for Beam 2.34 +default=True, action='store_true', help='Allow the use of unsafe triggers. Unsafe triggers have the ' 'potential to cause data loss due to finishing and/or never having ' diff --git a/sdks/python/apache_beam/transforms/core.py b/sdks/python/apache_beam/transforms/core.py index 25b05df..d5a1abd 100644 --- a/sdks/python/apache_beam/transforms/core.py +++ b/sdks/python/apache_beam/transforms/core.py @@ -2318,12 +2318,12 @@ class GroupByKey(PTransform): windowing.windowfn, GlobalWindows) and isinstance(trigger, DefaultTrigger): if pcoll.pipeline.allow_unsafe_triggers: -# TODO(BEAM-9487) Change comment for Beam 2.33 +# TODO(BEAM-9487) Change comment for Beam 2.34 _LOGGER.warning( -'%s: PCollection passed to GroupByKey is unbounded, has a global ' -'window, and uses a default trigger. This is being allowed ' -'because --allow_unsafe_triggers is set, but it may prevent ' -'data from making it through the pipeline.', +'PCollection passed to GroupByKey (label: %s) is unbounded, has a ' +'global window, and uses a default trigger. This will no longer ' +'be allowed starting with Beam 2.34 unless ' +'--allow_unsafe_triggers is set.', self.label) else: raise ValueError( @@ -2332,19 +2332,22 @@ class GroupByKey(PTransform): unsafe_reason = trigger.may_lose_data(windowing) if unsafe_reason != DataLossReason.NO_POTENTIAL_LOSS: - reason_msg = str(unsafe_reason).replace('DataLossReason.', '') if pcoll.pipeline.allow_unsafe_triggers: +# TODO(BEAM-9487): Switch back to this log for Beam 2.34. +# _LOGGER.warning( +# 'Skipping trigger safety check. ' +# 'This could lead to incomplete or missing groups.') _LOGGER.warning( -'%s: Unsafe trigger `%s` detected (reason: %s). This is ' -'being allowed because --allow_unsafe_triggers is set. This could ' -'lead to missing or incomplete groups.', +'%s: Unsafe trigger type (%s) detected. Starting
[beam] branch release-2.33.0 updated (a3c869d -> 583e523)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch release-2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git. from a3c869d Relocate breaking Go SDK change out of template (#15575) new 1abd6ba Port changes from Pub/Sub Lite to beam (#15418) new d60e52f Patch in #15515 new 583e523 Merge pull request #15608 from dpcollins-google/dpcollins-cherry-pick The 32777 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/gradle/BeamModulePlugin.groovy | 2 +- .../beam/sdk/io/gcp/pubsub/PubsubMessages.java | 42 ++-- .../sdk/io/gcp/pubsublite/CloudPubsubChecks.java | 51 .../io/gcp/pubsublite/CloudPubsubTransforms.java | 104 ...plier.java => ManagedBacklogReaderFactory.java} | 14 +- .../ManagedBacklogReaderFactoryImpl.java | 68 + ...criptionPartition.java => OffsetByteRange.java} | 21 +- ...rtitionCoder.java => OffsetByteRangeCoder.java} | 29 +-- .../io/gcp/pubsublite/OffsetByteRangeTracker.java | 66 ++--- .../io/gcp/pubsublite/PerServerPublisherCache.java | 4 + .../pubsublite/PerSubscriptionPartitionSdf.java| 57 +++-- .../beam/sdk/io/gcp/pubsublite/PublisherCache.java | 62 +++-- .../beam/sdk/io/gcp/pubsublite/Publishers.java | 72 -- .../beam/sdk/io/gcp/pubsublite/PubsubLiteIO.java | 2 +- .../beam/sdk/io/gcp/pubsublite/PubsubLiteSink.java | 10 +- .../sdk/io/gcp/pubsublite/SubscribeTransform.java | 44 ++-- .../sdk/io/gcp/pubsublite/SubscriberOptions.java | 26 +- .../SubscriptionPartitionProcessorFactory.java | 3 +- .../SubscriptionPartitionProcessorImpl.java| 23 +- .../gcp/pubsublite/TopicBacklogReaderSettings.java | 6 +- ...lOffsetReader.java => TrackerWithProgress.java} | 12 +- .../gcp/pubsublite/OffsetByteRangeTrackerTest.java | 34 ++- .../PerSubscriptionPartitionSdfTest.java | 70 +- .../beam/sdk/io/gcp/pubsublite/ReadWriteIT.java| 280 + .../SubscriptionPartitionProcessorImplTest.java| 49 ++-- 25 files changed, 818 insertions(+), 333 deletions(-) delete mode 100644 sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/CloudPubsubChecks.java create mode 100644 sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/CloudPubsubTransforms.java copy sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/{SerializableSupplier.java => ManagedBacklogReaderFactory.java} (69%) create mode 100644 sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/ManagedBacklogReaderFactoryImpl.java copy sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/{SubscriptionPartition.java => OffsetByteRange.java} (69%) copy sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/{SubscriptionPartitionCoder.java => OffsetByteRangeCoder.java} (60%) copy sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsublite/{InitialOffsetReader.java => TrackerWithProgress.java} (74%) create mode 100644 sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/pubsublite/ReadWriteIT.java
svn commit: r49968 - /dev/beam/2.33.0/python/
Author: udim Date: Thu Sep 16 16:33:56 2021 New Revision: 49968 Log: Staging Python artifacts for Apache Beam 2.33.0 RC1 Added: dev/beam/2.33.0/python/ dev/beam/2.33.0/python/apache-beam-2.33.0.zip (with props) dev/beam/2.33.0/python/apache-beam-2.33.0.zip.asc dev/beam/2.33.0/python/apache-beam-2.33.0.zip.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-macosx_10_9_x86_64.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-macosx_10_9_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-macosx_10_9_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_i686.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_i686.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_i686.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_x86_64.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux1_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_i686.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_i686.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_i686.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_x86_64.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2010_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2014_aarch64.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2014_aarch64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-manylinux2014_aarch64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win32.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win32.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win32.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win_amd64.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win_amd64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp36-cp36m-win_amd64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-macosx_10_9_x86_64.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-macosx_10_9_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-macosx_10_9_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_i686.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_i686.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_i686.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_x86_64.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux1_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_i686.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_i686.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_i686.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_x86_64.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2010_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2014_aarch64.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2014_aarch64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-manylinux2014_aarch64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win32.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win32.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win32.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win_amd64.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win_amd64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp37-cp37m-win_amd64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-macosx_10_9_x86_64.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-macosx_10_9_x86_64.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-macosx_10_9_x86_64.whl.sha512 dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-manylinux1_i686.whl (with props) dev/beam/2.33.0/python/apache_beam-2.33.0-cp38-cp38-manylinux1_i686.whl.asc dev/beam/2.33.0/python/apache_beam-2.33.0-cp38
svn commit: r49967 - in /dev/beam/2.33.0: ./ apache-beam-2.33.0-source-release.zip apache-beam-2.33.0-source-release.zip.asc apache-beam-2.33.0-source-release.zip.sha512
Author: udim Date: Thu Sep 16 16:29:13 2021 New Revision: 49967 Log: Staging Java artifacts for Apache Beam 2.33.0 RC1 Added: dev/beam/2.33.0/ dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip (with props) dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 Added: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip == Binary file - no diff available. Propchange: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip -- svn:mime-type = application/octet-stream Added: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc == --- dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc (added) +++ dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.asc Thu Sep 16 16:29:13 2021 @@ -0,0 +1,16 @@ +-BEGIN PGP SIGNATURE- + +iQIzBAABCgAdFiEEiWHz7455ZohAZ4fPWHsEnDbar+YFAmFDcKUACgkQWHsEnDba +r+Y0pg//fdCH6mCIzh/y/ufzAJhWN0wnuB8sZFr7PQVdfv04eKufD01g3hcU9s77 +V/f0K4kMUDaheD07UxL9OaqxhbzQsMxRXIdZ+vbx++H9v/2pyqGoi5b+akmogXvZ +JB0Ixio5QWks9ghaV9btkx+JencptT/Cj2YWsOMf4Dwa0+ppa62n7Mjg7WPRyjoc +RagxDfEsrJ5Sur8Ve8L8Q6HnnCPOX2VG4//wGsbX429pOQu0Ej037oGjIne2imx2 +chF1LOpNCAPZDJHPQbCG5vHVgBI/IDbl7IQC7yiRaY/sH34G6572ug8dosnQLlLM +nPIOZ7OQ260Bk7w4YB/BeQ5aPAbhCxuW7N7/xZSYBtuyTgC8b6Bd//WU3dvWRvuC +SDvLPf9FZMKZrRxApXW19gubjWze478u3UPRe+sVwkuUQWuviYlPbs3CwUeI3SWV +vdXrVeddWb64vnss35NT6A0ggDTqOzoK/+kNg3mBu1nSQqnY8eGF6HnjzWlYx/3G +DK1f4/t4oGOXlzncPfCa7L8phoX94fyvZ8EY8/GE5kvS/kqmzRXxsQPCAk1uggWA +O5c2zbOzD8fBFNYD/F9edcbvx2T1p9kLuAQqQe+df2o5+aA9/kTnI3V9NzJ8ZFL9 +GR9bAhqHGEV79X57OYgpm3PUfZhxvnvLCww8NRfmWwhetb7C9RE= +=4MBB +-END PGP SIGNATURE- Added: dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 == --- dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 (added) +++ dev/beam/2.33.0/apache-beam-2.33.0-source-release.zip.sha512 Thu Sep 16 16:29:13 2021 @@ -0,0 +1 @@ +fd7a91f75b4b48b6aeac8b3edc830c21b8a8a866a045132beb4af62a640fdf706eb20cc4d82b2ab2412abd18b1a1ef75da13256d543dd8efb67ffb545e575702 apache-beam-2.33.0-source-release.zip
[beam] annotated tag v2.33.0-RC1 updated (b358127 -> 707cb85)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to annotated tag v2.33.0-RC1 in repository https://gitbox.apache.org/repos/asf/beam.git. *** WARNING: tag v2.33.0-RC1 was modified! *** from b358127 (commit) to 707cb85 (tag) tagging b358127f9859f7e0fd90993fc9e7e8fa3e2c9a9c (commit) replaces jupyterlab-sidepanel-v1.0.0 by Udi Meiri on Tue Sep 14 14:15:13 2021 -0700 - Log - v2.33.0-RC1 --- No new revisions were added by this update. Summary of changes:
[beam] branch release-2.33.0 updated: [BEAM-12829] Fix Release Gradle Build (#15444)
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch release-2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.33.0 by this push: new 8b408c0 [BEAM-12829] Fix Release Gradle Build (#15444) 8b408c0 is described below commit 8b408c0ebc02c7dfa47b4a26f1f238e073174f34 Author: Robert Burke AuthorDate: Thu Sep 2 10:15:52 2021 -0700 [BEAM-12829] Fix Release Gradle Build (#15444) Co-authored-by: lostluck <13907733+lostl...@users.noreply.github.com> --- buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy | 1 + sdks/go/build.gradle| 1 + sdks/go/container/build.gradle | 1 + sdks/go/examples/build.gradle | 1 + sdks/go/test/build.gradle | 1 + sdks/go/test/load/build.gradle | 1 + sdks/java/container/build.gradle| 1 + sdks/python/container/build.gradle | 1 + 8 files changed, 8 insertions(+) diff --git a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy index ce507db..59ca672 100644 --- a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy +++ b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy @@ -2295,6 +2295,7 @@ class BeamModulePlugin implements Plugin { '**/build/**', '**/dist/**', '**/target/**', +'**/.gogradle/**', '**/*.pyc', 'sdks/python/*.egg*/**', 'sdks/python/test-suites/**', diff --git a/sdks/go/build.gradle b/sdks/go/build.gradle index 0752a2f..f71cfe3 100644 --- a/sdks/go/build.gradle +++ b/sdks/go/build.gradle @@ -25,6 +25,7 @@ description = "Apache Beam :: SDKs :: Go" installDependencies.enabled = false resolveBuildDependencies.enabled = false resolveTestDependencies.enabled = false +gofmt.enabled = false golang { packagePath = 'github.com/apache/beam/sdks/v2/go' diff --git a/sdks/go/container/build.gradle b/sdks/go/container/build.gradle index 643ffdc..7eebec7 100644 --- a/sdks/go/container/build.gradle +++ b/sdks/go/container/build.gradle @@ -26,6 +26,7 @@ description = "Apache Beam :: SDKs :: Go :: Container" installDependencies.enabled = false resolveBuildDependencies.enabled = false resolveTestDependencies.enabled = false +gofmt.enabled = false clean.dependsOn cleanVendor diff --git a/sdks/go/examples/build.gradle b/sdks/go/examples/build.gradle index 26f0fa8..2c35199 100644 --- a/sdks/go/examples/build.gradle +++ b/sdks/go/examples/build.gradle @@ -25,6 +25,7 @@ description = "Apache Beam :: SDKs :: Go :: Examples" installDependencies.enabled = false resolveBuildDependencies.enabled = false resolveTestDependencies.enabled = false +gofmt.enabled = false def getLocalPlatform = { String hostOs = com.github.blindpirate.gogradle.crossplatform.Os.getHostOs() diff --git a/sdks/go/test/build.gradle b/sdks/go/test/build.gradle index 90fd52f..c325942 100644 --- a/sdks/go/test/build.gradle +++ b/sdks/go/test/build.gradle @@ -26,6 +26,7 @@ description = "Apache Beam :: SDKs :: Go :: Test" installDependencies.enabled = false resolveBuildDependencies.enabled = false resolveTestDependencies.enabled = false +gofmt.enabled = false clean.dependsOn cleanVendor diff --git a/sdks/go/test/load/build.gradle b/sdks/go/test/load/build.gradle index a006fcd..3af32c8 100644 --- a/sdks/go/test/load/build.gradle +++ b/sdks/go/test/load/build.gradle @@ -25,6 +25,7 @@ description = "Apache Beam :: SDKs :: Go :: Test :: Load" installDependencies.enabled = false resolveBuildDependencies.enabled = false resolveTestDependencies.enabled = false +gofmt.enabled = false def getLocalPlatform = { String hostOs = com.github.blindpirate.gogradle.crossplatform.Os.getHostOs() diff --git a/sdks/java/container/build.gradle b/sdks/java/container/build.gradle index 23cfd42..b71010f 100644 --- a/sdks/java/container/build.gradle +++ b/sdks/java/container/build.gradle @@ -32,6 +32,7 @@ description = "Apache Beam :: SDKs :: Java :: Container" installDependencies.enabled = false resolveBuildDependencies.enabled = false resolveTestDependencies.enabled = false +gofmt.enabled = false configurations { dockerDependency diff --git a/sdks/python/container/build.gradle b/sdks/python/container/build.gradle index 32d9bf3..3dc14e2 100644 --- a/sdks/python/container/build.gradle +++ b/sdks/python/container/build.gradle @@ -25,6 +25,7 @@ description = "Apache Beam :: SDKs :: Python :: Container" installDependencies.enabled = false resol
[beam] branch master updated (0592083 -> a36e332)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 0592083 [BEAM-12767] Improve PipelineOption parsing UX add d09fceb Fix typo in BigQuery documentation add a36e332 Merge pull request #15413: Fix typo in BigQuery documentation No new revisions were added by this update. Summary of changes: sdks/python/apache_beam/io/gcp/bigquery.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
[beam] branch release-2.33.0 updated: Revert "Merge pull request #15271 Decreasing peak memory usage for beam.TupleCombineFn."
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch release-2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.33.0 by this push: new 16de773 Revert "Merge pull request #15271 Decreasing peak memory usage for beam.TupleCombineFn." new b3328ee Merge pull request #15422 from ibzib/combine-rollback-release 16de773 is described below commit 16de7733bd9b7060113da5df3d70e48276274390 Author: Kyle Weaver AuthorDate: Thu Aug 26 16:28:24 2021 -0700 Revert "Merge pull request #15271 Decreasing peak memory usage for beam.TupleCombineFn." This reverts commit 4559c75863d9d6c9dd9e48c2b4f12f2139410524, reversing changes made to 7611831443399f31fc505bc3451f2b56f245d4e4. --- sdks/python/apache_beam/transforms/combiners.py| 34 --- .../apache_beam/transforms/combiners_test.py | 38 +- 2 files changed, 7 insertions(+), 65 deletions(-) diff --git a/sdks/python/apache_beam/transforms/combiners.py b/sdks/python/apache_beam/transforms/combiners.py index 7e6b1f9..bcedd86 100644 --- a/sdks/python/apache_beam/transforms/combiners.py +++ b/sdks/python/apache_beam/transforms/combiners.py @@ -21,7 +21,6 @@ import copy import heapq -import itertools import operator import random from typing import Any @@ -598,24 +597,16 @@ class SampleCombineFn(core.CombineFn): class _TupleCombineFnBase(core.CombineFn): - def __init__(self, *combiners, merge_accumulators_batch_size=None): + def __init__(self, *combiners): self._combiners = [core.CombineFn.maybe_from_callable(c) for c in combiners] self._named_combiners = combiners -# If the `merge_accumulators_batch_size` value is not specified, we chose a -# bounded default that is inversely proportional to the number of -# accumulators in merged tuples. -self._merge_accumulators_batch_size = ( -merge_accumulators_batch_size or max(10, 1000 // len(combiners))) def display_data(self): combiners = [ c.__name__ if hasattr(c, '__name__') else c.__class__.__name__ for c in self._named_combiners ] -return { -'combiners': str(combiners), -'merge_accumulators_batch_size': self._merge_accumulators_batch_size -} +return {'combiners': str(combiners)} def setup(self, *args, **kwargs): for c in self._combiners: @@ -625,23 +616,10 @@ class _TupleCombineFnBase(core.CombineFn): return [c.create_accumulator(*args, **kwargs) for c in self._combiners] def merge_accumulators(self, accumulators, *args, **kwargs): -# Make sure that `accumulators` is an iterator (so that the position is -# remembered). -accumulators = iter(accumulators) -result = next(accumulators) -while True: - # Load accumulators into memory and merge in batches to decrease peak - # memory usage. - accumulators_batch = list( - itertools.islice(accumulators, self._merge_accumulators_batch_size)) - if not accumulators_batch: -break - accumulators_batch += [result] - result = [ - c.merge_accumulators(a, *args, **kwargs) for c, - a in zip(self._combiners, zip(*accumulators_batch)) - ] -return result +return [ +c.merge_accumulators(a, *args, **kwargs) for c, +a in zip(self._combiners, zip(*accumulators)) +] def compact(self, accumulator, *args, **kwargs): return [ diff --git a/sdks/python/apache_beam/transforms/combiners_test.py b/sdks/python/apache_beam/transforms/combiners_test.py index 68b273e..d826287 100644 --- a/sdks/python/apache_beam/transforms/combiners_test.py +++ b/sdks/python/apache_beam/transforms/combiners_test.py @@ -249,8 +249,7 @@ class CombineTest(unittest.TestCase): dd = DisplayData.create_from(transform) expected_items = [ DisplayDataItemMatcher('combine_fn', combine.TupleCombineFn), -DisplayDataItemMatcher('combiners', "['max', 'MeanCombineFn', 'sum']"), -DisplayDataItemMatcher('merge_accumulators_batch_size', 333), +DisplayDataItemMatcher('combiners', "['max', 'MeanCombineFn', 'sum']") ] hc.assert_that(dd.items, hc.contains_inanyorder(*expected_items)) @@ -359,41 +358,6 @@ class CombineTest(unittest.TestCase): max).with_common_input()).without_defaults()) assert_that(result, equal_to([(1, 7.0 / 4, 3)])) - def test_tuple_combine_fn_batched_merge(self): -num_combine_fns = 10 -max_num_accumulators_in_memory = 30 -# Maximum number of accumulator tuples in memory - 1 for the merge result. -merge_accumulators_batch_size = ( -max_num_accumulators_in_memory // num_combine_fns - 1) -num_accumulator_tuples_to_merge = 20 - -class CountedAccumulator: - count = 0 - oom = False - - def __init__(sel
[beam] branch master updated: Remove duplicate 2.33.0 section
This is an automated email from the ASF dual-hosted git repository. udim 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 c73ec81 Remove duplicate 2.33.0 section new dd7945f Merge pull request #15423 from angoenka/fix-release-notes c73ec81 is described below commit c73ec818ad4703aea60855c169eff4fd039ea5cb Author: Ankur AuthorDate: Mon Aug 30 13:32:10 2021 -0700 Remove duplicate 2.33.0 section --- CHANGES.md | 28 1 file changed, 28 deletions(-) diff --git a/CHANGES.md b/CHANGES.md index 7f6bf2e..1439ef0 100644 --- a/CHANGES.md +++ b/CHANGES.md @@ -121,34 +121,6 @@ * Fixed X (Java/Python) ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). - -# [2.33.0] - Unreleased - -## Highlights - -* New highly anticipated feature X added to Python SDK ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). -* New highly anticipated feature Y added to Java SDK ([BEAM-Y](https://issues.apache.org/jira/browse/BEAM-Y)). - -## I/Os - -* Support for X source added (Java/Python) ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). - -## New Features / Improvements - -* X feature added (Java/Python) ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). - -## Breaking Changes - -* X behavior was changed ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). - -## Deprecations - -* X behavior is deprecated and will be removed in X versions ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). - -## Known Issues - -* Fixed X (Java/Python) ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). - # [2.32.0] - 2021-08-25 ## Highlights
[beam] branch release-2.33.0 updated: Cherry-picking #15402.
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch release-2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git The following commit(s) were added to refs/heads/release-2.33.0 by this push: new 96b0c00 Cherry-picking #15402. new f1c8722 Merge pull request #15408 from vachan-shetty/cherrypick-15402 96b0c00 is described below commit 96b0c0058774ee3df19abd011a36ce4681ff Author: vachan-shetty AuthorDate: Fri Aug 27 10:44:10 2021 -0400 Cherry-picking #15402. --- sdks/python/apache_beam/io/gcp/bigquery.py | 340 + .../apache_beam/io/gcp/bigquery_read_it_test.py| 159 -- sdks/python/setup.py | 31 +- 3 files changed, 26 insertions(+), 504 deletions(-) diff --git a/sdks/python/apache_beam/io/gcp/bigquery.py b/sdks/python/apache_beam/io/gcp/bigquery.py index 4e5f639..88093b1 100644 --- a/sdks/python/apache_beam/io/gcp/bigquery.py +++ b/sdks/python/apache_beam/io/gcp/bigquery.py @@ -270,7 +270,6 @@ encoding when writing to BigQuery. # pytype: skip-file import collections -import io import itertools import json import logging @@ -278,20 +277,13 @@ import random import time import uuid from typing import Dict -from typing import List -from typing import Optional from typing import Union -import avro.schema -import fastavro -from avro import io as avroio - import apache_beam as beam from apache_beam import coders from apache_beam import pvalue from apache_beam.internal.gcp.json_value import from_json_value from apache_beam.internal.gcp.json_value import to_json_value -from apache_beam.io import range_trackers from apache_beam.io.avroio import _create_avro_source as create_avro_source from apache_beam.io.filesystems import CompressionTypes from apache_beam.io.filesystems import FileSystems @@ -332,7 +324,6 @@ from apache_beam.utils.annotations import experimental try: from apache_beam.io.gcp.internal.clients.bigquery import DatasetReference from apache_beam.io.gcp.internal.clients.bigquery import TableReference - import google.cloud.bigquery_storage_v1 as bq_storage except ImportError: DatasetReference = None TableReference = None @@ -896,276 +887,6 @@ class _CustomBigQuerySource(BoundedSource): return table.schema, metadata_list -class _CustomBigQueryStorageSourceBase(BoundedSource): - """A base class for BoundedSource implementations which read from BigQuery - using the BigQuery Storage API. - - Args: -table (str, TableReference): The ID of the table. The ID must contain only - letters ``a-z``, ``A-Z``, numbers ``0-9``, or underscores ``_`` If - **dataset** argument is :data:`None` then the table argument must - contain the entire table reference specified as: - ``'PROJECT:DATASET.TABLE'`` or must specify a TableReference. -dataset (str): Optional ID of the dataset containing this table or - :data:`None` if the table argument specifies a TableReference. -project (str): Optional ID of the project containing this table or - :data:`None` if the table argument specifies a TableReference. -selected_fields (List[str]): Optional List of names of the fields in the - table that should be read. If empty, all fields will be read. If the - specified field is a nested field, all the sub-fields in the field will be - selected. The output field order is unrelated to the order of fields in - selected_fields. -row_restriction (str): Optional SQL text filtering statement, similar to a - WHERE clause in a query. Aggregates are not supported. Restricted to a - maximum length for 1 MB. - """ - - # The maximum number of streams which will be requested when creating a read - # session, regardless of the desired bundle size. - MAX_SPLIT_COUNT = 1 - # The minimum number of streams which will be requested when creating a read - # session, regardless of the desired bundle size. Note that the server may - # still choose to return fewer than ten streams based on the layout of the - # table. - MIN_SPLIT_COUNT = 10 - - def __init__( - self, - table: Union[str, TableReference], - dataset: Optional[str] = None, - project: Optional[str] = None, - selected_fields: Optional[List[str]] = None, - row_restriction: Optional[str] = None, - use_fastavro_for_direct_read: Optional[bool] = None, - pipeline_options: Optional[GoogleCloudOptions] = None): - -self.table_reference = bigquery_tools.parse_table_reference( -table, dataset, project) -self.table = self.table_reference.tableId -self.dataset = self.table_reference.datasetId -self.project = self.table_reference.projectId -self.selected_fields = selected_fields -self.row_restriction = row_restriction -self.use_fastavro = \ - True if use_fastavro_for_direct_read is None else \ -
[beam] branch master updated (e0f0978 -> f4e54f2)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from e0f0978 Moving to 2.34.0-SNAPSHOT on master branch. add f4e54f2 Add 2.34.0 section to CHANGES.md No new revisions were added by this update. Summary of changes: CHANGES.md | 34 +- 1 file changed, 33 insertions(+), 1 deletion(-)
[beam] 01/01: Set Dataflow container to release version.
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch release-2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git commit 6d595f728b46155274795a989898e93e7799ecc5 Author: Udi Meiri AuthorDate: Wed Aug 25 14:18:22 2021 -0700 Set Dataflow container to release version. --- runners/google-cloud-dataflow-java/build.gradle | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/runners/google-cloud-dataflow-java/build.gradle b/runners/google-cloud-dataflow-java/build.gradle index b8ec9bb..969b2b3 100644 --- a/runners/google-cloud-dataflow-java/build.gradle +++ b/runners/google-cloud-dataflow-java/build.gradle @@ -45,8 +45,8 @@ processResources { filter org.apache.tools.ant.filters.ReplaceTokens, tokens: [ 'dataflow.legacy_environment_major_version' : '8', 'dataflow.fnapi_environment_major_version' : '8', -'dataflow.legacy_container_version' : 'beam-master-20210525', -'dataflow.fnapi_container_version' : 'beam-master-20210524', +'dataflow.legacy_container_version' : 'beam-2.33.0', +'dataflow.fnapi_container_version' : 'beam-2.33.0', 'dataflow.container_base_repository' : 'gcr.io/cloud-dataflow/v1beta3', ] }
[beam] branch release-2.33.0 created (now 6d595f7)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch release-2.33.0 in repository https://gitbox.apache.org/repos/asf/beam.git. at 6d595f7 Set Dataflow container to release version. This branch includes the following new commits: new 6d595f7 Set Dataflow container to release version. 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.
[beam] branch master updated: Moving to 2.34.0-SNAPSHOT on master branch.
This is an automated email from the ASF dual-hosted git repository. udim 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 e0f0978 Moving to 2.34.0-SNAPSHOT on master branch. e0f0978 is described below commit e0f09783e226c549160de5d9e896f3dd563ca040 Author: Udi Meiri AuthorDate: Wed Aug 25 14:13:45 2021 -0700 Moving to 2.34.0-SNAPSHOT on master branch. --- .../src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy| 2 +- gradle.properties | 4 ++-- sdks/go/pkg/beam/core/core.go | 2 +- sdks/python/apache_beam/version.py| 2 +- 4 files changed, 5 insertions(+), 5 deletions(-) diff --git a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy index ce507db..c4f5d85 100644 --- a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy +++ b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy @@ -372,7 +372,7 @@ class BeamModulePlugin implements Plugin { // Automatically use the official release version if we are performing a release // otherwise append '-SNAPSHOT' -project.version = '2.33.0' +project.version = '2.34.0' if (!isRelease(project)) { project.version += '-SNAPSHOT' } diff --git a/gradle.properties b/gradle.properties index e71c2f6..e3a005d 100644 --- a/gradle.properties +++ b/gradle.properties @@ -24,8 +24,8 @@ offlineRepositoryRoot=offline-repository signing.gnupg.executable=gpg signing.gnupg.useLegacyGpg=true -version=2.33.0-SNAPSHOT -sdk_version=2.33.0.dev +version=2.34.0-SNAPSHOT +sdk_version=2.34.0.dev javaVersion=1.8 diff --git a/sdks/go/pkg/beam/core/core.go b/sdks/go/pkg/beam/core/core.go index 34fe071..74e42b4 100644 --- a/sdks/go/pkg/beam/core/core.go +++ b/sdks/go/pkg/beam/core/core.go @@ -27,5 +27,5 @@ const ( // SdkName is the human readable name of the SDK for UserAgents. SdkName = "Apache Beam SDK for Go" // SdkVersion is the current version of the SDK. - SdkVersion = "2.33.0.dev" + SdkVersion = "2.34.0.dev" ) diff --git a/sdks/python/apache_beam/version.py b/sdks/python/apache_beam/version.py index 27d47f2..264029d 100644 --- a/sdks/python/apache_beam/version.py +++ b/sdks/python/apache_beam/version.py @@ -17,4 +17,4 @@ """Apache Beam SDK version information and utilities.""" -__version__ = '2.33.0.dev' +__version__ = '2.34.0.dev'
[beam] branch master updated: [BEAM-9487] Disable allowing unsafe triggers by default
This is an automated email from the ASF dual-hosted git repository. udim 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 706e20f [BEAM-9487] Disable allowing unsafe triggers by default new cf8e08f Merge pull request #15340 from zhoufek/gbk_233 706e20f is described below commit 706e20f7c3f0dafe834981e2dd2715082d38e548 Author: zhoufek AuthorDate: Tue Aug 17 09:03:56 2021 -0400 [BEAM-9487] Disable allowing unsafe triggers by default --- CHANGES.md | 6 +++-- .../python/apache_beam/options/pipeline_options.py | 3 +-- sdks/python/apache_beam/transforms/core.py | 29 ++ 3 files changed, 18 insertions(+), 20 deletions(-) diff --git a/CHANGES.md b/CHANGES.md index 6da170d..6fc25b8 100644 --- a/CHANGES.md +++ b/CHANGES.md @@ -63,7 +63,8 @@ ## Breaking Changes -* X behavior was changed ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). +* Python GBK by defualt will fail on unbounded PCollections that have global windowing and a default trigger. The `--allow_unsafe_triggers` flag can be used to override this. ([BEAM-9487](https://issues.apache.org/jira/browse/BEAM-9487)). +* Python GBK will fail if it detects an unsafe trigger unless the `--allow_unsafe_triggers` flag is set. ([BEAM-9487](https://issues.apache.org/jira/browse/BEAM-9487)). ## Deprecations @@ -123,7 +124,8 @@ ## Deprecations -* X behavior is deprecated and will be removed in X versions ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). +* Python GBK will stop supporting unbounded PCollections that have global windowing and a default trigger in Beam 2.33. This can be overriden with `--allow_unsafe_triggers`. ([BEAM-9487](https://issues.apache.org/jira/browse/BEAM-9487)). +* Python GBK will start requiring safe triggers or the `--allow_unsafe_triggers` flag starting with Beam 2.33. ([BEAM-9487](https://issues.apache.org/jira/browse/BEAM-9487)). ## Known Issues diff --git a/sdks/python/apache_beam/options/pipeline_options.py b/sdks/python/apache_beam/options/pipeline_options.py index c597b4a..bba56ef 100644 --- a/sdks/python/apache_beam/options/pipeline_options.py +++ b/sdks/python/apache_beam/options/pipeline_options.py @@ -534,8 +534,7 @@ class TypeOptions(PipelineOptions): 'compatibility. See BEAM-11719.') parser.add_argument( '--allow_unsafe_triggers', -# TODO(BEAM-9487): Set to False for Beam 2.33 -default=True, +default=False, action='store_true', help='Allow the use of unsafe triggers. Unsafe triggers have the ' 'potential to cause data loss due to finishing and/or never having ' diff --git a/sdks/python/apache_beam/transforms/core.py b/sdks/python/apache_beam/transforms/core.py index 93d29ed..25b05df 100644 --- a/sdks/python/apache_beam/transforms/core.py +++ b/sdks/python/apache_beam/transforms/core.py @@ -2320,10 +2320,10 @@ class GroupByKey(PTransform): if pcoll.pipeline.allow_unsafe_triggers: # TODO(BEAM-9487) Change comment for Beam 2.33 _LOGGER.warning( -'PCollection passed to GroupByKey (label: %s) is unbounded, has a ' -'global window, and uses a default trigger. This will no longer ' -'be allowed starting with Beam 2.33 unless ' -'--allow_unsafe_triggers is set.', +'%s: PCollection passed to GroupByKey is unbounded, has a global ' +'window, and uses a default trigger. This is being allowed ' +'because --allow_unsafe_triggers is set, but it may prevent ' +'data from making it through the pipeline.', self.label) else: raise ValueError( @@ -2332,22 +2332,19 @@ class GroupByKey(PTransform): unsafe_reason = trigger.may_lose_data(windowing) if unsafe_reason != DataLossReason.NO_POTENTIAL_LOSS: + reason_msg = str(unsafe_reason).replace('DataLossReason.', '') if pcoll.pipeline.allow_unsafe_triggers: -# TODO(BEAM-9487): Switch back to this log for Beam 2.33. -# _LOGGER.warning( -# 'Skipping trigger safety check. ' -# 'This could lead to incomplete or missing groups.') _LOGGER.warning( -'%s: Unsafe trigger type (%s) detected. Starting with ' -'Beam 2.33, this will raise an error by default. ' -'Either change the pipeline to use a safe trigger or ' -'set the --allow_unsafe_triggers flag.', +'%s: Unsafe trigger `%s` detected (reason: %s). This is ' +'being allowed because --allow_unsafe_triggers is set. This could ' +'lead to missing or incomplete groups.', self.label, -unsafe_reason) +trigger, +reason_msg) else: -msg = 'Unsafe trigger: `{}` may lose data
[beam] branch master updated: [BEAM-3713] Cleanup, remove nosetest references (#15245)
This is an automated email from the ASF dual-hosted git repository. udim 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 9188bbf [BEAM-3713] Cleanup, remove nosetest references (#15245) 9188bbf is described below commit 9188bbf05c4bdd0d306257d1c38c501c61398577 Author: Benjamin Gonzalez <74670721+benw...@users.noreply.github.com> AuthorDate: Wed Aug 18 15:58:26 2021 -0500 [BEAM-3713] Cleanup, remove nosetest references (#15245) * [BEAM-3713] Fix lint * [BEAM-3713] Remove --pytest param because is no longer required Co-authored-by: Udi Meiri --- .../org/apache/beam/gradle/BeamModulePlugin.groovy | 2 - .../cookbook/bigquery_tornadoes_it_test.py | 3 -- sdks/python/apache_beam/testing/test_pipeline.py | 4 +- .../transforms/periodicsequence_test.py| 2 - .../apache_beam/transforms/ptransform_test.py | 3 -- .../apache_beam/transforms/sideinputs_test.py | 3 -- .../transforms_keyword_only_args_test.py | 4 -- sdks/python/container/base_image_requirements.txt | 2 - sdks/python/scripts/run_integration_test.sh| 45 +++- sdks/python/setup.py | 7 sdks/python/test-suites/dataflow/common.gradle | 5 --- sdks/python/test-suites/direct/common.gradle | 3 -- sdks/python/test-suites/portable/common.gradle | 5 +-- sdks/python/test_config.py | 49 -- 14 files changed, 18 insertions(+), 119 deletions(-) diff --git a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy index f5a3c19..ce507db 100644 --- a/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy +++ b/buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy @@ -2174,7 +2174,6 @@ class BeamModulePlugin implements Plugin { "pipeline_opts": config.pythonPipelineOptions + sdkLocationOpt, "test_opts": config.pytestOptions, "suite": "xlangValidateRunner", - "pytest": true, // TODO(BEAM-3713): Remove this once nose is removed. "collect": config.pythonTestAttr ] def cmdArgs = project.project(':sdks:python').mapToArgString(beamPythonTestPipelineOptions) @@ -2221,7 +2220,6 @@ class BeamModulePlugin implements Plugin { "pipeline_opts": config.pythonPipelineOptions + sdkLocationOpt, "test_opts": config.pytestOptions, "suite": "xlangSqlValidateRunner", -"pytest": true, // TODO(BEAM-3713): Remove this once nose is removed. "collect": "xlang_sql_expansion_service" ] def cmdArgs = project.project(':sdks:python').mapToArgString(beamPythonTestPipelineOptions) diff --git a/sdks/python/apache_beam/examples/cookbook/bigquery_tornadoes_it_test.py b/sdks/python/apache_beam/examples/cookbook/bigquery_tornadoes_it_test.py index fa5f12c..8d44615 100644 --- a/sdks/python/apache_beam/examples/cookbook/bigquery_tornadoes_it_test.py +++ b/sdks/python/apache_beam/examples/cookbook/bigquery_tornadoes_it_test.py @@ -35,9 +35,6 @@ from apache_beam.testing.test_pipeline import TestPipeline class BigqueryTornadoesIT(unittest.TestCase): - # Enable nose tests running in parallel - _multiprocess_can_split_ = True - # The default checksum is a SHA-1 hash generated from sorted rows reading # from expected Bigquery table. DEFAULT_CHECKSUM = 'd860e636050c559a16a791aff40d6ad809d4daf0' diff --git a/sdks/python/apache_beam/testing/test_pipeline.py b/sdks/python/apache_beam/testing/test_pipeline.py index 2ba273e..910f149 100644 --- a/sdks/python/apache_beam/testing/test_pipeline.py +++ b/sdks/python/apache_beam/testing/test_pipeline.py @@ -40,7 +40,7 @@ class TestPipeline(Pipeline): It has a functionality to parse arguments from command line and build pipeline options for tests who runs against a pipeline runner and utilizes resources of the pipeline runner. Those test functions are recommended to be tagged by - ``@attr("ValidatesRunner")`` annotation. + ``@pytest.mark.it_validatesrunner`` annotation. In order to configure the test with customized pipeline options from command line, system argument ``--test-pipeline-options`` can be used to obtains a @@ -48,7 +48,7 @@ class TestPipeline(Pipeline): For example, use following command line to execute all ValidatesRunner tests:: -python setup.py nosetests -a ValidatesRunner \\ +pytest -m it_validatesrunner \\ --test-pipeline-options="--runner=DirectRunner \\ --job_name=myJobName \\ --num_workers=1
[beam] 01/01: Merge pull request #15233: Loosen typing extensions requirement
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit dcf5e8e64bdff766e67ac988af0ef1494d616998 Merge: 6fb4afe 31f81bc Author: Udi Meiri AuthorDate: Thu Aug 12 11:47:23 2021 -0700 Merge pull request #15233: Loosen typing extensions requirement sdks/python/setup.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
[beam] branch master updated (6fb4afe -> dcf5e8e)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 6fb4afe Merge pull request #15265 from [BEAM-12545] Extend FileSink by create_metadata add 31f81bc Loosen typing extensions requirement new dcf5e8e Merge pull request #15233: Loosen typing extensions requirement 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: sdks/python/setup.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
[beam] branch master updated (e4abe0d -> a1abdf1)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from e4abe0d [BEAM-12368] Updated doc nav to match wireframe (#15229) add a1abdf1 [BEAM-3713] Move PostCommit_Python from nosetest to pytest (#14859) No new revisions were added by this update. Summary of changes: .test-infra/jenkins/job_PostCommit_Python.groovy | 2 +- .../examples/complete/autocomplete_test.py | 4 +- .../examples/complete/game/game_stats_it_test.py | 6 +- .../complete/game/hourly_team_score_it_test.py | 6 +- .../examples/complete/game/leader_board_it_test.py | 6 +- .../examples/complete/game/user_score_it_test.py | 6 +- .../complete/juliaset/juliaset/juliaset_test_it.py | 4 +- .../cookbook/bigquery_tornadoes_it_test.py | 4 +- .../cookbook/datastore_wordcount_it_test.py| 4 +- .../examples/dataframe/flight_delays_it_test.py| 4 +- .../examples/dataframe/taxiride_it_test.py | 6 +- .../apache_beam/examples/fastavro_it_test.py | 8 +-- .../streaming_wordcount_debugging_it_test.py | 4 +- .../examples/streaming_wordcount_it_test.py| 4 +- .../apache_beam/examples/wordcount_it_test.py | 8 +-- sdks/python/apache_beam/io/fileio_test.py | 4 +- .../io/gcp/big_query_query_to_table_it_test.py | 14 ++-- .../apache_beam/io/gcp/bigquery_file_loads_test.py | 8 +-- .../apache_beam/io/gcp/bigquery_io_read_it_test.py | 6 +- .../apache_beam/io/gcp/bigquery_read_it_test.py| 12 ++-- sdks/python/apache_beam/io/gcp/bigquery_test.py| 19 ++ .../apache_beam/io/gcp/bigquery_write_it_test.py | 12 ++-- .../gcp/datastore/v1new/datastore_write_it_test.py | 4 +- .../apache_beam/io/gcp/dicomio_integration_test.py | 6 +- .../io/gcp/experimental/spannerio_read_it_test.py | 6 +- .../io/gcp/experimental/spannerio_write_it_test.py | 8 +-- .../apache_beam/io/gcp/gcsio_integration_test.py | 14 ++-- .../apache_beam/io/gcp/pubsub_integration_test.py | 6 +- sdks/python/apache_beam/io/parquetio_it_test.py| 4 +- .../python/apache_beam/ml/gcp/cloud_dlp_it_test.py | 6 +- .../ml/gcp/naturallanguageml_test_it.py| 4 +- .../ml/gcp/recommendations_ai_test_it.py | 4 +- .../ml/gcp/videointelligenceml_test_it.py | 4 +- sdks/python/apache_beam/ml/gcp/visionml_test_it.py | 4 +- .../dataflow_exercise_metrics_pipeline_test.py | 5 +- .../apache_beam/testing/test_stream_it_test.py | 8 +-- .../apache_beam/transforms/external_it_test.py | 4 +- sdks/python/pytest.ini | 1 + sdks/python/test-suites/dataflow/common.gradle | 34 +- sdks/python/test-suites/direct/common.gradle | 74 +++--- sdks/python/test-suites/portable/common.gradle | 15 +++-- 41 files changed, 175 insertions(+), 187 deletions(-)
[beam] branch master updated (796fb9a -> 9c24ba6)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 796fb9a Merge pull request #15049: [BEAM-12456] Parallel querying in JdbcIO add a232b9d Fix generator send_type / return_type warning. new 9c24ba6 Merge pull request #15061: [BEAM-8473] Fix generator send_type / return_type warning 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: sdks/python/apache_beam/typehints/typehints.py | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-)
[beam] 01/01: Merge pull request #15061: [BEAM-8473] Fix generator send_type / return_type warning
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 9c24ba6fa2b0a5ee2dc463c7a1c99a5ee0027a9c Merge: 796fb9a a232b9d Author: Udi Meiri AuthorDate: Fri Jul 2 09:22:23 2021 -0700 Merge pull request #15061: [BEAM-8473] Fix generator send_type / return_type warning sdks/python/apache_beam/typehints/typehints.py | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-)
[beam] branch master updated (7494d14 -> 9418994)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 7494d14 Merge pull request #14460 from MaksymSkorupskyi/BEAM-12122-Python-IO-MongoDB-integer-and-string-`_id`-keys-are-not-supported add 9418994 [BEAM-3713] Move PerformanceTest and CrossLanguageValidateRunner from nosetest to pytest (#14795) No new revisions were added by this update. Summary of changes: .../jenkins/job_PerformanceTests_Python.groovy | 4 +-- ...it_CrossLanguageValidatesRunner_Dataflow.groovy | 2 +- .../org/apache/beam/gradle/BeamModulePlugin.groovy | 32 ++ runners/google-cloud-dataflow-java/build.gradle| 11 .../io/external/generate_sequence_test.py | 4 +-- .../io/external/xlang_parquetio_test.py| 5 ++-- sdks/python/apache_beam/transforms/sql_test.py | 7 ++--- .../transforms/validate_runner_xlang_test.py | 12 ++-- sdks/python/pytest.ini | 2 ++ sdks/python/test-suites/dataflow/common.gradle | 7 ++--- 10 files changed, 45 insertions(+), 41 deletions(-)
[beam] branch master updated: [BEAM-9487] Disable GBK safety checks by default
This is an automated email from the ASF dual-hosted git repository. udim 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 c5a6988 [BEAM-9487] Disable GBK safety checks by default new f2bbae8 Merge pull request #15003 from zhoufek/aut c5a6988 is described below commit c5a6988835046f635aa54ce95414ab97ceb1940b Author: zhoufek AuthorDate: Fri Jun 11 15:29:02 2021 -0400 [BEAM-9487] Disable GBK safety checks by default --- CHANGES.md | 3 ++ .../python/apache_beam/options/pipeline_options.py | 3 +- sdks/python/apache_beam/transforms/core.py | 39 -- .../apache_beam/transforms/ptransform_test.py | 8 +++-- sdks/python/apache_beam/transforms/trigger.py | 6 ++-- 5 files changed, 42 insertions(+), 17 deletions(-) diff --git a/CHANGES.md b/CHANGES.md index b231074..98a5fd9 100644 --- a/CHANGES.md +++ b/CHANGES.md @@ -92,6 +92,7 @@ * `CREATE FUNCTION` DDL statement added to Calcite SQL syntax. `JAR` and `AGGREGATE` are now reserved keywords. ([BEAM-12339](https://issues.apache.org/jira/browse/BEAM-12339)). * Flink 1.13 is now supported by the Flink runner ([BEAM-12277](https://issues.apache.org/jira/browse/BEAM-12277)). +* Python `TriggerFn` has a new `may_lose_data` method to signal potential data loss. Default behavior assumes safe (necessary for backwards compatibility). See Deprecations for potential impact of overriding this. ([BEAM-9487](https://issues.apache.org/jira/browse/BEAM-9487)). * X feature added (Java/Python) ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). ## Breaking Changes @@ -107,6 +108,8 @@ ## Deprecations * X behavior is deprecated and will be removed in X versions ([BEAM-X](https://issues.apache.org/jira/browse/BEAM-X)). +* Python GBK will stop supporting unbounded PCollections that have global windowing and a default trigger in Beam 2.33. This can be overriden with `--allow_unsafe_triggers`. ([BEAM-9487](https://issues.apache.org/jira/browse/BEAM-9487)). +* Python GBK will start requiring safe triggers or the `--allow_unsafe_triggers` flag starting with Beam 2.33. ([BEAM-9487](https://issues.apache.org/jira/browse/BEAM-9487)). ## Known Issues diff --git a/sdks/python/apache_beam/options/pipeline_options.py b/sdks/python/apache_beam/options/pipeline_options.py index 9cf9eb6..5073a86 100644 --- a/sdks/python/apache_beam/options/pipeline_options.py +++ b/sdks/python/apache_beam/options/pipeline_options.py @@ -534,7 +534,8 @@ class TypeOptions(PipelineOptions): 'compatibility. See BEAM-11719.') parser.add_argument( '--allow_unsafe_triggers', -default=False, +# TODO(BEAM-9487): Set to False for Beam 2.33 +default=True, action='store_true', help='Allow the use of unsafe triggers. Unsafe triggers have the ' 'potential to cause data loss due to finishing and/or never having ' diff --git a/sdks/python/apache_beam/transforms/core.py b/sdks/python/apache_beam/transforms/core.py index acc3c70..897046d 100644 --- a/sdks/python/apache_beam/transforms/core.py +++ b/sdks/python/apache_beam/transforms/core.py @@ -2321,22 +2321,39 @@ class GroupByKey(PTransform): if not pcoll.is_bounded and isinstance( windowing.windowfn, GlobalWindows) and isinstance(trigger, DefaultTrigger): - raise ValueError( - 'GroupByKey cannot be applied to an unbounded ' + - 'PCollection with global windowing and a default trigger') - -if not pcoll.pipeline.allow_unsafe_triggers: - unsafe_reason = trigger.may_lose_data(windowing) - if unsafe_reason != DataLossReason.NO_POTENTIAL_LOSS: + if pcoll.pipeline.allow_unsafe_triggers: +# TODO(BEAM-9487) Change comment for Beam 2.33 +_LOGGER.warning( +'PCollection passed to GroupByKey (label: %s) is unbounded, has a ' +'global window, and uses a default trigger. This will no longer ' +'be allowed starting with Beam 2.33 unless ' +'--allow_unsafe_triggers is set.', +self.label) + else: +raise ValueError( +'GroupByKey cannot be applied to an unbounded ' + +'PCollection with global windowing and a default trigger') + +unsafe_reason = trigger.may_lose_data(windowing) +if unsafe_reason != DataLossReason.NO_POTENTIAL_LOSS: + if pcoll.pipeline.allow_unsafe_triggers: +# TODO(BEAM-9487): Switch back to this log for Beam 2.33. +# _LOGGER.warning( +# 'Skipping trigger safety check. ' +# 'This could lead to incomplete or missing groups.') +_LOGGER.warning( +'%s: Unsafe trigger type (%s) detected. Starting with ' +'Beam 2.33, this will raise an error by default
[beam] 01/01: Merge pull request #14975: [BEAM-12465] Fix nested subscripted Generics
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit c55f1ae581c2c7a8b772351dbfe771ac011f9842 Merge: fd9a6bd a37b04f Author: Udi Meiri AuthorDate: Wed Jun 9 14:21:10 2021 -0700 Merge pull request #14975: [BEAM-12465] Fix nested subscripted Generics .../typehints/native_type_compatibility_test.py | 13 + sdks/python/apache_beam/typehints/typehints.py | 20 +++- sdks/python/apache_beam/typehints/typehints_test.py | 9 + 3 files changed, 41 insertions(+), 1 deletion(-)
[beam] branch master updated (fd9a6bd -> c55f1ae)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from fd9a6bd Merge pull request #14957 from TheNeuralBit/interactive-beam-docs add 7e24935 [BEAM-12465] Fix nested subscripted Generics add a37b04f [BEAM-12469] Fix _unified_repr to not expect __name__ to exist. new c55f1ae Merge pull request #14975: [BEAM-12465] Fix nested subscripted Generics 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: .../typehints/native_type_compatibility_test.py | 13 + sdks/python/apache_beam/typehints/typehints.py | 20 +++- sdks/python/apache_beam/typehints/typehints_test.py | 9 + 3 files changed, 41 insertions(+), 1 deletion(-)
[beam] branch master updated: [BEAM-12434] Implement side_input for num_shards in iobase (#14916)
This is an automated email from the ASF dual-hosted git repository. udim 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 a4abc91 [BEAM-12434] Implement side_input for num_shards in iobase (#14916) a4abc91 is described below commit a4abc91436eadf78abda304664a3e0f64019ea61 Author: hoshimura AuthorDate: Fri Jun 4 19:19:20 2021 +0200 [BEAM-12434] Implement side_input for num_shards in iobase (#14916) Co-authored-by: Udi Meiri Co-authored-by: Johan Sternby --- sdks/python/apache_beam/io/iobase.py | 16 ++-- 1 file changed, 6 insertions(+), 10 deletions(-) diff --git a/sdks/python/apache_beam/io/iobase.py b/sdks/python/apache_beam/io/iobase.py index 71d8037..5d8e5df 100644 --- a/sdks/python/apache_beam/io/iobase.py +++ b/sdks/python/apache_beam/io/iobase.py @@ -1125,7 +1125,7 @@ class WriteImpl(ptransform.PTransform): if min_shards == 1: keyed_pcoll = pcoll | core.Map(lambda x: (None, x)) else: -keyed_pcoll = pcoll | core.ParDo(_RoundRobinKeyFn(min_shards)) +keyed_pcoll = pcoll | core.ParDo(_RoundRobinKeyFn(), count=min_shards) write_result_coll = ( keyed_pcoll | core.WindowInto(window.GlobalWindows()) @@ -1226,17 +1226,13 @@ def _finalize_write( class _RoundRobinKeyFn(core.DoFn): - def __init__(self, count): -# type: (int) -> None -self.count = count - def start_bundle(self): -self.counter = random.randint(0, self.count - 1) +self.counter = None - def process(self, element): -self.counter += 1 -if self.counter >= self.count: - self.counter -= self.count + def process(self, element, count): +if self.counter is None: + self.counter = random.randrange(0, count) +self.counter = (1 + self.counter) % count yield self.counter, element
[beam] branch master updated (1ffda46 -> 5dcd382)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 1ffda46 Merge pull request #14814 from emilymye/beamjava add 5dcd382 [BEAM-12284] Use the element coder when pickling beam.Create. (#14834) No new revisions were added by this update. Summary of changes: sdks/python/apache_beam/transforms/core.py| 17 +++--- sdks/python/apache_beam/transforms/create_test.py | 41 +++ 2 files changed, 53 insertions(+), 5 deletions(-)
[beam] branch master updated: [BEAM-12352] Skip GcsIOIntegrationTest.test_copy{, _batch}_rewrite_token tests
This is an automated email from the ASF dual-hosted git repository. udim 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 a9d1f3e [BEAM-12352] Skip GcsIOIntegrationTest.test_copy{,_batch}_rewrite_token tests new 40326dd Merge pull request #14851 from udim/beam-12352 a9d1f3e is described below commit a9d1f3e3c351d0567ff3fb420f0de57f1cc45da4 Author: Udi Meiri AuthorDate: Thu May 20 13:04:44 2021 -0700 [BEAM-12352] Skip GcsIOIntegrationTest.test_copy{,_batch}_rewrite_token tests These rely on a GCS feature that is not currently working. --- sdks/python/apache_beam/io/gcp/gcsio_integration_test.py | 2 ++ 1 file changed, 2 insertions(+) diff --git a/sdks/python/apache_beam/io/gcp/gcsio_integration_test.py b/sdks/python/apache_beam/io/gcp/gcsio_integration_test.py index d4a812b..5aedb6c 100644 --- a/sdks/python/apache_beam/io/gcp/gcsio_integration_test.py +++ b/sdks/python/apache_beam/io/gcp/gcsio_integration_test.py @@ -122,6 +122,7 @@ class GcsIOIntegrationTest(unittest.TestCase): self._test_copy("test_copy_kms", self.kms_key_name) @attr('IT') + @unittest.skip('BEAM-12352: enable once maxBytesRewrittenPerCall works again') def test_copy_rewrite_token(self): # Tests a multi-part copy (rewrite) operation. This is triggered by a # combination of 3 conditions: @@ -175,6 +176,7 @@ class GcsIOIntegrationTest(unittest.TestCase): self._test_copy_batch("test_copy_batch_kms", self.kms_key_name) @attr('IT') + @unittest.skip('BEAM-12352: enable once maxBytesRewrittenPerCall works again') def test_copy_batch_rewrite_token(self): # Tests a multi-part copy (rewrite) operation. This is triggered by a # combination of 3 conditions:
[beam] branch master updated (e0e3432 -> 02a2222)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from e0e3432 Merge pull request #14823: [BEAM-12342] Upgrade Spark 2 to version 2.4.8 add 02a [BEAM-3713] Move validatesRunnerBatchTests and validatesRunnerStreaming tests from nose to pytest (#14788) No new revisions were added by this update. Summary of changes: ...stCommit_Python_ValidatesRunner_Dataflow.groovy | 2 +- ...ommit_Python_ValidatesRunner_Dataflow_V2.groovy | 2 +- sdks/python/apache_beam/metrics/metric_test.py | 4 +- sdks/python/apache_beam/pipeline_test.py | 9 ++-- ...low_exercise_streaming_metrics_pipeline_test.py | 6 ++- .../portability/fn_api_runner/fn_runner_test.py| 4 +- .../portability/fn_api_runner/translations_test.py | 6 +-- .../transforms/combinefn_lifecycle_test.py | 4 +- .../apache_beam/transforms/combiners_test.py | 4 +- .../apache_beam/transforms/deduplicate_test.py | 6 ++- .../apache_beam/transforms/dofn_lifecycle_test.py | 4 +- .../apache_beam/transforms/ptransform_test.py | 26 +- .../apache_beam/transforms/sideinputs_test.py | 29 ++- sdks/python/apache_beam/transforms/util_test.py| 4 +- sdks/python/pytest.ini | 3 ++ sdks/python/test-suites/dataflow/common.gradle | 58 ++ 16 files changed, 110 insertions(+), 61 deletions(-)
[beam] branch master updated (db49dd4 -> d5b62ad)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from db49dd4 Merge pull request #14781: Set 5 second timeout for BeamFnLoggingServiceTest.testMultipleClientsFailingIsHandledGrafullyByServer add d5b62ad [BEAM-3713] Move dataflow:validatesContainerTests from nosetest to pytest (#14716) No new revisions were added by this update. Summary of changes: ...ommit_Python_ValidatesContainer_Dataflow.groovy | 2 +- .../apache_beam/examples/wordcount_it_test.py | 8 +++-- .../dataflow_exercise_metrics_pipeline_test.py | 4 ++- sdks/python/apache_beam/testing/test_pipeline.py | 13 +--- sdks/python/conftest.py| 7 sdks/python/container/run_validatescontainer.sh| 18 +- sdks/python/pytest.ini | 3 +- sdks/python/scripts/run_integration_test.sh| 39 +- 8 files changed, 67 insertions(+), 27 deletions(-)
[beam] branch master updated (d152c9c -> 46b704a)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from d152c9c Merge pull request #14646: [BEAM-12226] JdbcIO DefaultRetryStrategy: retry on PostgreSQL-specific 40P01 error code add 46b704a Clarify when to use `@DefaultCoder` (#14326) No new revisions were added by this update. Summary of changes: .../site/content/en/documentation/programming-guide.md | 17 ++--- 1 file changed, 14 insertions(+), 3 deletions(-)
[beam] branch master updated (e1ed8ec -> 38e879b)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from e1ed8ec Merge pull request #14625: [BEAM-11418][BEAM-9920] Fix Go universal runner x-lang artifact staging. add 38e879b [BEAM-3713] Moving integration tests from nose to pytest (#14481) No new revisions were added by this update. Summary of changes: .test-infra/jenkins/job_PostCommit_Python.groovy | 2 +- ...ommit_Python_ValidatesContainer_Dataflow.groovy | 2 +- ...stCommit_Python_ValidatesRunner_Dataflow.groovy | 2 +- ...ommit_Python_ValidatesRunner_Dataflow_V2.groovy | 2 +- ..._PostCommit_Python_ValidatesRunner_Flink.groovy | 4 ++ .../examples/complete/autocomplete_test.py | 4 +- .../examples/complete/game/game_stats_it_test.py | 6 +- .../complete/game/hourly_team_score_it_test.py | 6 +- .../examples/complete/game/leader_board_it_test.py | 6 +- .../examples/complete/game/user_score_it_test.py | 6 +- .../complete/juliaset/juliaset/juliaset_test_it.py | 4 +- .../cookbook/bigquery_tornadoes_it_test.py | 4 +- .../cookbook/datastore_wordcount_it_test.py| 4 +- .../apache_beam/examples/fastavro_it_test.py | 8 +-- .../streaming_wordcount_debugging_it_test.py | 4 +- .../examples/streaming_wordcount_it_test.py| 4 +- .../apache_beam/examples/wordcount_it_test.py | 11 ++-- sdks/python/apache_beam/io/fileio_test.py | 4 +- .../io/gcp/big_query_query_to_table_it_test.py | 14 ++-- .../apache_beam/io/gcp/bigquery_file_loads_test.py | 8 +-- .../apache_beam/io/gcp/bigquery_io_read_it_test.py | 6 +- .../apache_beam/io/gcp/bigquery_read_it_test.py| 12 ++-- sdks/python/apache_beam/io/gcp/bigquery_test.py| 12 ++-- .../apache_beam/io/gcp/bigquery_write_it_test.py | 12 ++-- .../gcp/datastore/v1new/datastore_write_it_test.py | 4 +- .../apache_beam/io/gcp/dicomio_integration_test.py | 6 +- .../io/gcp/experimental/spannerio_read_it_test.py | 6 +- .../io/gcp/experimental/spannerio_write_it_test.py | 8 +-- .../apache_beam/io/gcp/gcsio_integration_test.py | 14 ++-- .../apache_beam/io/gcp/pubsub_integration_test.py | 6 +- sdks/python/apache_beam/io/parquetio_it_test.py| 4 +- sdks/python/apache_beam/metrics/metric_test.py | 4 +- .../python/apache_beam/ml/gcp/cloud_dlp_it_test.py | 6 +- .../ml/gcp/naturallanguageml_test_it.py| 4 +- .../ml/gcp/videointelligenceml_test_it.py | 4 +- sdks/python/apache_beam/ml/gcp/visionml_test_it.py | 4 +- sdks/python/apache_beam/pipeline_test.py | 9 ++- .../dataflow_exercise_metrics_pipeline_test.py | 7 +- ...low_exercise_streaming_metrics_pipeline_test.py | 6 +- .../portability/fn_api_runner/fn_runner_test.py| 4 +- .../portability/fn_api_runner/translations_test.py | 6 +- sdks/python/apache_beam/testing/test_pipeline.py | 13 ++-- .../apache_beam/testing/test_stream_it_test.py | 8 +-- .../transforms/combinefn_lifecycle_test.py | 4 +- .../apache_beam/transforms/dofn_lifecycle_test.py | 4 +- .../apache_beam/transforms/external_it_test.py | 4 +- .../apache_beam/transforms/ptransform_test.py | 26 .../apache_beam/transforms/sideinputs_test.py | 29 + sdks/python/apache_beam/transforms/util_test.py| 4 +- sdks/python/conftest.py| 7 ++ sdks/python/container/run_validatescontainer.sh| 17 +++-- sdks/python/pytest.ini | 5 ++ sdks/python/scripts/run_integration_test.sh| 39 --- sdks/python/test-suites/dataflow/common.gradle | 52 --- sdks/python/test-suites/direct/common.gradle | 76 -- sdks/python/test-suites/portable/common.gradle | 19 +++--- 56 files changed, 310 insertions(+), 246 deletions(-)
[beam] branch master updated (247915c -> 6558ce3)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 247915c Merge pull request #14624 from Fixing Cloud Healthcare counter names #14233 add 6558ce3 [BEAM-10937] Tour of Beam use FileSystems for I/O (#14431) No new revisions were added by this update. Summary of changes: .../tour-of-beam/reading-and-writing-data.ipynb| 436 + 1 file changed, 265 insertions(+), 171 deletions(-)
[beam] branch master updated (7d682db -> 4732bed)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 7d682db [BEAM-12136] Warn when using a deterministic fallback coder (#14455) add 18795f8 [BEAM-12136] Add deterministic coder for enum.Enum add 4732bed Merge pull request #14532: [BEAM-12136] Add deterministic coder for enum.Enum No new revisions were added by this update. Summary of changes: sdks/python/apache_beam/coders/coder_impl.py | 13 +-- .../apache_beam/coders/coders_test_common.py | 25 -- 2 files changed, 34 insertions(+), 4 deletions(-)
[beam] branch master updated (c730223 -> 7d682db)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from c730223 Merge pull request #13492 from [BEAM-2888]-GSoD: Script to generate data for capability matrix add 7d682db [BEAM-12136] Warn when using a deterministic fallback coder (#14455) No new revisions were added by this update. Summary of changes: sdks/python/apache_beam/coders/coder_impl.pxd | 1 + sdks/python/apache_beam/coders/coder_impl.py | 10 ++ 2 files changed, 11 insertions(+)
[beam] branch master updated (b43952c -> 5474e0a)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from b43952c Merge pull request #14380 from [BEAM-10994] Java: Update V1Beta3 API and add DebugOptions add c2baacd Update dev Dataflow containers to latest version. add 5474e0a Merge pull request #14402: Update dev Dataflow containers to latest version. No new revisions were added by this update. Summary of changes: sdks/python/apache_beam/runners/dataflow/internal/names.py | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-)
[beam] branch master updated (5b105d4 -> 5356056)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 5b105d4 Merge pull request #13723 from [BEAM-11736] Propagate pipeline options to direct runner add 8078706 [BEAM-11688] Support partial proto encoding add 5356056 Merge pull request #13812: [BEAM-11688] Support partial proto encoding No new revisions were added by this update. Summary of changes: sdks/python/apache_beam/coders/coder_impl.py | 8 ++-- .../apache_beam/coders/coders_test_common.py | 53 ++ 2 files changed, 47 insertions(+), 14 deletions(-)
[beam] branch master updated (fc84d02 -> 0757359)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from fc84d02 Merge pull request #13573 from pabloem/cl151 add 4f6f923 [BEAM-10658] Update website to recommend the right version add 0757359 Merge pull request #13576: [BEAM-10658] Update website to recommend the right version No new revisions were added by this update. Summary of changes: .../www/site/content/en/documentation/io/built-in/google-bigquery.md | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-)
[beam] branch master updated (edc087e -> 44a2ac5)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from edc087e [BEAM-11312] Log cloud build url and enable kaniko cache in sdk_container_builder add 44a2ac5 [BEAM-10475] Add typehints for ShardedKeyCoder (#13474) No new revisions were added by this update. Summary of changes: sdks/python/apache_beam/coders/coders.py | 16 + .../apache_beam/coders/coders_test_common.py | 17 + .../apache_beam/typehints/sharded_key_type.py | 75 .../apache_beam/typehints/sharded_key_type_test.py | 80 ++ 4 files changed, 188 insertions(+) create mode 100644 sdks/python/apache_beam/typehints/sharded_key_type.py create mode 100644 sdks/python/apache_beam/typehints/sharded_key_type_test.py
[beam] branch master updated (ed33e8c -> 1a6b9d8)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from ed33e8c [BEAM-11207] Adding PipelineResult to session runner. (#13371) add 5f133ca [BEAM-11211] parquetio_test using multiple pyarrow versions add 1a6b9d8 Merge pull request #13369: [BEAM-11211] parquetio_test using multiple pyarrow versions No new revisions were added by this update. Summary of changes: sdks/python/test-suites/tox/common.gradle | 3 ++- sdks/python/test-suites/tox/py38/build.gradle | 7 +++ sdks/python/tox.ini | 21 + 3 files changed, 30 insertions(+), 1 deletion(-)
[beam] branch master updated (ed33e8c -> 1a6b9d8)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from ed33e8c [BEAM-11207] Adding PipelineResult to session runner. (#13371) add 5f133ca [BEAM-11211] parquetio_test using multiple pyarrow versions add 1a6b9d8 Merge pull request #13369: [BEAM-11211] parquetio_test using multiple pyarrow versions No new revisions were added by this update. Summary of changes: sdks/python/test-suites/tox/common.gradle | 3 ++- sdks/python/test-suites/tox/py38/build.gradle | 7 +++ sdks/python/tox.ini | 21 + 3 files changed, 30 insertions(+), 1 deletion(-)
[beam] branch master updated (39b95c4 -> 15cdc43)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 39b95c4 Merge pull request #13356 from [BEAM-11270] Fix container clean-up logic add 5c8a748 Update Beam Dataflow container versions for Python add 15cdc43 Merge pull request #13358: Update Beam Dataflow container versions for Python No new revisions were added by this update. Summary of changes: sdks/python/apache_beam/runners/dataflow/internal/names.py | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-)
[beam] 01/01: Revert "[BEAM-11255] Adding upper bound on urllib3 dependency"
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch revert-13321-urllib3-fix in repository https://gitbox.apache.org/repos/asf/beam.git commit 97d27f20b42d4d6883c9f4439b55635c124c78c7 Author: Udi Meiri AuthorDate: Thu Nov 12 16:56:40 2020 -0800 Revert "[BEAM-11255] Adding upper bound on urllib3 dependency" --- sdks/python/setup.py | 2 -- 1 file changed, 2 deletions(-) diff --git a/sdks/python/setup.py b/sdks/python/setup.py index 731bdd0..0afc602 100644 --- a/sdks/python/setup.py +++ b/sdks/python/setup.py @@ -153,8 +153,6 @@ REQUIRED_PACKAGES = [ 'pytz>=2018.3', 'requests>=2.24.0,<3.0.0', 'typing-extensions>=3.7.0,<3.8.0', -# TODO(BEAM-11255): urllib3 upper bound added to fix incompatibility. -'urllib3<1.26', ] # [BEAM-8181] pyarrow cannot be installed on 32-bit Windows platforms.
[beam] branch revert-13321-urllib3-fix created (now 97d27f2)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch revert-13321-urllib3-fix in repository https://gitbox.apache.org/repos/asf/beam.git. at 97d27f2 Revert "[BEAM-11255] Adding upper bound on urllib3 dependency" This branch includes the following new commits: new 97d27f2 Revert "[BEAM-11255] Adding upper bound on urllib3 dependency" 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.
[beam] branch master updated (7461918 -> 2339ffc)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 7461918 Merge pull request #13278 [BEAM-9547] stubs for non-implemented IO. add 80952c5 [BEAM-9855] Fix merge conflict between #13116 and #13240. add 2339ffc Merge pull request #13297: [BEAM-9855] Fix merge conflict between #13116 and #13240. No new revisions were added by this update. Summary of changes: .../apache/beam/runners/flink/FlinkExecutionEnvironmentsTest.java | 8 1 file changed, 4 insertions(+), 4 deletions(-)
[beam] branch master updated (7461918 -> 2339ffc)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 7461918 Merge pull request #13278 [BEAM-9547] stubs for non-implemented IO. add 80952c5 [BEAM-9855] Fix merge conflict between #13116 and #13240. add 2339ffc Merge pull request #13297: [BEAM-9855] Fix merge conflict between #13116 and #13240. No new revisions were added by this update. Summary of changes: .../apache/beam/runners/flink/FlinkExecutionEnvironmentsTest.java | 8 1 file changed, 4 insertions(+), 4 deletions(-)
[beam] branch master updated (eeb1497 -> d90f926)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from eeb1497 [BEAM-11091] Allow to specify coder for HadoopFormatIO.Read (#13166) add d90f926 [BEAM-11162] Fetch missing projectId from options (#13234) No new revisions were added by this update. Summary of changes: .../sdk/io/gcp/bigquery/DynamicDestinationsHelpers.java | 3 +++ .../beam/sdk/io/gcp/testing/FakeDatasetService.java | 3 +++ .../beam/sdk/io/gcp/bigquery/BigQueryIOWriteTest.java | 15 --- 3 files changed, 18 insertions(+), 3 deletions(-)
[beam] branch master updated (eeb1497 -> d90f926)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from eeb1497 [BEAM-11091] Allow to specify coder for HadoopFormatIO.Read (#13166) add d90f926 [BEAM-11162] Fetch missing projectId from options (#13234) No new revisions were added by this update. Summary of changes: .../sdk/io/gcp/bigquery/DynamicDestinationsHelpers.java | 3 +++ .../beam/sdk/io/gcp/testing/FakeDatasetService.java | 3 +++ .../beam/sdk/io/gcp/bigquery/BigQueryIOWriteTest.java | 15 --- 3 files changed, 18 insertions(+), 3 deletions(-)
[beam] branch master updated (376d2a6 -> 319d3ff)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 376d2a6 [BEAM-10893] Add Json support to Kafka Table Provider (#12839) add 70a3b89 [INFRA-20858] Update JDK name to match Jenkins. add 319d3ff Merge pull request #13213: [INFRA-20858] Update JDK name to match Jenkins. No new revisions were added by this update. Summary of changes: .test-infra/jenkins/CommonJobProperties.groovy | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
[beam] branch master updated (376d2a6 -> 319d3ff)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 376d2a6 [BEAM-10893] Add Json support to Kafka Table Provider (#12839) add 70a3b89 [INFRA-20858] Update JDK name to match Jenkins. add 319d3ff Merge pull request #13213: [INFRA-20858] Update JDK name to match Jenkins. No new revisions were added by this update. Summary of changes: .test-infra/jenkins/CommonJobProperties.groovy | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-)
[beam] branch master updated (29787b3 -> b138502)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 29787b3 Merge pull request #12924 from rezarokni/contextualio_rebase add b138502 Minor fixes to the get-started/wordcount-example webpage. (#12913) No new revisions were added by this update. Summary of changes: .../www/site/content/en/get-started/wordcount-example.md | 15 +-- 1 file changed, 9 insertions(+), 6 deletions(-)
[beam] branch master updated: [BEAM-7463] Fix BQ IT flake with streaming inserts (#12951)
This is an automated email from the ASF dual-hosted git repository. udim 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 fe6c2c7 [BEAM-7463] Fix BQ IT flake with streaming inserts (#12951) fe6c2c7 is described below commit fe6c2c7b49a22f71526d2f949e9ad483eb898be3 Author: Udi Meiri AuthorDate: Mon Oct 5 10:25:15 2020 -0700 [BEAM-7463] Fix BQ IT flake with streaming inserts (#12951) * [BEAM-7463] Fix BQ IT flake with streaming inserts - In short: streaming inserts are not immediately available after InsertAll RPC is done. See bug for details. - Added cumulative time limit option to retry mechanism. This may also be useful for ITs where the matcher needs to wait for a streaming pipeline to complete writing its output. * Fix lint --- .../io/gcp/big_query_query_to_table_it_test.py | 7 +++- .../apache_beam/io/gcp/tests/bigquery_matcher.py | 25 - .../io/gcp/tests/bigquery_matcher_test.py | 43 +- sdks/python/apache_beam/testing/test_utils.py | 9 ++--- sdks/python/apache_beam/utils/retry.py | 26 +++-- sdks/python/apache_beam/utils/retry_test.py| 20 ++ 6 files changed, 114 insertions(+), 16 deletions(-) diff --git a/sdks/python/apache_beam/io/gcp/big_query_query_to_table_it_test.py b/sdks/python/apache_beam/io/gcp/big_query_query_to_table_it_test.py index c806629..c20ed47 100644 --- a/sdks/python/apache_beam/io/gcp/big_query_query_to_table_it_test.py +++ b/sdks/python/apache_beam/io/gcp/big_query_query_to_table_it_test.py @@ -152,8 +152,9 @@ class BigQueryQueryToTableIT(unittest.TestCase): # handling the encoding in beam for row in table_data: row['bytes'] = base64.b64encode(row['bytes']).decode('utf-8') -self.bigquery_client.insert_rows( +passed, errors = self.bigquery_client.insert_rows( self.project, self.dataset_id, NEW_TYPES_INPUT_TABLE, table_data) +self.assertTrue(passed, 'Error in BQ setup: %s' % errors) @attr('IT') def test_big_query_legacy_sql(self): @@ -293,7 +294,9 @@ class BigQueryQueryToTableIT(unittest.TestCase): BigqueryMatcher( project=self.project, query=verify_query, -checksum=expected_checksum) +checksum=expected_checksum, +timeout_secs=30, +) ] self._setup_new_types_env() extra_opts = { diff --git a/sdks/python/apache_beam/io/gcp/tests/bigquery_matcher.py b/sdks/python/apache_beam/io/gcp/tests/bigquery_matcher.py index 1a72d72..20ca128 100644 --- a/sdks/python/apache_beam/io/gcp/tests/bigquery_matcher.py +++ b/sdks/python/apache_beam/io/gcp/tests/bigquery_matcher.py @@ -63,13 +63,16 @@ class BigqueryMatcher(BaseMatcher): Fetch Bigquery data with given query, compute a hash string and compare with expected checksum. """ - def __init__(self, project, query, checksum): + def __init__(self, project, query, checksum, timeout_secs=0): """Initialize BigQueryMatcher object. Args: project: The name (string) of the project. query: The query (string) to perform. checksum: SHA-1 hash generated from a sorted list of lines read from expected output. + timeout_secs: Duration to retry query until checksum matches. This +is useful for DF streaming pipelines or BQ streaming inserts. The +default (0) never retries. """ if bigquery is None: raise ImportError('Bigquery dependencies are not installed.') @@ -82,9 +85,16 @@ class BigqueryMatcher(BaseMatcher): self.query = query self.expected_checksum = checksum self.checksum = None +self.timeout_secs = timeout_secs def _matches(self, _): -if self.checksum is None: +@retry.with_exponential_backoff( +num_retries=1000, +initial_delay_secs=0.5, +max_delay_secs=30, +stop_after_secs=self.timeout_secs, +) +def get_checksum(): response = self._query_with_retry() _LOGGER.info( 'Read from given query (%s), total rows %d', @@ -92,6 +102,17 @@ class BigqueryMatcher(BaseMatcher): len(response)) self.checksum = compute_hash(response) _LOGGER.info('Generate checksum: %s', self.checksum) + if self.checksum != self.expected_checksum: +# This exception is never raised beyond the enclosing method. +raise ValueError( +'Checksums do not match. Expected: %s, got: %s' % +(self.expected_checksum, self.checksum)) + +if self.checksum is None: + try: +get_checksum() + except ValueError: +pass return self.checksum == self.expected_checksum diff --git a/sdks/python/apache_beam/io/gcp/tests/bigquery_matcher_test.py b/sdks/python/apache_beam/io/gcp
[beam] branch master updated (b42bcba -> d1261ad)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from b42bcba [BEAM-10960] Fix DropFields does not maintain the original fields order (#12928) add d1261ad [BEAM-6103] Enable BQ streaming insert timeouts (#12893) No new revisions were added by this update. Summary of changes: CHANGES.md | 29 ++ .../gcp/util/RetryHttpRequestInitializer.java | 1 + .../beam/sdk/io/gcp/bigquery/BigQueryOptions.java | 5 ++-- 3 files changed, 32 insertions(+), 3 deletions(-)
[beam] branch master updated (93f1076 -> 32b5518)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 93f1076 [BEAM-9547] Raise NotImplementedError and WontImplementError throughout DeferredDataframe (#12858) add 32b5518 [BEAM-4091] Pass type hints in ptransform_fn (#9907) No new revisions were added by this update. Summary of changes: CHANGES.md | 5 ++ .../python/apache_beam/options/pipeline_options.py | 43 .../options/pipeline_options_validator_test.py | 19 +++ sdks/python/apache_beam/transforms/ptransform.py | 29 +-- sdks/python/apache_beam/transforms/util.py | 2 +- sdks/python/apache_beam/typehints/decorators.py| 4 ++ .../apache_beam/typehints/typed_pipeline_test.py | 58 ++ .../typehints/typed_pipeline_test_py3.py | 32 sdks/python/conftest.py| 7 +++ .../en/documentation/sdks/python-type-safety.md| 7 +++ 10 files changed, 202 insertions(+), 4 deletions(-)
[beam] branch master updated (3d647db -> 68e7f2e)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 3d647db [BEAM-10701] Change coveralls badge to codecov (#12768) add b45a1b6 [BEAM-10701] Fix Python coverage reporting add 68e7f2e Merge pull request #12801: [BEAM-10701] Fix Python coverage reporting No new revisions were added by this update. Summary of changes: .gitignore| 2 +- sdks/python/scripts/run_pytest.sh | 11 ++- sdks/python/tox.ini | 4 +++- 3 files changed, 10 insertions(+), 7 deletions(-)
[beam] branch master updated (9befacb -> faf76d5)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 9befacb Merge pull request #12691 from lukecwik/beam10820 add 0eafc6e [BEAM-10701] Fix paths yet again add faf76d5 Merge pull request #12696: [BEAM-10701] Fix paths yet again No new revisions were added by this update. Summary of changes: .github/codecov.yml | 2 -- 1 file changed, 2 deletions(-)
[beam] branch master updated (7973f41 -> 9c4339b)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 7973f41 Merge pull request #12640 from y1chi/bundle_runner_sdf add 9c4339b [BEAM-10821] Add Python SDK typing improvements to CHANGES.md (#12693) No new revisions were added by this update. Summary of changes: CHANGES.md | 4 1 file changed, 4 insertions(+)
[beam] branch master updated (be1f175 -> eb9b93f)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from be1f175 Merge pull request #12690: [BEAM-10710] Fix author name typo for blog posts add eb9b93f [BEAM-10701] Add codecov config, fix paths hopefully (#12684) No new revisions were added by this update. Summary of changes: .../aggregation/distinct.py => .github/codecov.yml | 57 +- .gitignore | 4 +- .../{apache_beam/ml/__init__.py => .coveragerc}| 3 ++ sdks/python/tox.ini| 2 +- 4 files changed, 40 insertions(+), 26 deletions(-) copy sdks/python/apache_beam/examples/snippets/transforms/aggregation/distinct.py => .github/codecov.yml (53%) copy sdks/python/{apache_beam/ml/__init__.py => .coveragerc} (97%)
[beam] branch master updated (5d8aa2c -> be1f175)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 5d8aa2c Merge pull request #12675 from [BEAM-2855] implement query 12 add 18cea02 Fix author names add be1f175 Merge pull request #12690: [BEAM-10710] Fix author name typo for blog posts No new revisions were added by this update. Summary of changes: website/www/site/content/en/blog/python-improved-annotations.md | 2 +- .../site/content/en/blog/python-performance-runtime-type-checking.md| 2 +- 2 files changed, 2 insertions(+), 2 deletions(-)
[beam] branch master updated (a99c682 -> ab1112c)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from a99c682 Merge pull request #12663 from [BEAM-10597] Propagate BigQuery streaming insert throttled time to Dataflow worker in Python SDK add ab1112c [BEAM-10777] Add two blog posts detailing changes to the type hints module of the Python SDK (#12657) No new revisions were added by this update. Summary of changes: .../content/en/blog/python-improved-annotations.md | 110 +++ .../python-performance-runtime-type-checking.md| 154 + website/www/site/data/authors.yml | 6 +- 3 files changed, 269 insertions(+), 1 deletion(-) create mode 100644 website/www/site/content/en/blog/python-improved-annotations.md create mode 100644 website/www/site/content/en/blog/python-performance-runtime-type-checking.md
[beam] branch master updated (a99c682 -> ab1112c)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from a99c682 Merge pull request #12663 from [BEAM-10597] Propagate BigQuery streaming insert throttled time to Dataflow worker in Python SDK add ab1112c [BEAM-10777] Add two blog posts detailing changes to the type hints module of the Python SDK (#12657) No new revisions were added by this update. Summary of changes: .../content/en/blog/python-improved-annotations.md | 110 +++ .../python-performance-runtime-type-checking.md| 154 + website/www/site/data/authors.yml | 6 +- 3 files changed, 269 insertions(+), 1 deletion(-) create mode 100644 website/www/site/content/en/blog/python-improved-annotations.md create mode 100644 website/www/site/content/en/blog/python-performance-runtime-type-checking.md
[beam] branch master updated (094d06c -> 3add33e)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 094d06c Merge pull request #12541: [BEAM-10682] Add workflow to run Java tests on Linux/Windows/Mac add c168c60 Silence red PYTHONPATH warning from tox new 3add33e Merge pull request #12602: Silence red PYTHONPATH warning from tox 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: sdks/python/tox.ini | 2 ++ 1 file changed, 2 insertions(+)
[beam] 01/01: Merge pull request #12602: Silence red PYTHONPATH warning from tox
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit 3add33ea16b6fe2f7efdcd4485cc03dcb8771a77 Merge: 094d06c c168c60 Author: Udi Meiri AuthorDate: Mon Aug 24 10:21:25 2020 -0700 Merge pull request #12602: Silence red PYTHONPATH warning from tox sdks/python/tox.ini | 2 ++ 1 file changed, 2 insertions(+)
[beam] branch master updated (e6db419 -> f82ae1c)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from e6db419 [BEAM-10766] Fix flake where ByteString coercion could modify cached type hints (#12654) add 7c6dfd2 [BEAM-10751] Revert "Extending archiveJunit post-commit task with stability history" add f82ae1c Merge pull request #12664: [BEAM-10751] Revert "Extending archiveJunit post-commit task with sta… No new revisions were added by this update. Summary of changes: .test-infra/jenkins/CommonJobProperties.groovy| 8 .test-infra/jenkins/job_PerformanceTests_BigQueryIO_Java.groovy | 2 +- .test-infra/jenkins/job_PerformanceTests_Python.groovy| 2 +- .test-infra/jenkins/job_PerformanceTests_SQLIO_Java.groovy| 2 +- .../job_PostCommit_CrossLanguageValidatesRunner_Direct.groovy | 2 +- .../job_PostCommit_CrossLanguageValidatesRunner_Flink.groovy | 2 +- .../job_PostCommit_CrossLanguageValidatesRunner_Spark.groovy | 2 +- .test-infra/jenkins/job_PostCommit_Java.groovy| 2 +- .../jenkins/job_PostCommit_Java_Dataflow_Examples_Java11.groovy | 2 +- ...job_PostCommit_Java_PortableValidatesRunner_Flink_Batch.groovy | 2 +- ...PostCommit_Java_PortableValidatesRunner_Flink_Streaming.groovy | 2 +- ...job_PostCommit_Java_PortableValidatesRunner_Spark_Batch.groovy | 2 +- .../jenkins/job_PostCommit_Java_ValidatesRunner_Dataflow.groovy | 2 +- .../job_PostCommit_Java_ValidatesRunner_Dataflow_Java11.groovy| 2 +- .../jenkins/job_PostCommit_Java_ValidatesRunner_Direct.groovy | 2 +- .../job_PostCommit_Java_ValidatesRunner_Direct_Java11.groovy | 2 +- .../jenkins/job_PostCommit_Java_ValidatesRunner_Flink.groovy | 2 +- .../job_PostCommit_Java_ValidatesRunner_Flink_Java11.groovy | 2 +- .../jenkins/job_PostCommit_Java_ValidatesRunner_Samza.groovy | 2 +- .../jenkins/job_PostCommit_Java_ValidatesRunner_Spark.groovy | 2 +- ...ostCommit_Java_ValidatesRunner_SparkStructuredStreaming.groovy | 2 +- .../jenkins/job_PostCommit_Java_ValidatesRunner_Twister2.groovy | 2 +- .test-infra/jenkins/job_PostCommit_Python2.groovy | 2 +- .test-infra/jenkins/job_PostCommit_Python35.groovy| 2 +- .test-infra/jenkins/job_PostCommit_Python36.groovy| 2 +- .test-infra/jenkins/job_PostCommit_Python37.groovy| 2 +- .test-infra/jenkins/job_PostCommit_Python38.groovy| 2 +- .../job_PostCommit_Python_ValidatesContainer_Dataflow.groovy | 2 +- .../jenkins/job_PostCommit_Python_ValidatesRunner_Dataflow.groovy | 2 +- .../job_PostCommit_Python_ValidatesRunner_Dataflow_V2.groovy | 2 +- .test-infra/jenkins/job_PostCommit_SQL.groovy | 2 +- .test-infra/jenkins/job_PreCommit_Java.groovy | 3 +-- .test-infra/jenkins/job_PreCommit_Java_Examples_Dataflow.groovy | 3 +-- .../jenkins/job_PreCommit_Java_Examples_Dataflow_Java11.groovy| 2 +- .test-infra/jenkins/job_PreCommit_Java_PortabilityApi.groovy | 3 +-- .../jenkins/job_PreCommit_Java_PortabilityApi_Java11.groovy | 2 +- .test-infra/jenkins/job_PreCommit_Python.groovy | 3 +-- .test-infra/jenkins/job_PreCommit_SQL.groovy | 3 +-- .test-infra/jenkins/job_PreCommit_SQL_Java11.groovy | 2 +- .test-infra/jenkins/job_sonarqube_report.groovy | 2 +- 40 files changed, 39 insertions(+), 52 deletions(-)
[beam] branch master updated (0040865 -> 6b472e1)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 0040865 Propagate BigQuery streaming insert throttled time to Dataflow worker in Java SDK (#12403) add 2489cf1 Color Jenkins logs support add 6b472e1 Merge pull request #2: Color Jenkins logs support No new revisions were added by this update. Summary of changes: .test-infra/jenkins/CommonJobProperties.groovy | 1 + sdks/python/scripts/run_tox.sh | 4 2 files changed, 5 insertions(+)
[beam] branch master updated (d20f6fa -> 132243b)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from d20f6fa Merge pull request #12633 from ibzib/BEAM-9118 add 132243b [BEAM-10701] Generate Codecov XML (#12566) No new revisions were added by this update. Summary of changes: sdks/python/scripts/run_pytest.sh | 6 -- sdks/python/tox.ini | 2 +- 2 files changed, 5 insertions(+), 3 deletions(-)
[beam] 01/01: Merge pull request #12604: [BEAM-10697] Remove testPy2Cython from precommit
This is an automated email from the ASF dual-hosted git repository. udim pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/beam.git commit dfbc3d84a052d6f58b70f3daaa01d0df86655b0e Merge: 988b804 329ed87 Author: Udi Meiri AuthorDate: Mon Aug 17 14:24:30 2020 -0700 Merge pull request #12604: [BEAM-10697] Remove testPy2Cython from precommit sdks/python/test-suites/tox/py2/build.gradle | 1 - 1 file changed, 1 deletion(-)
[beam] branch master updated (988b804 -> dfbc3d8)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 988b804 Merge pull request #12599 from lukecwik/beam10670 add 329ed87 [BEAM-10697] Remove testPy2Cython from precommit new dfbc3d8 Merge pull request #12604: [BEAM-10697] Remove testPy2Cython from precommit 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: sdks/python/test-suites/tox/py2/build.gradle | 1 - 1 file changed, 1 deletion(-)
[beam] branch master updated (36cf935 -> 8f95ef8)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 36cf935 Merge pull request #12489 from [BEAM-6064] Add an option to avoid insert_ids on BQ in exchange for faster insertions add 8f95ef8 [BEAM-2762] Generate Python coverage reports during pre-commit (#12257) No new revisions were added by this update. Summary of changes: .test-infra/jenkins/CommonJobProperties.groovy | 1 + sdks/python/test-suites/tox/common.gradle | 10 +- sdks/python/test-suites/tox/py38/build.gradle | 2 +- sdks/python/tox.ini| 26 -- 4 files changed, 23 insertions(+), 16 deletions(-)
[beam] branch master updated (18503a6 -> 277832e)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 18503a6 [BEAM-7996] Add support for MapType and Nulls in container types for Python RowCoder (#12426) add 277832e [BEAM-10258] Support type hint annotations on PTransform's expand() (#12009) No new revisions were added by this update. Summary of changes: .../examples/snippets/snippets_test_py3.py | 12 ++ sdks/python/apache_beam/transforms/ptransform.py | 13 ++ sdks/python/apache_beam/typehints/decorators.py| 70 +++ .../typehints/typed_pipeline_test_py3.py | 130 .../apache_beam/typehints/typehints_test_py3.py| 220 - .../en/documentation/sdks/python-type-safety.md| 13 +- 6 files changed, 456 insertions(+), 2 deletions(-)
[beam] branch master updated (0ba300f -> 02092dc)
This is an automated email from the ASF dual-hosted git repository. udim pushed a change to branch master in repository https://gitbox.apache.org/repos/asf/beam.git. from 0ba300f [BEAM-10562]Eliminate nullability errors from :sdks:java:extensions:join-library (#12351) add 02092dc [BEAM-9865] Cleanup Jenkins WS on successful jobs (#12326) No new revisions were added by this update. Summary of changes: .test-infra/jenkins/CommonJobProperties.groovy | 5 + 1 file changed, 5 insertions(+)