[
https://issues.apache.org/jira/browse/BEAM-5875?focusedWorklogId=159674&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-159674
]
ASF GitHub Bot logged work on BEAM-5875:
----------------------------------------
Author: ASF GitHub Bot
Created on: 27/Oct/18 22:07
Start Date: 27/Oct/18 22:07
Worklog Time Spent: 10m
Work Description: kennknowles closed pull request #6858: [BEAM-5875]
Revert "[BEAM-5716] Extract MetricReader class, test it and use in Nexmark
URL: https://github.com/apache/beam/pull/6858
This is a PR merged from a forked repository.
As GitHub hides the original diff on merge, it is displayed below for
the sake of provenance:
As this is a foreign pull request (from a fork), the diff is supplied
below (as it won't show otherwise due to GitHub magic):
diff --git a/sdks/java/testing/nexmark/build.gradle
b/sdks/java/testing/nexmark/build.gradle
index 328a16912db..5ee76179be7 100644
--- a/sdks/java/testing/nexmark/build.gradle
+++ b/sdks/java/testing/nexmark/build.gradle
@@ -48,7 +48,6 @@ dependencies {
shadow project(path:
":beam-sdks-java-extensions-google-cloud-platform-core", configuration:
"shadow")
shadow project(path: ":beam-sdks-java-extensions-sql", configuration:
"shadow")
shadow project(path: ":beam-sdks-java-io-kafka", configuration: "shadow")
- shadow project(path: ":beam-sdks-java-test-utils", configuration: "shadow")
shadow library.java.google_api_services_bigquery
shadow library.java.jackson_core
shadow library.java.jackson_annotations
diff --git
a/sdks/java/testing/nexmark/src/main/java/org/apache/beam/sdk/nexmark/NexmarkLauncher.java
b/sdks/java/testing/nexmark/src/main/java/org/apache/beam/sdk/nexmark/NexmarkLauncher.java
index ed1953f819e..99d79c1bb5c 100644
---
a/sdks/java/testing/nexmark/src/main/java/org/apache/beam/sdk/nexmark/NexmarkLauncher.java
+++
b/sdks/java/testing/nexmark/src/main/java/org/apache/beam/sdk/nexmark/NexmarkLauncher.java
@@ -32,6 +32,7 @@
import java.util.Arrays;
import java.util.Collections;
import java.util.List;
+import java.util.NoSuchElementException;
import java.util.concurrent.ThreadLocalRandom;
import javax.annotation.Nullable;
import org.apache.beam.sdk.Pipeline;
@@ -42,6 +43,11 @@
import org.apache.beam.sdk.io.gcp.pubsub.PubsubIO;
import org.apache.beam.sdk.io.gcp.pubsub.PubsubMessage;
import org.apache.beam.sdk.io.kafka.KafkaIO;
+import org.apache.beam.sdk.metrics.DistributionResult;
+import org.apache.beam.sdk.metrics.MetricNameFilter;
+import org.apache.beam.sdk.metrics.MetricQueryResults;
+import org.apache.beam.sdk.metrics.MetricResult;
+import org.apache.beam.sdk.metrics.MetricsFilter;
import org.apache.beam.sdk.nexmark.NexmarkUtils.PubSubMode;
import org.apache.beam.sdk.nexmark.NexmarkUtils.SourceType;
import org.apache.beam.sdk.nexmark.model.Auction;
@@ -82,7 +88,6 @@
import org.apache.beam.sdk.nexmark.queries.sql.SqlQuery5;
import org.apache.beam.sdk.nexmark.queries.sql.SqlQuery7;
import org.apache.beam.sdk.testing.PAssert;
-import org.apache.beam.sdk.testutils.metrics.MetricsReader;
import org.apache.beam.sdk.transforms.DoFn;
import org.apache.beam.sdk.transforms.ParDo;
import org.apache.beam.sdk.util.CoderUtils;
@@ -168,6 +173,79 @@ private int maxNumWorkers() {
return 5;
}
+ /**
+ * Return the current value for a long counter, or a default value if can't
be retrieved. Note
+ * this uses only attempted metrics because some runners don't support
committed metrics.
+ */
+ private long getCounterMetric(
+ PipelineResult result, String namespace, String name, long defaultValue)
{
+ MetricQueryResults metrics =
+ result
+ .metrics()
+ .queryMetrics(
+ MetricsFilter.builder()
+ .addNameFilter(MetricNameFilter.named(namespace, name))
+ .build());
+ Iterable<MetricResult<Long>> counters = metrics.getCounters();
+ try {
+ MetricResult<Long> metricResult = counters.iterator().next();
+ return metricResult.getAttempted();
+ } catch (NoSuchElementException e) {
+ LOG.error("Failed to get metric {}, from namespace {}", name, namespace);
+ }
+ return defaultValue;
+ }
+
+ /**
+ * Return the current value for a long counter, or a default value if can't
be retrieved. Note
+ * this uses only attempted metrics because some runners don't support
committed metrics.
+ */
+ private long getDistributionMetric(
+ PipelineResult result,
+ String namespace,
+ String name,
+ DistributionType distType,
+ long defaultValue) {
+ MetricQueryResults metrics =
+ result
+ .metrics()
+ .queryMetrics(
+ MetricsFilter.builder()
+ .addNameFilter(MetricNameFilter.named(namespace, name))
+ .build());
+ Iterable<MetricResult<DistributionResult>> distributions =
metrics.getDistributions();
+ try {
+ MetricResult<DistributionResult> distributionResult =
distributions.iterator().next();
+ switch (distType) {
+ case MIN:
+ return distributionResult.getAttempted().getMin();
+ case MAX:
+ return distributionResult.getAttempted().getMax();
+ default:
+ return defaultValue;
+ }
+ } catch (NoSuchElementException e) {
+ LOG.error("Failed to get distribution metric {} for namespace {}", name,
namespace);
+ }
+ return defaultValue;
+ }
+
+ private enum DistributionType {
+ MIN,
+ MAX
+ }
+
+ /** Return the current value for a time counter, or -1 if can't be
retrieved. */
+ private long getTimestampMetric(long now, long value) {
+ // timestamp metrics are used to monitor time of execution of transforms.
+ // If result timestamp metric is too far from now, consider that metric is
erroneous
+
+ if (Math.abs(value - now) > Duration.standardDays(10000).getMillis()) {
+ return -1;
+ }
+ return value;
+ }
+
/**
* Find a 'steady state' events/sec from {@code snapshots} and store it in
{@code perf} if found.
*/
@@ -245,22 +323,69 @@ private NexmarkPerf currentPerf(
Monitor<?> resultMonitor) {
NexmarkPerf perf = new NexmarkPerf();
- MetricsReader eventMetrics = new MetricsReader(result, eventMonitor.name);
-
- long numEvents = eventMetrics.getCounterMetric(eventMonitor.prefix +
".elements", -1);
- long numEventBytes = eventMetrics.getCounterMetric(eventMonitor.prefix +
".bytes", -1);
- long eventStart = eventMetrics.getStartTimeMetric(now, eventMonitor.prefix
+ ".startTime");
- long eventEnd = eventMetrics.getEndTimeMetric(now, eventMonitor.prefix +
".endTime");
-
- MetricsReader resultMetrics = new MetricsReader(result,
resultMonitor.name);
-
- long numResults = resultMetrics.getCounterMetric(resultMonitor.prefix +
".elements", -1);
- long numResultBytes = resultMetrics.getCounterMetric(resultMonitor.prefix
+ ".bytes", -1);
- long resultStart = resultMetrics.getStartTimeMetric(now,
resultMonitor.prefix + ".startTime");
- long resultEnd = resultMetrics.getEndTimeMetric(now, resultMonitor.prefix
+ ".endTime");
+ long numEvents =
+ getCounterMetric(result, eventMonitor.name, eventMonitor.prefix +
".elements", -1);
+ long numEventBytes =
+ getCounterMetric(result, eventMonitor.name, eventMonitor.prefix +
".bytes", -1);
+ long eventStart =
+ getTimestampMetric(
+ now,
+ getDistributionMetric(
+ result,
+ eventMonitor.name,
+ eventMonitor.prefix + ".startTime",
+ DistributionType.MIN,
+ -1));
+ long eventEnd =
+ getTimestampMetric(
+ now,
+ getDistributionMetric(
+ result,
+ eventMonitor.name,
+ eventMonitor.prefix + ".endTime",
+ DistributionType.MAX,
+ -1));
+
+ long numResults =
+ getCounterMetric(result, resultMonitor.name, resultMonitor.prefix +
".elements", -1);
+ long numResultBytes =
+ getCounterMetric(result, resultMonitor.name, resultMonitor.prefix +
".bytes", -1);
+ long resultStart =
+ getTimestampMetric(
+ now,
+ getDistributionMetric(
+ result,
+ resultMonitor.name,
+ resultMonitor.prefix + ".startTime",
+ DistributionType.MIN,
+ -1));
+ long resultEnd =
+ getTimestampMetric(
+ now,
+ getDistributionMetric(
+ result,
+ resultMonitor.name,
+ resultMonitor.prefix + ".endTime",
+ DistributionType.MAX,
+ -1));
long timestampStart =
- resultMetrics.getStartTimeMetric(now, resultMonitor.prefix +
".startTimestamp");
- long timestampEnd = resultMetrics.getEndTimeMetric(now,
resultMonitor.prefix + ".endTimestamp");
+ getTimestampMetric(
+ now,
+ getDistributionMetric(
+ result,
+ resultMonitor.name,
+ resultMonitor.prefix + ".startTimestamp",
+ DistributionType.MIN,
+ -1));
+ long timestampEnd =
+ getTimestampMetric(
+ now,
+ getDistributionMetric(
+ result,
+ resultMonitor.name,
+ resultMonitor.prefix + ".endTimestamp",
+ DistributionType.MAX,
+ -1));
long effectiveEnd = -1;
if (eventEnd >= 0 && resultEnd >= 0) {
@@ -444,7 +569,7 @@ private NexmarkPerf monitor(NexmarkQuery query) {
if (options.isStreaming() && !waitingForShutdown) {
Duration quietFor = new Duration(lastActivityMsSinceEpoch, now);
- long fatalCount = new MetricsReader(job,
query.getName()).getCounterMetric("fatal", 0);
+ long fatalCount = getCounterMetric(job, query.getName(), "fatal", 0);
if (fatalCount > 0) {
NexmarkUtils.console("job has fatal errors, cancelling.");
errors.add(String.format("Pipeline reported %s fatal errors",
fatalCount));
diff --git a/sdks/java/testing/test-utils/build.gradle
b/sdks/java/testing/test-utils/build.gradle
index c76591f2141..6617e9f3b44 100644
--- a/sdks/java/testing/test-utils/build.gradle
+++ b/sdks/java/testing/test-utils/build.gradle
@@ -19,14 +19,4 @@
apply plugin: org.apache.beam.gradle.BeamModulePlugin
applyJavaNature()
-description = "Apache Beam :: SDKs :: Java :: Test Utils"
-
-dependencies {
- shadow project(path: ":beam-sdks-java-core", configuration: "shadow")
-
- shadowTest library.java.junit
- shadowTest library.java.mockito_core
- shadowTest library.java.hamcrest_core
- shadowTest library.java.hamcrest_library
- shadowTest project(path: ":beam-runners-direct-java", configuration:
"shadowTest")
-}
+description = "Apache Beam :: SDKs :: Java :: Test Utils"
\ No newline at end of file
diff --git
a/sdks/java/testing/test-utils/src/main/java/org/apache/beam/sdk/testutils/metrics/MetricsReader.java
b/sdks/java/testing/test-utils/src/main/java/org/apache/beam/sdk/testutils/metrics/MetricsReader.java
deleted file mode 100644
index 77dccc03261..00000000000
---
a/sdks/java/testing/test-utils/src/main/java/org/apache/beam/sdk/testutils/metrics/MetricsReader.java
+++ /dev/null
@@ -1,142 +0,0 @@
-/*
- * 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 implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-package org.apache.beam.sdk.testutils.metrics;
-
-import com.google.common.base.Preconditions;
-import com.google.common.collect.Iterables;
-import java.util.NoSuchElementException;
-import org.apache.beam.sdk.PipelineResult;
-import org.apache.beam.sdk.metrics.DistributionResult;
-import org.apache.beam.sdk.metrics.MetricNameFilter;
-import org.apache.beam.sdk.metrics.MetricQueryResults;
-import org.apache.beam.sdk.metrics.MetricResult;
-import org.apache.beam.sdk.metrics.MetricsFilter;
-import org.joda.time.Duration;
-import org.slf4j.LoggerFactory;
-
-/** Provides methods for querying metrics from {@link PipelineResult} per
namespace. */
-public class MetricsReader {
-
- private static final org.slf4j.Logger LOG =
LoggerFactory.getLogger(MetricsReader.class);
-
- private enum DistributionType {
- MIN,
- MAX
- }
-
- private final PipelineResult result;
-
- private final String namespace;
-
- public MetricsReader(PipelineResult result, String namespace) {
- this.result = result;
- this.namespace = namespace;
- }
-
- /**
- * Return the current value for a long counter, or a default value if can't
be retrieved. Note
- * this uses only attempted metrics because some runners don't support
committed metrics.
- */
- public long getCounterMetric(String name, long defaultValue) {
- MetricQueryResults metrics =
- result
- .metrics()
- .queryMetrics(
- MetricsFilter.builder()
- .addNameFilter(MetricNameFilter.named(namespace, name))
- .build());
- Iterable<MetricResult<Long>> counters = metrics.getCounters();
-
- checkIfMetricResultIsUnique(name, counters);
-
- try {
- MetricResult<Long> metricResult = counters.iterator().next();
- return metricResult.getAttempted();
- } catch (NoSuchElementException e) {
- LOG.error("Failed to get metric {}, from namespace {}", name, namespace);
- }
- return defaultValue;
- }
-
- /**
- * Return start time metric by counting the difference between "now" and min
value from a
- * distribution metric.
- */
- public long getStartTimeMetric(long now, String name) {
- return this.getTimestampMetric(now, this.getDistributionMetric(name,
DistributionType.MIN, -1));
- }
-
- /**
- * Return end time metric by counting the difference between "now" and MAX
value from a
- * distribution metric.
- */
- public long getEndTimeMetric(long now, String name) {
- return this.getTimestampMetric(now, this.getDistributionMetric(name,
DistributionType.MAX, -1));
- }
-
- /**
- * Return the current value for a long counter, or a default value if can't
be retrieved. Note
- * this uses only attempted metrics because some runners don't support
committed metrics.
- */
- private long getDistributionMetric(String name, DistributionType distType,
long defaultValue) {
- MetricQueryResults metrics =
- result
- .metrics()
- .queryMetrics(
- MetricsFilter.builder()
- .addNameFilter(MetricNameFilter.named(namespace, name))
- .build());
- Iterable<MetricResult<DistributionResult>> distributions =
metrics.getDistributions();
-
- checkIfMetricResultIsUnique(name, distributions);
-
- try {
- MetricResult<DistributionResult> distributionResult =
distributions.iterator().next();
- switch (distType) {
- case MIN:
- return distributionResult.getAttempted().getMin();
- case MAX:
- return distributionResult.getAttempted().getMax();
- default:
- return defaultValue;
- }
- } catch (NoSuchElementException e) {
- LOG.error("Failed to get distribution metric {} for namespace {}", name,
namespace);
- }
- return defaultValue;
- }
-
- private <T> void checkIfMetricResultIsUnique(String name,
Iterable<MetricResult<T>> metricResult)
- throws IllegalStateException {
-
- Preconditions.checkState(
- Iterables.size(metricResult) == 1,
- String.format("More than one metric matches name: %s in namespace
%s.", name, namespace));
- }
-
- /** Return the current value for a time counter, or -1 if can't be
retrieved. */
- private long getTimestampMetric(long now, long value) {
- // timestamp metrics are used to monitor time of execution of transforms.
- // If result timestamp metric is too far from now, consider that metric is
erroneous
-
- if (Math.abs(value - now) > Duration.standardDays(10000).getMillis()) {
- return -1;
- }
- return value;
- }
-}
diff --git
a/sdks/java/testing/test-utils/src/main/java/org/apache/beam/sdk/testutils/metrics/package-info.java
b/sdks/java/testing/test-utils/src/main/java/org/apache/beam/sdk/testutils/metrics/package-info.java
deleted file mode 100644
index c171bee8d3a..00000000000
---
a/sdks/java/testing/test-utils/src/main/java/org/apache/beam/sdk/testutils/metrics/package-info.java
+++ /dev/null
@@ -1,19 +0,0 @@
-/*
- * 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 implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-/** Metrics related testing utilities for Java SDK. */
-package org.apache.beam.sdk.testutils.metrics;
diff --git
a/sdks/java/testing/test-utils/src/test/java/org/apache/beam/sdk/testutils/metrics/MetricsReaderTest.java
b/sdks/java/testing/test-utils/src/test/java/org/apache/beam/sdk/testutils/metrics/MetricsReaderTest.java
deleted file mode 100644
index 521317f2665..00000000000
---
a/sdks/java/testing/test-utils/src/test/java/org/apache/beam/sdk/testutils/metrics/MetricsReaderTest.java
+++ /dev/null
@@ -1,128 +0,0 @@
-/*
- * 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 implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-package org.apache.beam.sdk.testutils.metrics;
-
-import static org.junit.Assert.assertEquals;
-
-import java.util.Arrays;
-import java.util.List;
-import org.apache.beam.sdk.PipelineResult;
-import org.apache.beam.sdk.metrics.Counter;
-import org.apache.beam.sdk.metrics.Distribution;
-import org.apache.beam.sdk.metrics.Metrics;
-import org.apache.beam.sdk.testing.TestPipeline;
-import org.apache.beam.sdk.transforms.Create;
-import org.apache.beam.sdk.transforms.DoFn;
-import org.apache.beam.sdk.transforms.ParDo;
-import org.junit.Rule;
-import org.junit.Test;
-import org.junit.runner.RunWith;
-import org.junit.runners.JUnit4;
-
-/** Tests for {@link MetricsReader}. */
-@RunWith(JUnit4.class)
-public class MetricsReaderTest {
-
- @Rule public TestPipeline testPipeline = TestPipeline.create();
-
- private static final String NAMESPACE = "Testing";
-
- @Test
- public void testCounterMetricReceivedFromPipelineResult() {
- List<Integer> sampleInputData = Arrays.asList(1, 1, 1, 1, 1);
-
- createTestPipeline(sampleInputData, new MonitorWithCounter());
- PipelineResult result = testPipeline.run();
-
- MetricsReader reader = new MetricsReader(result, NAMESPACE);
-
- assertEquals(5, reader.getCounterMetric("counter", -1));
- }
-
- @Test
- public void testStartTimeIsTheMinimumOfTheDistribution() {
- List<Integer> sampleInputData = Arrays.asList(1, 2, 3, 4, 5);
-
- createTestPipeline(sampleInputData, new MonitorWithTimeDistribution());
- PipelineResult result = testPipeline.run();
-
- MetricsReader reader = new MetricsReader(result, NAMESPACE);
-
- assertEquals(1, reader.getStartTimeMetric(0, "timeDist"));
- }
-
- @Test
- public void testEndTimeIsTheMaximumOfTheDistribution() {
- List<Integer> sampleInputData = Arrays.asList(1, 2, 3, 4, 5);
-
- createTestPipeline(sampleInputData, new MonitorWithTimeDistribution());
-
- PipelineResult result = testPipeline.run();
-
- MetricsReader reader = new MetricsReader(result, NAMESPACE);
-
- assertEquals(5, reader.getEndTimeMetric(0, "timeDist"));
- }
-
- @Test(expected = IllegalStateException.class)
- public void
throwsIllegalStateExceptionWhenThereAreMultipleCountersOfTheSameNameAndType() {
- Metrics.counter(NAMESPACE, "counter");
- Metrics.counter(NAMESPACE, "counter");
-
- PipelineResult result = testPipeline.run();
- MetricsReader reader = new MetricsReader(result, NAMESPACE);
- reader.getCounterMetric("counter", -1);
- }
-
- @Test
- public void testTimeIsMinusOneIfTimeMetricIsTooFarFromNow() {
- List<Integer> sampleInputData = Arrays.asList(1, 5, 5, 5, 5);
-
- createTestPipeline(sampleInputData, new MonitorWithTimeDistribution());
- PipelineResult result = testPipeline.run();
-
- MetricsReader reader = new MetricsReader(result, NAMESPACE);
-
- assertEquals(-1, reader.getStartTimeMetric(900000000001L, "timeDist"));
- assertEquals(-1, reader.getEndTimeMetric(900000000001L, "timeDist"));
- }
-
- private void createTestPipeline(List<Integer> sampleInputData, DoFn<Integer,
Integer> monitor) {
- testPipeline.apply(Create.of(sampleInputData)).apply(ParDo.of(monitor));
- }
-
- /** Counts total elements of the input data provided. */
- private static class MonitorWithCounter extends DoFn<Integer, Integer> {
- private final Counter elementCounter = Metrics.counter(NAMESPACE,
"counter");
-
- @ProcessElement
- public void processElement(ProcessContext c) {
- elementCounter.inc();
- }
- }
-
- /** Simulates time flow by updating the distribution metric with input
collection elements. */
- private static class MonitorWithTimeDistribution extends DoFn<Integer,
Integer> {
- private final Distribution timeDistribution =
Metrics.distribution(NAMESPACE, "timeDist");
-
- @ProcessElement
- public void processElement(ProcessContext c) {
- timeDistribution.update(c.element().longValue());
- }
- }
-}
----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
For queries about this service, please contact Infrastructure at:
[email protected]
Issue Time Tracking
-------------------
Worklog Id: (was: 159674)
Time Spent: 2h (was: 1h 50m)
> Nexmark perf tests fail due to NoClassDefFoundError for Iterables.
> ------------------------------------------------------------------
>
> Key: BEAM-5875
> URL: https://issues.apache.org/jira/browse/BEAM-5875
> Project: Beam
> Issue Type: Bug
> Components: test-failures
> Reporter: Henning Rohde
> Assignee: Kenneth Knowles
> Priority: Critical
> Labels: currently-failing
> Time Spent: 2h
> Remaining Estimate: 0h
>
> https://scans.gradle.com/s/vjkiys2xc3age/console-log?task=:beam-sdks-java-nexmark:run
> I see:
> Caused by: java.lang.NoClassDefFoundError:
> org/apache/beam/repackaged/beam_sdks_java_test_utils/com/google/common/collect/Iterables
> at
> org.apache.beam.sdk.testutils.metrics.MetricsReader.checkIfMetricResultIsUnique(MetricsReader.java:128)
> at
> org.apache.beam.sdk.testutils.metrics.MetricsReader.getCounterMetric(MetricsReader.java:65)
> at
> org.apache.beam.sdk.nexmark.NexmarkLauncher.currentPerf(NexmarkLauncher.java:250)
> at
> org.apache.beam.sdk.nexmark.NexmarkLauncher.monitor(NexmarkLauncher.java:435)
> at
> org.apache.beam.sdk.nexmark.NexmarkLauncher.run(NexmarkLauncher.java:1156)
> at org.apache.beam.sdk.nexmark.Main$Run.call(Main.java:108)
> at org.apache.beam.sdk.nexmark.Main$Run.call(Main.java:96)
> at java.util.concurrent.FutureTask.run(FutureTask.java:266)
> at
> java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:511)
> at java.util.concurrent.FutureTask.run(FutureTask.java:266)
> at
> java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1149)
> at
> java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:624)
> at java.lang.Thread.run(Thread.java:748)
> Caused by: java.lang.ClassNotFoundException:
> org.apache.beam.repackaged.beam_sdks_java_test_utils.com.google.common.collect.Iterables
> at java.net.URLClassLoader.findClass(URLClassLoader.java:381)
> at java.lang.ClassLoader.loadClass(ClassLoader.java:424)
> at sun.misc.Launcher$AppClassLoader.loadClass(Launcher.java:349)
> at java.lang.ClassLoader.loadClass(ClassLoader.java:357)
> ... 13 more
> PRs for the first red run:
> [BEAM-5716] Move nexmark to "testing" directory in java sdk (commit: 0074138)
> (detail / githubweb)
> [BEAM-5716] Move load-tests code to "testing" directory in java sdk (commit:
> 6674c9d) (detail / githubweb)
> [BEAM-5716] Create module for testing utils (commit: 0628951) (detail /
> githubweb)
> [BEAM-5716] Extract MetricReader class, test it and use in Nexmark code
> (commit: 69730fc) (detail / githubweb)
> [BEAM-5355] Use MetricsReader in GroupByKeyLoadTest (commit: 7374eb6) (detail
> / githubweb)
> Ćukasz -- would you mind taking a look? Looks like a shading issue.
--
This message was sent by Atlassian JIRA
(v7.6.3#76005)