Repository: kafka Updated Branches: refs/heads/trunk 1cf7ec87d -> 616321bcb
KAFKA-6102; Consolidate MockTime implementations between connect and clients Author: Colin P. Mccabe <cmcc...@confluent.io> Reviewers: Randall Hauch <rha...@gmail.com>, Jason Gustafson <ja...@confluent.io> Closes #4105 from cmccabe/KAFKA-6102 Project: http://git-wip-us.apache.org/repos/asf/kafka/repo Commit: http://git-wip-us.apache.org/repos/asf/kafka/commit/616321bc Tree: http://git-wip-us.apache.org/repos/asf/kafka/tree/616321bc Diff: http://git-wip-us.apache.org/repos/asf/kafka/diff/616321bc Branch: refs/heads/trunk Commit: 616321bcb618b57d063273df7764058dbd2b7a9c Parents: 1cf7ec8 Author: Colin P. Mccabe <cmcc...@confluent.io> Authored: Thu Dec 14 14:47:15 2017 -0800 Committer: Jason Gustafson <ja...@confluent.io> Committed: Thu Dec 14 14:50:34 2017 -0800 ---------------------------------------------------------------------- .../connect/runtime/ConnectMetricsTest.java | 2 +- .../connect/runtime/MockConnectMetrics.java | 6 +-- .../kafka/connect/runtime/StateTrackerTest.java | 2 +- .../connect/runtime/WorkerSinkTaskTest.java | 2 +- .../runtime/WorkerSinkTaskThreadedTest.java | 2 +- .../kafka/connect/runtime/WorkerTaskTest.java | 2 +- .../kafka/connect/runtime/WorkerTest.java | 2 +- .../kafka/connect/util/KafkaBasedLogTest.java | 3 +- .../org/apache/kafka/connect/util/MockTime.java | 55 -------------------- 9 files changed, 9 insertions(+), 67 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/kafka/blob/616321bc/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/ConnectMetricsTest.java ---------------------------------------------------------------------- diff --git a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/ConnectMetricsTest.java b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/ConnectMetricsTest.java index 2de7cb6..d496cbe 100644 --- a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/ConnectMetricsTest.java +++ b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/ConnectMetricsTest.java @@ -18,7 +18,7 @@ package org.apache.kafka.connect.runtime; import org.apache.kafka.connect.runtime.ConnectMetrics.MetricGroup; import org.apache.kafka.connect.runtime.ConnectMetrics.MetricGroupId; -import org.apache.kafka.connect.util.MockTime; +import org.apache.kafka.common.utils.MockTime; import org.junit.After; import org.junit.Before; import org.junit.Test; http://git-wip-us.apache.org/repos/asf/kafka/blob/616321bc/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/MockConnectMetrics.java ---------------------------------------------------------------------- diff --git a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/MockConnectMetrics.java b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/MockConnectMetrics.java index f1df140..3f65029 100644 --- a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/MockConnectMetrics.java +++ b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/MockConnectMetrics.java @@ -20,7 +20,7 @@ import org.apache.kafka.clients.CommonClientConfigs; import org.apache.kafka.common.MetricName; import org.apache.kafka.common.metrics.KafkaMetric; import org.apache.kafka.common.metrics.MetricsReporter; -import org.apache.kafka.connect.util.MockTime; +import org.apache.kafka.common.utils.MockTime; import java.util.HashMap; import java.util.List; @@ -56,10 +56,6 @@ public class MockConnectMetrics extends ConnectMetrics { this(new MockTime()); } - public MockConnectMetrics(org.apache.kafka.common.utils.MockTime time) { - super("mock", new WorkerConfig(WorkerConfig.baseConfigDef(), DEFAULT_WORKER_CONFIG), time); - } - public MockConnectMetrics(MockTime time) { super("mock", new WorkerConfig(WorkerConfig.baseConfigDef(), DEFAULT_WORKER_CONFIG), time); } http://git-wip-us.apache.org/repos/asf/kafka/blob/616321bc/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/StateTrackerTest.java ---------------------------------------------------------------------- diff --git a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/StateTrackerTest.java b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/StateTrackerTest.java index 7423854..2d9f02c 100644 --- a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/StateTrackerTest.java +++ b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/StateTrackerTest.java @@ -17,7 +17,7 @@ package org.apache.kafka.connect.runtime; import org.apache.kafka.connect.runtime.AbstractStatus.State; -import org.apache.kafka.connect.util.MockTime; +import org.apache.kafka.common.utils.MockTime; import org.junit.Before; import org.junit.Test; http://git-wip-us.apache.org/repos/asf/kafka/blob/616321bc/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskTest.java ---------------------------------------------------------------------- diff --git a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskTest.java b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskTest.java index 48d8740..b714dcc 100644 --- a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskTest.java +++ b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskTest.java @@ -37,7 +37,7 @@ import org.apache.kafka.connect.sink.SinkRecord; import org.apache.kafka.connect.sink.SinkTask; import org.apache.kafka.connect.storage.Converter; import org.apache.kafka.connect.util.ConnectorTaskId; -import org.apache.kafka.connect.util.MockTime; +import org.apache.kafka.common.utils.MockTime; import org.easymock.Capture; import org.easymock.CaptureType; import org.easymock.EasyMock; http://git-wip-us.apache.org/repos/asf/kafka/blob/616321bc/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskThreadedTest.java ---------------------------------------------------------------------- diff --git a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskThreadedTest.java b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskThreadedTest.java index ce29757..4b32f0c 100644 --- a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskThreadedTest.java +++ b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskThreadedTest.java @@ -35,7 +35,7 @@ import org.apache.kafka.connect.sink.SinkRecord; import org.apache.kafka.connect.sink.SinkTask; import org.apache.kafka.connect.storage.Converter; import org.apache.kafka.connect.util.ConnectorTaskId; -import org.apache.kafka.connect.util.MockTime; +import org.apache.kafka.common.utils.MockTime; import org.apache.kafka.connect.util.ThreadedTest; import org.easymock.Capture; import org.easymock.CaptureType; http://git-wip-us.apache.org/repos/asf/kafka/blob/616321bc/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerTaskTest.java ---------------------------------------------------------------------- diff --git a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerTaskTest.java b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerTaskTest.java index 516b71a..8b86dee 100644 --- a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerTaskTest.java +++ b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerTaskTest.java @@ -20,7 +20,7 @@ import org.apache.kafka.connect.errors.ConnectException; import org.apache.kafka.connect.runtime.WorkerTask.TaskMetricsGroup; import org.apache.kafka.connect.sink.SinkTask; import org.apache.kafka.connect.util.ConnectorTaskId; -import org.apache.kafka.connect.util.MockTime; +import org.apache.kafka.common.utils.MockTime; import org.easymock.EasyMock; import org.easymock.IAnswer; import org.junit.After; http://git-wip-us.apache.org/repos/asf/kafka/blob/616321bc/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerTest.java ---------------------------------------------------------------------- diff --git a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerTest.java b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerTest.java index e78ccc8..b2af1de 100644 --- a/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerTest.java +++ b/connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerTest.java @@ -42,7 +42,7 @@ import org.apache.kafka.connect.storage.OffsetBackingStore; import org.apache.kafka.connect.storage.OffsetStorageReader; import org.apache.kafka.connect.storage.OffsetStorageWriter; import org.apache.kafka.connect.util.ConnectorTaskId; -import org.apache.kafka.connect.util.MockTime; +import org.apache.kafka.common.utils.MockTime; import org.apache.kafka.connect.util.ThreadedTest; import org.easymock.Capture; import org.easymock.EasyMock; http://git-wip-us.apache.org/repos/asf/kafka/blob/616321bc/connect/runtime/src/test/java/org/apache/kafka/connect/util/KafkaBasedLogTest.java ---------------------------------------------------------------------- diff --git a/connect/runtime/src/test/java/org/apache/kafka/connect/util/KafkaBasedLogTest.java b/connect/runtime/src/test/java/org/apache/kafka/connect/util/KafkaBasedLogTest.java index b2c164d..6d5efe8 100644 --- a/connect/runtime/src/test/java/org/apache/kafka/connect/util/KafkaBasedLogTest.java +++ b/connect/runtime/src/test/java/org/apache/kafka/connect/util/KafkaBasedLogTest.java @@ -34,6 +34,7 @@ import org.apache.kafka.common.errors.WakeupException; import org.apache.kafka.common.protocol.Errors; import org.apache.kafka.common.record.TimestampType; import org.apache.kafka.common.utils.Time; +import org.apache.kafka.common.utils.MockTime; import org.easymock.Capture; import org.easymock.EasyMock; import org.junit.Before; @@ -483,4 +484,4 @@ public class KafkaBasedLogTest { return ByteBuffer.wrap(v.getBytes()); } -} \ No newline at end of file +} http://git-wip-us.apache.org/repos/asf/kafka/blob/616321bc/connect/runtime/src/test/java/org/apache/kafka/connect/util/MockTime.java ---------------------------------------------------------------------- diff --git a/connect/runtime/src/test/java/org/apache/kafka/connect/util/MockTime.java b/connect/runtime/src/test/java/org/apache/kafka/connect/util/MockTime.java deleted file mode 100644 index 554c5f9..0000000 --- a/connect/runtime/src/test/java/org/apache/kafka/connect/util/MockTime.java +++ /dev/null @@ -1,55 +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.kafka.connect.util; - -import org.apache.kafka.common.utils.Time; - -import java.util.concurrent.TimeUnit; -import java.util.concurrent.atomic.AtomicLong; - -/** - * A clock that you can manually advance by calling sleep - */ -public class MockTime implements Time { - - private final AtomicLong nanos; - - public MockTime() { - this.nanos = new AtomicLong(System.nanoTime()); - } - - @Override - public long milliseconds() { - return TimeUnit.MILLISECONDS.convert(this.nanos.get(), TimeUnit.NANOSECONDS); - } - - @Override - public long hiResClockMs() { - return TimeUnit.NANOSECONDS.toMillis(nanos.get()); - } - - @Override - public long nanoseconds() { - return nanos.get(); - } - - @Override - public void sleep(long ms) { - this.nanos.addAndGet(TimeUnit.NANOSECONDS.convert(ms, TimeUnit.MILLISECONDS)); - } - -}