HBASE-2578 Add ability for tests to override server-side timestamp setting (currentTimeMillis)

git-svn-id: https://svn.apache.org/repos/asf/hbase/trunk@952479 13f79535-47bb-0310-9956-ffa450edef68
This commit is contained in:
Ryan Rawson 2010-06-07 23:06:24 +00:00
parent 304464ee06
commit a01c22af11
12 changed files with 426 additions and 49 deletions

View File

@ -671,6 +671,8 @@ Release 0.21.0 - Unreleased
failing hudson on occasion)
HBASE-2651 Allow alternate column separators to be specified for ImportTsv
HBASE-2661 Add test case for row atomicity guarantee
HBASE-2578 Add ability for tests to override server-side timestamp
setting (currentTimeMillis) (Daniel Ploeg via Ryan Rawson)
NEW FEATURES
HBASE-1961 HBase EC2 scripts

View File

@ -761,7 +761,7 @@
$ mvn deploy
(or)
$ mvn -s /my/path/settings.xml deploy
$ mvn -s /my/path/settings.xml deploy
-->
<distributionManagement>

View File

@ -52,6 +52,7 @@ import org.apache.hadoop.hbase.regionserver.wal.HLog;
import org.apache.hadoop.hbase.regionserver.wal.WALEdit;
import org.apache.hadoop.hbase.util.Bytes;
import org.apache.hadoop.hbase.util.ClassSize;
import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
import org.apache.hadoop.hbase.util.FSUtils;
import org.apache.hadoop.hbase.util.Writables;
import org.apache.hadoop.io.Writable;
@ -358,7 +359,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
// HRegion is ready to go!
this.writestate.compacting = false;
this.lastFlushTime = System.currentTimeMillis();
this.lastFlushTime = EnvironmentEdgeManager.currentTimeMillis();
LOG.info("region " + this +
" available; sequence id is " + this.minSequenceId);
}
@ -665,7 +666,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
}
// Calculate regionid to use. Can't be less than that of parent else
// it'll insert into wrong location over in .META. table: HBASE-710.
long rid = System.currentTimeMillis();
long rid = EnvironmentEdgeManager.currentTimeMillis();
if (rid < this.regionInfo.getRegionId()) {
LOG.warn("Clock skew; parent regions id is " +
this.regionInfo.getRegionId() + " but current time here is " + rid);
@ -830,7 +831,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
}
LOG.info("Starting" + (majorCompaction? " major " : " ") +
"compaction on region " + this);
long startTime = System.currentTimeMillis();
long startTime = EnvironmentEdgeManager.currentTimeMillis();
doRegionCompactionPrep();
long maxSize = -1;
for (Store store: stores.values()) {
@ -841,7 +842,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
}
}
doRegionCompactionCleanup();
String timeTaken = StringUtils.formatTimeDiff(System.currentTimeMillis(),
String timeTaken = StringUtils.formatTimeDiff(EnvironmentEdgeManager.currentTimeMillis(),
startTime);
LOG.info("compaction completed on region " + this + " in " + timeTaken);
} finally {
@ -943,7 +944,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
* because a Snapshot was not properly persisted.
*/
protected boolean internalFlushcache() throws IOException {
final long startTime = System.currentTimeMillis();
final long startTime = EnvironmentEdgeManager.currentTimeMillis();
// Clear flush flag.
// Record latest flush time
this.lastFlushTime = startTime;
@ -1082,7 +1083,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
}
if (LOG.isDebugEnabled()) {
long now = System.currentTimeMillis();
long now = EnvironmentEdgeManager.currentTimeMillis();
LOG.debug("Finished memstore flush of ~" +
StringUtils.humanReadableInt(currentMemStoreSize) + " for region " +
this + " in " + (now - startTime) + "ms, sequence id=" + sequenceId +
@ -1268,7 +1269,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
*/
public void delete(Map<byte[], List<KeyValue>> familyMap, boolean writeToWAL)
throws IOException {
long now = System.currentTimeMillis();
long now = EnvironmentEdgeManager.currentTimeMillis();
byte [] byteNow = Bytes.toBytes(now);
boolean flush = false;
@ -1429,7 +1430,6 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
// If we did not pass an existing row lock, obtain a new one
Integer lid = getLock(lockid, row);
byte [] now = Bytes.toBytes(System.currentTimeMillis());
try {
// All edits for the given row (across all column families) must happen atomically.
put(put.getFamilyMap(), writeToWAL);
@ -1476,8 +1476,6 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
checkFamily(family);
get.addColumn(family, qualifier);
byte [] now = Bytes.toBytes(System.currentTimeMillis());
// Lock row
Integer lid = getLock(lockId, get.getRow());
List<KeyValue> result = new ArrayList<KeyValue>();
@ -1627,7 +1625,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
*/
private void put(final Map<byte [], List<KeyValue>> familyMap,
boolean writeToWAL) throws IOException {
long now = System.currentTimeMillis();
long now = EnvironmentEdgeManager.currentTimeMillis();
byte[] byteNow = Bytes.toBytes(now);
boolean flush = false;
this.updatesLock.readLock().lock();
@ -1899,7 +1897,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
}
}
}
public void bulkLoadHFile(String hfilePath, byte[] familyName)
throws IOException {
splitsAndClosesLock.readLock().lock();
@ -1913,7 +1911,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
} finally {
splitsAndClosesLock.readLock().unlock();
}
}
@ -2256,7 +2254,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
try {
List<KeyValue> edits = new ArrayList<KeyValue>();
edits.add(new KeyValue(row, CATALOG_FAMILY, REGIONINFO_QUALIFIER,
System.currentTimeMillis(), Writables.getBytes(r.getRegionInfo())));
EnvironmentEdgeManager.currentTimeMillis(), Writables.getBytes(r.getRegionInfo())));
meta.put(HConstants.CATALOG_FAMILY, edits);
} finally {
meta.releaseRowLock(lid);
@ -2692,12 +2690,12 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
// bulid the KeyValue now:
KeyValue newKv = new KeyValue(row, family,
qualifier, System.currentTimeMillis(),
qualifier, EnvironmentEdgeManager.currentTimeMillis(),
Bytes.toBytes(result));
// now log it:
if (writeToWAL) {
long now = System.currentTimeMillis();
long now = EnvironmentEdgeManager.currentTimeMillis();
WALEdit walEdit = new WALEdit();
walEdit.add(newKv);
this.log.append(regionInfo, regionInfo.getTableDesc().getName(),
@ -2908,7 +2906,7 @@ public class HRegion implements HConstants, HeapSize { // , Writable{
Configuration c = HBaseConfiguration.create();
FileSystem fs = FileSystem.get(c);
Path logdir = new Path(c.get("hbase.tmp.dir"),
"hlog" + tableDir.getName() + System.currentTimeMillis());
"hlog" + tableDir.getName() + EnvironmentEdgeManager.currentTimeMillis());
Path oldLogDir = new Path(c.get("hbase.tmp.dir"), HREGION_OLDLOGDIR_NAME);
HLog log = new HLog(fs, logdir, oldLogDir, c, null);
try {

View File

@ -0,0 +1,37 @@
/*
* Copyright 2010 The Apache Software Foundation
*
* 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.hadoop.hbase.util;
/**
* Default implementation of an environment edge.
*/
public class DefaultEnvironmentEdge implements EnvironmentEdge {
/**
* {@inheritDoc}
* <p/>
* This implementation returns {@link System#currentTimeMillis()}
*/
@Override
public long currentTimeMillis() {
return System.currentTimeMillis();
}
}

View File

@ -0,0 +1,36 @@
/*
* Copyright 2010 The Apache Software Foundation
*
* 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.hadoop.hbase.util;
/**
* Has some basic interaction with the environment. Alternate implementations
* can be used where required (eg in tests).
*
* @see EnvironmentEdgeManager
*/
public interface EnvironmentEdge {
/**
* Returns the currentTimeMillis.
*
* @return currentTimeMillis.
*/
long currentTimeMillis();
}

View File

@ -0,0 +1,75 @@
/*
* Copyright 2010 The Apache Software Foundation
*
* 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.hadoop.hbase.util;
/**
* Manages a singleton instance of the environment edge. This class shall
* implement static versions of the interface {@link EnvironmentEdge}, then
* defer to the delegate on invocation.
*/
public class EnvironmentEdgeManager {
private static volatile EnvironmentEdge delegate = new DefaultEnvironmentEdge();
private EnvironmentEdgeManager() {
}
/**
* Retrieves the singleton instance of the {@link EnvironmentEdge} that is
* being managed.
*
* @return the edge.
*/
public static EnvironmentEdge getDelegate() {
return delegate;
}
/**
* Resets the managed instance to the default instance: {@link
* DefaultEnvironmentEdge}.
*/
static void reset() {
injectEdge(new DefaultEnvironmentEdge());
}
/**
* Injects the given edge such that it becomes the managed entity. If null is
* passed to this method, the default type is assigned to the delegate.
*
* @param edge the new edge.
*/
static void injectEdge(EnvironmentEdge edge) {
if (edge == null) {
reset();
} else {
delegate = edge;
}
}
/**
* Defers to the delegate and calls the
* {@link EnvironmentEdge#currentTimeMillis()} method.
*
* @return current time in millis according to the delegate.
*/
public static long currentTimeMillis() {
return getDelegate().currentTimeMillis();
}
}

View File

@ -0,0 +1,39 @@
/*
* Copyright 2010 The Apache Software Foundation
*
* 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.hadoop.hbase.util;
/**
* Uses an incrementing algorithm instead of the default.
*/
public class IncrementingEnvironmentEdge implements EnvironmentEdge {
private long timeIncrement = 1;
/**
* {@inheritDoc}
* <p/>
* This method increments a known value for the current time each time this
* method is called. The first value is 1.
*/
@Override
public synchronized long currentTimeMillis() {
return timeIncrement++;
}
}

View File

@ -24,12 +24,12 @@ import org.apache.commons.logging.LogFactory;
import org.apache.hadoop.fs.Path;
import org.apache.hadoop.hbase.HBaseConfiguration;
import org.apache.hadoop.hbase.HBaseTestCase;
import org.apache.hadoop.hbase.HBaseTestingUtility;
import org.apache.hadoop.hbase.HColumnDescriptor;
import org.apache.hadoop.hbase.HConstants;
import org.apache.hadoop.hbase.HRegionInfo;
import org.apache.hadoop.hbase.HTableDescriptor;
import org.apache.hadoop.hbase.KeyValue;
import org.apache.hadoop.hbase.HBaseTestingUtility;
import org.apache.hadoop.hbase.client.Delete;
import org.apache.hadoop.hbase.client.Get;
import org.apache.hadoop.hbase.client.Put;
@ -45,6 +45,8 @@ import org.apache.hadoop.hbase.filter.PrefixFilter;
import org.apache.hadoop.hbase.filter.SingleColumnValueFilter;
import org.apache.hadoop.hbase.regionserver.HRegion.RegionScanner;
import org.apache.hadoop.hbase.util.Bytes;
import org.apache.hadoop.hbase.util.EnvironmentEdgeManagerTestHelper;
import org.apache.hadoop.hbase.util.IncrementingEnvironmentEdge;
import org.apache.hadoop.hbase.util.Threads;
import java.io.IOException;
@ -90,6 +92,12 @@ public class TestHRegion extends HBaseTestCase {
super.setUp();
}
@Override
protected void tearDown() throws Exception {
super.tearDown();
EnvironmentEdgeManagerTestHelper.reset();
}
//////////////////////////////////////////////////////////////////////////////
// New tests that doesn't spin up a mini cluster but rather just test the
// individual code pieces in the HRegion. Putting files locally in
@ -100,7 +108,7 @@ public class TestHRegion extends HBaseTestCase {
HBaseConfiguration hc = initSplit();
int numRows = 100;
byte [][] families = {fam1, fam2, fam3};
//Setting up region
String method = this.getName();
initHRegion(tableName, method, hc, families);
@ -339,7 +347,7 @@ public class TestHRegion extends HBaseTestCase {
//Putting data in key
Put put = new Put(row1);
put.add(fam1, qf1, val1);
//checkAndPut with correct value
boolean res = region.checkAndMutate(row1, fam1, qf1, emptyVal, put, lockId,
true);
@ -351,7 +359,7 @@ public class TestHRegion extends HBaseTestCase {
Delete delete = new Delete(row1);
delete.deleteColumn(fam1, qf1);
res = region.checkAndMutate(row1, fam1, qf1, emptyVal, delete, lockId,
res = region.checkAndMutate(row1, fam1, qf1, emptyVal, delete, lockId,
true);
assertFalse(res);
@ -369,7 +377,7 @@ public class TestHRegion extends HBaseTestCase {
assertTrue(res);
delete = new Delete(row1);
res = region.checkAndMutate(row1, fam1, qf1, emptyVal, delete, lockId,
res = region.checkAndMutate(row1, fam1, qf1, emptyVal, delete, lockId,
true);
assertTrue(res);
}
@ -513,12 +521,12 @@ public class TestHRegion extends HBaseTestCase {
put.add(fam1, qf3, val1);
region.put(put);
//Multi-column delete
//Multi-column delete
Delete delete = new Delete(row1);
delete.deleteColumn(fam1, qf1);
delete.deleteColumn(fam2, qf1);
delete.deleteColumn(fam1, qf3);
boolean res = region.checkAndMutate(row1, fam1, qf1, val2, delete, lockId,
boolean res = region.checkAndMutate(row1, fam1, qf1, val2, delete, lockId,
true);
assertEquals(true, res);
@ -531,10 +539,10 @@ public class TestHRegion extends HBaseTestCase {
assertEquals(val1, r.getValue(fam1, qf1));
assertEquals(val2, r.getValue(fam2, qf2));
//Family delete
//Family delete
delete = new Delete(row1);
delete.deleteFamily(fam2);
res = region.checkAndMutate(row1, fam2, qf1, emptyVal, delete, lockId,
res = region.checkAndMutate(row1, fam2, qf1, emptyVal, delete, lockId,
true);
assertEquals(true, res);
@ -545,7 +553,7 @@ public class TestHRegion extends HBaseTestCase {
//Row delete
delete = new Delete(row1);
res = region.checkAndMutate(row1, fam1, qf1, val1, delete, lockId,
res = region.checkAndMutate(row1, fam1, qf1, val1, delete, lockId,
true);
assertEquals(true, res);
get = new Get(row1);
@ -629,6 +637,7 @@ public class TestHRegion extends HBaseTestCase {
byte [][] families = {fam};
String method = this.getName();
initHRegion(tableName, method, families);
EnvironmentEdgeManagerTestHelper.injectEdge(new IncrementingEnvironmentEdge());
byte [] row = Bytes.toBytes("table_name");
// column names
@ -667,9 +676,6 @@ public class TestHRegion extends HBaseTestCase {
result = region.get(get, null);
assertEquals(1, result.size());
// Sleep to ensure timestamp of next Put is bigger than previous delete
Thread.sleep(10);
// Assert that after a delete, I can put.
put = new Put(row);
put.add(fam, splitA, Bytes.toBytes("reference_A"));
@ -682,10 +688,7 @@ public class TestHRegion extends HBaseTestCase {
delete = new Delete(row);
region.delete(delete, null, false);
assertEquals(0, region.get(get, null).size());
// Sleep to ensure timestamp of next Put is bigger than previous delete
Thread.sleep(10);
region.put(new Put(row).add(fam, splitA, Bytes.toBytes("reference_A")));
result = region.get(get, null);
assertEquals(1, result.size());
@ -781,16 +784,14 @@ public class TestHRegion extends HBaseTestCase {
public void doTestDelete_AndPostInsert(Delete delete)
throws IOException, InterruptedException {
initHRegion(tableName, getName(), fam1);
EnvironmentEdgeManagerTestHelper.injectEdge(new IncrementingEnvironmentEdge());
Put put = new Put(row);
put.add(fam1, qual1, value1);
region.put(put);
Thread.sleep(10);
// now delete the value:
region.delete(delete, null, true);
Thread.sleep(10);
// ok put data:
put = new Put(row);
@ -1223,8 +1224,8 @@ public class TestHRegion extends HBaseTestCase {
Scan scan = null;
HRegion.RegionScanner is = null;
//Testing to see how many scanners that is produced by getScanner, starting
//Testing to see how many scanners that is produced by getScanner, starting
//with known number, 2 - current = 1
scan = new Scan();
scan.addFamily(fam2);
@ -1232,11 +1233,11 @@ public class TestHRegion extends HBaseTestCase {
is = (RegionScanner) region.getScanner(scan);
is.initHeap(); // i dont like this test
assertEquals(1, ((RegionScanner)is).storeHeap.getHeap().size());
scan = new Scan();
is = (RegionScanner) region.getScanner(scan);
is.initHeap();
assertEquals(families.length -1,
assertEquals(families.length -1,
((RegionScanner)is).storeHeap.getHeap().size());
}
@ -1744,7 +1745,7 @@ public class TestHRegion extends HBaseTestCase {
assertTrue("ICV failed to upgrade timestamp",
first.getTimestamp() != second.getTimestamp());
}
public void testIncrementColumnValue_ConcurrentFlush() throws IOException {
initHRegion(tableName, getName(), fam1);
@ -2240,10 +2241,10 @@ public class TestHRegion extends HBaseTestCase {
PutThread putThread = new PutThread(numRows, families, qualifiers);
putThread.start();
putThread.waitForFirstPut();
FlushThread flushThread = new FlushThread();
flushThread.start();
Scan scan = new Scan(Bytes.toBytes("row0"), Bytes.toBytes("row1"));
// scan.setFilter(new RowFilter(CompareFilter.CompareOp.EQUAL,
// new BinaryComparator(Bytes.toBytes("row0"))));
@ -2291,7 +2292,7 @@ public class TestHRegion extends HBaseTestCase {
protected class PutThread extends Thread {
private volatile boolean done;
private volatile int numPutsFinished = 0;
private Throwable error = null;
private int numRows;
private byte[][] families;
@ -2395,7 +2396,7 @@ public class TestHRegion extends HBaseTestCase {
PutThread putThread = new PutThread(numRows, families, qualifiers);
putThread.start();
putThread.waitForFirstPut();
FlushThread flushThread = new FlushThread();
flushThread.start();
@ -2444,7 +2445,7 @@ public class TestHRegion extends HBaseTestCase {
}
putThread.done();
region.flushcache();
putThread.join();
@ -2500,7 +2501,7 @@ public class TestHRegion extends HBaseTestCase {
}
private void putData(int startRow, int numRows, byte [] qf,
byte [] ...families)
throws IOException {

View File

@ -0,0 +1,36 @@
/*
* Copyright 2010 The Apache Software Foundation
*
* 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.hadoop.hbase.util;
/**
* Used by tests to inject an edge into the manager. The intent is to minimise
* the use of the injectEdge method giving it default permissions, but in
* testing we may need to use this functionality elsewhere.
*/
public class EnvironmentEdgeManagerTestHelper {
public static void reset() {
EnvironmentEdgeManager.reset();
}
public static void injectEdge(EnvironmentEdge edge) {
EnvironmentEdgeManager.injectEdge(edge);
}
}

View File

@ -0,0 +1,50 @@
/*
* Copyright 2010 The Apache Software Foundation
*
* 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.hadoop.hbase.util;
import org.junit.Test;
import static junit.framework.Assert.assertTrue;
import static junit.framework.Assert.fail;
/**
* Tests to make sure that the default environment edge conforms to appropriate
* behaviour.
*/
public class TestDefaultEnvironmentEdge {
@Test
public void testGetCurrentTimeUsesSystemClock() {
DefaultEnvironmentEdge edge = new DefaultEnvironmentEdge();
long systemTime = System.currentTimeMillis();
long edgeTime = edge.currentTimeMillis();
assertTrue("System time must be either the same or less than the edge time",
systemTime < edgeTime || systemTime == edgeTime);
try {
Thread.sleep(1);
} catch (InterruptedException e) {
fail(e.getMessage());
}
long secondEdgeTime = edge.currentTimeMillis();
assertTrue("Second time must be greater than the first",
secondEdgeTime > edgeTime);
}
}

View File

@ -0,0 +1,63 @@
/*
* Copyright 2010 The Apache Software Foundation
*
* 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.hadoop.hbase.util;
import org.junit.Test;
import static org.mockito.Mockito.mock;
import static org.mockito.Mockito.when;
import static org.mockito.Mockito.verify;
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertFalse;
import static org.junit.Assert.assertNotNull;
import static org.junit.Assert.assertTrue;
public class TestEnvironmentEdgeManager {
@Test
public void testManageSingleton() {
EnvironmentEdge edge = EnvironmentEdgeManager.getDelegate();
assertNotNull(edge);
assertTrue(edge instanceof DefaultEnvironmentEdge);
EnvironmentEdgeManager.reset();
EnvironmentEdge edge2 = EnvironmentEdgeManager.getDelegate();
assertFalse(edge == edge2);
IncrementingEnvironmentEdge newEdge = new IncrementingEnvironmentEdge();
EnvironmentEdgeManager.injectEdge(newEdge);
assertEquals(newEdge, EnvironmentEdgeManager.getDelegate());
//injecting null will result in default being assigned.
EnvironmentEdgeManager.injectEdge(null);
EnvironmentEdge nullResult = EnvironmentEdgeManager.getDelegate();
assertTrue(nullResult instanceof DefaultEnvironmentEdge);
}
@Test
public void testCurrentTimeInMillis() {
EnvironmentEdge mock = mock(EnvironmentEdge.class);
EnvironmentEdgeManager.injectEdge(mock);
long expectation = 3456;
when(mock.currentTimeMillis()).thenReturn(expectation);
long result = EnvironmentEdgeManager.currentTimeMillis();
verify(mock).currentTimeMillis();
assertEquals(expectation, result);
}
}

View File

@ -0,0 +1,40 @@
/*
* Copyright 2010 The Apache Software Foundation
*
* 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.hadoop.hbase.util;
import org.junit.Test;
import static junit.framework.Assert.assertEquals;
/**
* Tests that the incrementing environment edge increments time instead of using
* the default.
*/
public class TestIncrementingEnvironmentEdge {
@Test
public void testGetCurrentTimeUsesSystemClock() {
IncrementingEnvironmentEdge edge = new IncrementingEnvironmentEdge();
assertEquals(1, edge.currentTimeMillis());
assertEquals(2, edge.currentTimeMillis());
assertEquals(3, edge.currentTimeMillis());
assertEquals(4, edge.currentTimeMillis());
}
}