Skip to content

Commit a9fcaaf

Browse files
authored
Skip BoundedTrie on Dataflow till service is have BoundedTrie (#33921) (#33947)
* Temporarily stop publishing BoundedTrie metrics till it is supported in Dataflow. * Exclude UsesBoundedTrieMetrics from Dataflow runner * Fix failing test * Trivial change to postcommit to trigger it
1 parent c5b4db0 commit a9fcaaf

File tree

5 files changed

+27
-7
lines changed

5 files changed

+27
-7
lines changed

.github/trigger_files/beam_PostCommit_Java_ValidatesRunner_Dataflow.json

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -2,5 +2,6 @@
22
"comment": "Modify this file in a trivial way to cause this test suite to run",
33
"https://github.com/apache/beam/pull/31156": "noting that PR #31156 should run this test",
44
"https://github.com/apache/beam/pull/31268": "noting that PR #31268 should run this test",
5-
"https://github.com/apache/beam/pull/31490": "noting that PR #31490 should run this test"
5+
"https://github.com/apache/beam/pull/31490": "noting that PR #31490 should run this test",
6+
"https://github.com/apache/beam/pull/33921": "noting that PR #33921 should run this test"
67
}

runners/google-cloud-dataflow-java/build.gradle

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -191,12 +191,12 @@ def commonLegacyExcludeCategories = [
191191
'org.apache.beam.sdk.testing.UsesParDoLifecycle', // doesn't support remote runner
192192
'org.apache.beam.sdk.testing.UsesMetricsPusher',
193193
'org.apache.beam.sdk.testing.UsesBundleFinalizer',
194+
'org.apache.beam.sdk.testing.UsesBoundedTrieMetrics',
194195
]
195196

196197
def commonRunnerV2ExcludeCategories = [
197198
'org.apache.beam.sdk.testing.UsesExternalService',
198199
'org.apache.beam.sdk.testing.UsesGaugeMetrics',
199-
'org.apache.beam.sdk.testing.UsesStringSetMetrics',
200200
'org.apache.beam.sdk.testing.UsesSetState',
201201
'org.apache.beam.sdk.testing.UsesMapState',
202202
'org.apache.beam.sdk.testing.UsesMultimapState',
@@ -205,6 +205,7 @@ def commonRunnerV2ExcludeCategories = [
205205
'org.apache.beam.sdk.testing.UsesTestStream',
206206
'org.apache.beam.sdk.testing.UsesTestStreamWithProcessingTime',
207207
'org.apache.beam.sdk.testing.UsesRequiresTimeSortedInput',
208+
'org.apache.beam.sdk.testing.UsesBoundedTrieMetrics',
208209
]
209210

210211
// For the following test tasks using legacy worker, set workerHarnessContainerImage to empty to

runners/google-cloud-dataflow-java/worker/src/main/java/org/apache/beam/runners/dataflow/worker/BatchModeExecutionContext.java

Lines changed: 15 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -19,6 +19,7 @@
1919

2020
import com.google.api.services.dataflow.model.CounterUpdate;
2121
import com.google.api.services.dataflow.model.SideInputInfo;
22+
import java.util.Collections;
2223
import java.util.Objects;
2324
import java.util.concurrent.TimeUnit;
2425
import org.apache.beam.runners.core.InMemoryStateInternals;
@@ -77,14 +78,18 @@ public class BatchModeExecutionContext
7778
protected static final String BIGQUERY_READ_THROTTLE_TIME_NAMESPACE =
7879
"org.apache.beam.sdk.io.gcp.bigquery.BigQueryServicesImpl$StorageClientImpl";
7980

81+
// TODO(BEAM-33720): Remove once Dataflow legacy runner supports BoundedTries.
82+
private final boolean populateBoundedTrieMetrics;
83+
8084
private BatchModeExecutionContext(
8185
CounterFactory counterFactory,
8286
Cache<?, WeightedValue<?>> dataCache,
8387
Cache<?, ?> logicalReferenceCache,
8488
ReaderFactory readerFactory,
8589
PipelineOptions options,
8690
DataflowExecutionStateTracker executionStateTracker,
87-
DataflowExecutionStateRegistry executionStateRegistry) {
91+
DataflowExecutionStateRegistry executionStateRegistry,
92+
boolean populateBoundedTrieMetrics) {
8893
super(
8994
counterFactory,
9095
createMetricsContainerRegistry(),
@@ -97,6 +102,7 @@ private BatchModeExecutionContext(
97102
this.dataCache = dataCache;
98103
this.containerRegistry =
99104
(MetricsContainerRegistry<MetricsContainerImpl>) getMetricsContainerRegistry();
105+
this.populateBoundedTrieMetrics = populateBoundedTrieMetrics;
100106
}
101107

102108
private static MetricsContainerRegistry<MetricsContainerImpl> createMetricsContainerRegistry() {
@@ -132,7 +138,8 @@ public static BatchModeExecutionContext forTesting(
132138
counterFactory,
133139
options,
134140
"test-work-item-id"),
135-
stateRegistry);
141+
stateRegistry,
142+
true);
136143
}
137144

138145
public static BatchModeExecutionContext forTesting(PipelineOptions options, String stageName) {
@@ -245,7 +252,8 @@ public static BatchModeExecutionContext create(
245252
counterFactory,
246253
options,
247254
workItemId),
248-
executionStateRegistry);
255+
executionStateRegistry,
256+
false);
249257
}
250258

251259
/** Create a new {@link StepContext}. */
@@ -520,7 +528,10 @@ public Iterable<CounterUpdate> extractMetricUpdates(boolean isFinalUpdate) {
520528
update ->
521529
MetricsToCounterUpdateConverter.fromStringSet(
522530
update.getKey(), true, update.getUpdate())),
523-
FluentIterable.from(updates.boundedTrieUpdates())
531+
FluentIterable.from(
532+
populateBoundedTrieMetrics
533+
? updates.boundedTrieUpdates()
534+
: Collections.emptyList())
524535
.transform(
525536
update ->
526537
MetricsToCounterUpdateConverter.fromBoundedTrie(

runners/google-cloud-dataflow-java/worker/src/main/java/org/apache/beam/runners/dataflow/worker/StreamingStepMetricsContainer.java

Lines changed: 5 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -23,6 +23,7 @@
2323
import java.time.Clock;
2424
import java.time.Duration;
2525
import java.time.Instant;
26+
import java.util.Collections;
2627
import java.util.HashSet;
2728
import java.util.Map;
2829
import java.util.Map.Entry;
@@ -99,6 +100,9 @@ public class StreamingStepMetricsContainer implements MetricsContainer {
99100

100101
private final Clock clock;
101102

103+
// TODO(BEAM-33720): Remove once Dataflow legacy runner supports BoundedTries.
104+
@VisibleForTesting boolean populateBoundedTrieMetrics;
105+
102106
private StreamingStepMetricsContainer(String stepName) {
103107
this.stepName = stepName;
104108
this.perWorkerCountersByFirstStaleTime = new ConcurrentHashMap<>();
@@ -219,7 +223,7 @@ public Iterable<CounterUpdate> extractUpdates() {
219223
.append(distributionUpdates())
220224
.append(gaugeUpdates())
221225
.append(stringSetUpdates())
222-
.append(boundedTrieUpdates());
226+
.append(populateBoundedTrieMetrics ? boundedTrieUpdates() : Collections.emptyList());
223227
}
224228

225229
private FluentIterable<CounterUpdate> counterUpdates() {

runners/google-cloud-dataflow-java/worker/src/test/java/org/apache/beam/runners/dataflow/worker/StreamingStepMetricsContainerTest.java

Lines changed: 3 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -357,6 +357,7 @@ public void testBoundedTrieUpdateExtraction() {
357357
.setBoundedTrie(
358358
MetricsToCounterUpdateConverter.getBoundedTrie(expectedName1.toProto()));
359359

360+
((StreamingStepMetricsContainer) c1).populateBoundedTrieMetrics = true;
360361
Iterable<CounterUpdate> updates = StreamingStepMetricsContainer.extractMetricUpdates(registry);
361362
assertThat(updates, containsInAnyOrder(name1Update));
362363

@@ -385,6 +386,7 @@ public void testBoundedTrieUpdateExtraction() {
385386
.setBoundedTrie(
386387
MetricsToCounterUpdateConverter.getBoundedTrie(expectedName2.toProto()));
387388

389+
((StreamingStepMetricsContainer) c2).populateBoundedTrieMetrics = true;
388390
updates = StreamingStepMetricsContainer.extractMetricUpdates(registry);
389391
assertThat(updates, containsInAnyOrder(name2Update));
390392

@@ -396,6 +398,7 @@ public void testBoundedTrieUpdateExtraction() {
396398
name1Update.setBoundedTrie(
397399
MetricsToCounterUpdateConverter.getBoundedTrie(expectedName1.toProto()));
398400

401+
((StreamingStepMetricsContainer) c1).populateBoundedTrieMetrics = true;
399402
updates = StreamingStepMetricsContainer.extractMetricUpdates(registry);
400403
assertThat(updates, containsInAnyOrder(name1Update));
401404
}

0 commit comments

Comments
 (0)