Author: shv
Date: Tue Aug 26 14:13:48 2008
New Revision: 689230
URL: http://svn.apache.org/viewvc?rev=689230&view=rev
Log:
HADOOP-3965. Convert DataBlockScanner into a package private class. Contributed
by Konstantin Shvachko.
Added:
hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/server/datanode/TestInterDatanodeProtocol.java
Removed:
hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/TestInterDatanodeProtocol.java
Modified:
hadoop/core/trunk/CHANGES.txt
hadoop/core/trunk/src/hdfs/org/apache/hadoop/hdfs/server/datanode/DataBlockScanner.java
hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/MiniDFSCluster.java
hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/TestLeaseRecovery.java
hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/TestLeaseRecovery2.java
Modified: hadoop/core/trunk/CHANGES.txt
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/CHANGES.txt?rev=689230&r1=689229&r2=689230&view=diff
==============================================================================
--- hadoop/core/trunk/CHANGES.txt (original)
+++ hadoop/core/trunk/CHANGES.txt Tue Aug 26 14:13:48 2008
@@ -224,6 +224,8 @@
HADOOP-2330. Preallocate HDFS transaction log to improve performance.
(dhruba and hairong)
+ HADOOP-3965. Convert DataBlockScanner into a package private class. (shv)
+
OPTIMIZATIONS
HADOOP-3556. Removed lock contention in MD5Hash by changing the
Modified:
hadoop/core/trunk/src/hdfs/org/apache/hadoop/hdfs/server/datanode/DataBlockScanner.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/hdfs/org/apache/hadoop/hdfs/server/datanode/DataBlockScanner.java?rev=689230&r1=689229&r2=689230&view=diff
==============================================================================
---
hadoop/core/trunk/src/hdfs/org/apache/hadoop/hdfs/server/datanode/DataBlockScanner.java
(original)
+++
hadoop/core/trunk/src/hdfs/org/apache/hadoop/hdfs/server/datanode/DataBlockScanner.java
Tue Aug 26 14:13:48 2008
@@ -57,7 +57,7 @@
* Currently it does not modify the metadata for block.
*/
-public class DataBlockScanner implements Runnable {
+class DataBlockScanner implements Runnable {
public static final Log LOG = LogFactory.getLog(DataBlockScanner.class);
@@ -122,7 +122,7 @@
compareTo((BlockScanInfo)other) == 0;
}
- public long getLastScanTime() {
+ long getLastScanTime() {
return ( lastScanType == ScanType.NONE) ? 0 : lastScanTime;
}
@@ -283,7 +283,7 @@
}
/** @return the last scan time */
- public synchronized long getLastScanTime(Block block) {
+ synchronized long getLastScanTime(Block block) {
if (!isInitiliazed()) {
return 0;
}
Modified: hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/MiniDFSCluster.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/MiniDFSCluster.java?rev=689230&r1=689229&r2=689230&view=diff
==============================================================================
--- hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/MiniDFSCluster.java
(original)
+++ hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/MiniDFSCluster.java Tue
Aug 26 14:13:48 2008
@@ -514,7 +514,7 @@
}
/** @return the datanode having the ipc server listen port */
- DataNode getDataNode(int ipcPort) {
+ public DataNode getDataNode(int ipcPort) {
for(DataNode dn : getDataNodes()) {
if (dn.ipcServer.getListenerAddress().getPort() == ipcPort) {
return dn;
Modified:
hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/TestLeaseRecovery.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/TestLeaseRecovery.java?rev=689230&r1=689229&r2=689230&view=diff
==============================================================================
--- hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/TestLeaseRecovery.java
(original)
+++ hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/TestLeaseRecovery.java
Tue Aug 26 14:13:48 2008
@@ -26,6 +26,7 @@
import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
import org.apache.hadoop.hdfs.protocol.LocatedBlock;
import org.apache.hadoop.hdfs.server.datanode.DataNode;
+import org.apache.hadoop.hdfs.server.datanode.TestInterDatanodeProtocol;
import org.apache.hadoop.hdfs.server.protocol.BlockMetaDataInfo;
import org.apache.hadoop.hdfs.server.protocol.InterDatanodeProtocol;
Modified:
hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/TestLeaseRecovery2.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/TestLeaseRecovery2.java?rev=689230&r1=689229&r2=689230&view=diff
==============================================================================
--- hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/TestLeaseRecovery2.java
(original)
+++ hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/TestLeaseRecovery2.java
Tue Aug 26 14:13:48 2008
@@ -23,6 +23,7 @@
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.*;
import org.apache.hadoop.hdfs.protocol.Block;
+import org.apache.hadoop.hdfs.server.datanode.TestInterDatanodeProtocol;
import org.apache.hadoop.hdfs.server.protocol.InterDatanodeProtocol;
public class TestLeaseRecovery2 extends junit.framework.TestCase {
@@ -90,7 +91,7 @@
dfs.dfs.clientName += "_1";
while (true) {
try {
- FSDataOutputStream newstm = dfs.create(filepath, false,
+ dfs.create(filepath, false,
dfs.getConf().getInt("io.file.buffer.size", 4096),
(short)repl, (long)BLOCK_SIZE);
assertTrue("Creation of an existing file should never succeed.",
false);
Added:
hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/server/datanode/TestInterDatanodeProtocol.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/server/datanode/TestInterDatanodeProtocol.java?rev=689230&view=auto
==============================================================================
---
hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/server/datanode/TestInterDatanodeProtocol.java
(added)
+++
hadoop/core/trunk/src/test/org/apache/hadoop/hdfs/server/datanode/TestInterDatanodeProtocol.java
Tue Aug 26 14:13:48 2008
@@ -0,0 +1,114 @@
+/**
+ * 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.hdfs.server.datanode;
+
+import java.io.IOException;
+import java.util.List;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.fs.*;
+import org.apache.hadoop.hdfs.DFSTestUtil;
+import org.apache.hadoop.hdfs.MiniDFSCluster;
+import org.apache.hadoop.hdfs.DistributedFileSystem;
+import org.apache.hadoop.hdfs.protocol.Block;
+import org.apache.hadoop.hdfs.protocol.ClientProtocol;
+import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
+import org.apache.hadoop.hdfs.protocol.LocatedBlock;
+import org.apache.hadoop.hdfs.protocol.LocatedBlocks;
+import org.apache.hadoop.hdfs.server.datanode.DataBlockScanner;
+import org.apache.hadoop.hdfs.server.datanode.DataNode;
+import org.apache.hadoop.hdfs.server.protocol.BlockMetaDataInfo;
+import org.apache.hadoop.hdfs.server.protocol.InterDatanodeProtocol;
+
+/**
+ * This tests InterDataNodeProtocol for block handling.
+ */
+public class TestInterDatanodeProtocol extends junit.framework.TestCase {
+ public static void checkMetaInfo(Block b, InterDatanodeProtocol idp,
+ DataBlockScanner scanner) throws IOException {
+ BlockMetaDataInfo metainfo = idp.getBlockMetaDataInfo(b);
+ assertEquals(b.getBlockId(), metainfo.getBlockId());
+ assertEquals(b.getNumBytes(), metainfo.getNumBytes());
+ if (scanner != null) {
+ assertEquals(scanner.getLastScanTime(b),
+ metainfo.getLastScanTime());
+ }
+ }
+
+ public static LocatedBlock getLastLocatedBlock(
+ ClientProtocol namenode, String src
+ ) throws IOException {
+ //get block info for the last block
+ LocatedBlocks locations = namenode.getBlockLocations(src, 0,
Long.MAX_VALUE);
+ List<LocatedBlock> blocks = locations.getLocatedBlocks();
+ DataNode.LOG.info("blocks.size()=" + blocks.size());
+ assertTrue(blocks.size() > 0);
+
+ return blocks.get(blocks.size() - 1);
+ }
+
+ /**
+ * The following test first creates a file.
+ * It verifies the block information from a datanode.
+ * Then, it updates the block with new information and verifies again.
+ */
+ public void testBlockMetaDataInfo() throws Exception {
+ Configuration conf = new Configuration();
+ MiniDFSCluster cluster = null;
+
+ try {
+ cluster = new MiniDFSCluster(conf, 3, true, null);
+ cluster.waitActive();
+
+ //create a file
+ DistributedFileSystem dfs =
(DistributedFileSystem)cluster.getFileSystem();
+ String filestr = "/foo";
+ Path filepath = new Path(filestr);
+ DFSTestUtil.createFile(dfs, filepath, 1024L, (short)3, 0L);
+ assertTrue(dfs.getClient().exists(filestr));
+
+ //get block info
+ LocatedBlock locatedblock =
getLastLocatedBlock(dfs.getClient().namenode, filestr);
+ DatanodeInfo[] datanodeinfo = locatedblock.getLocations();
+ assertTrue(datanodeinfo.length > 0);
+
+ //connect to a data node
+ InterDatanodeProtocol idp = DataNode.createInterDataNodeProtocolProxy(
+ datanodeinfo[0], conf);
+ DataNode datanode = cluster.getDataNode(datanodeinfo[0].getIpcPort());
+ assertTrue(datanode != null);
+
+ //stop block scanner, so we could compare lastScanTime
+ datanode.blockScannerThread.interrupt();
+
+ //verify BlockMetaDataInfo
+ Block b = locatedblock.getBlock();
+ InterDatanodeProtocol.LOG.info("b=" + b + ", " + b.getClass());
+ checkMetaInfo(b, idp, datanode.blockScanner);
+
+ //verify updateBlock
+ Block newblock = new Block(
+ b.getBlockId(), b.getNumBytes()/2, b.getGenerationStamp()+1);
+ idp.updateBlock(b, newblock, false);
+ checkMetaInfo(newblock, idp, datanode.blockScanner);
+ }
+ finally {
+ if (cluster != null) {cluster.shutdown();}
+ }
+ }
+}