source: proiecte/HadoopJUnit/hadoop-0.20.1/src/test/org/apache/hadoop/ipc/TestSocketFactory.java @ 120

Last change on this file since 120 was 120, checked in by (none), 14 years ago

Added the mail files for the Hadoop JUNit Project

  • Property svn:executable set to *
File size: 5.9 KB
Line 
1/**
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements.  See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership.  The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License.  You may obtain a copy of the License at
9 *
10 *     http://www.apache.org/licenses/LICENSE-2.0
11 *
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
17 */
18package org.apache.hadoop.ipc;
19
20import java.io.IOException;
21import java.net.InetSocketAddress;
22import java.net.Socket;
23import java.net.SocketAddress;
24
25import junit.framework.TestCase;
26
27import org.apache.hadoop.conf.Configuration;
28import org.apache.hadoop.hdfs.DistributedFileSystem;
29import org.apache.hadoop.hdfs.MiniDFSCluster;
30import org.apache.hadoop.fs.FileSystem;
31import org.apache.hadoop.fs.Path;
32import org.apache.hadoop.mapred.JobClient;
33import org.apache.hadoop.mapred.JobConf;
34import org.apache.hadoop.mapred.JobStatus;
35import org.apache.hadoop.mapred.MiniMRCluster;
36import org.apache.hadoop.net.StandardSocketFactory;
37
38/**
39 * This class checks that RPCs can use specialized socket factories.
40 */
41public class TestSocketFactory extends TestCase {
42
43  /**
44   * Check that we can reach a NameNode or a JobTracker using a specific
45   * socket factory
46   */
47  public void testSocketFactory() throws IOException {
48    // Create a standard mini-cluster
49    Configuration sconf = new Configuration();
50    MiniDFSCluster cluster = new MiniDFSCluster(sconf, 1, true, null);
51    final int nameNodePort = cluster.getNameNodePort();
52
53    // Get a reference to its DFS directly
54    FileSystem fs = cluster.getFileSystem();
55    assertTrue(fs instanceof DistributedFileSystem);
56    DistributedFileSystem directDfs = (DistributedFileSystem) fs;
57
58    // Get another reference via network using a specific socket factory
59    Configuration cconf = new Configuration();
60    FileSystem.setDefaultUri(cconf, String.format("hdfs://localhost:%s/",
61        nameNodePort + 10));
62    cconf.set("hadoop.rpc.socket.factory.class.default",
63        "org.apache.hadoop.ipc.DummySocketFactory");
64    cconf.set("hadoop.rpc.socket.factory.class.ClientProtocol",
65        "org.apache.hadoop.ipc.DummySocketFactory");
66    cconf.set("hadoop.rpc.socket.factory.class.JobSubmissionProtocol",
67        "org.apache.hadoop.ipc.DummySocketFactory");
68
69    fs = FileSystem.get(cconf);
70    assertTrue(fs instanceof DistributedFileSystem);
71    DistributedFileSystem dfs = (DistributedFileSystem) fs;
72
73    JobClient client = null;
74    MiniMRCluster mr = null;
75    try {
76      // This will test RPC to the NameNode only.
77      // could we test Client-DataNode connections?
78      Path filePath = new Path("/dir");
79
80      assertFalse(directDfs.exists(filePath));
81      assertFalse(dfs.exists(filePath));
82
83      directDfs.mkdirs(filePath);
84      assertTrue(directDfs.exists(filePath));
85      assertTrue(dfs.exists(filePath));
86
87      // This will test TPC to a JobTracker
88      fs = FileSystem.get(sconf);
89      mr = new MiniMRCluster(1, fs.getUri().toString(), 1);
90      final int jobTrackerPort = mr.getJobTrackerPort();
91
92      JobConf jconf = new JobConf(cconf);
93      jconf.set("mapred.job.tracker", String.format("localhost:%d",
94          jobTrackerPort + 10));
95      client = new JobClient(jconf);
96
97      JobStatus[] jobs = client.jobsToComplete();
98      assertTrue(jobs.length == 0);
99
100    } finally {
101      try {
102        if (client != null)
103          client.close();
104      } catch (Exception ignored) {
105        // nothing we can do
106        ignored.printStackTrace();
107      }
108      try {
109        if (dfs != null)
110          dfs.close();
111
112      } catch (Exception ignored) {
113        // nothing we can do
114        ignored.printStackTrace();
115      }
116      try {
117        if (directDfs != null)
118          directDfs.close();
119
120      } catch (Exception ignored) {
121        // nothing we can do
122        ignored.printStackTrace();
123      }
124      try {
125        if (cluster != null)
126          cluster.shutdown();
127
128      } catch (Exception ignored) {
129        // nothing we can do
130        ignored.printStackTrace();
131      }
132      if (mr != null) {
133        try {
134          mr.shutdown();
135        } catch (Exception ignored) {
136          ignored.printStackTrace();
137        }
138      }
139    }
140  }
141}
142
143/**
144 * Dummy socket factory which shift TPC ports by subtracting 10 when
145 * establishing a connection
146 */
147class DummySocketFactory extends StandardSocketFactory {
148  /**
149   * Default empty constructor (for use with the reflection API).
150   */
151  public DummySocketFactory() {
152  }
153
154  /* @inheritDoc */
155  @Override
156  public Socket createSocket() throws IOException {
157    return new Socket() {
158      @Override
159      public void connect(SocketAddress addr, int timeout)
160          throws IOException {
161
162        assert (addr instanceof InetSocketAddress);
163        InetSocketAddress iaddr = (InetSocketAddress) addr;
164        SocketAddress newAddr = null;
165        if (iaddr.isUnresolved())
166          newAddr =
167              new InetSocketAddress(iaddr.getHostName(),
168                  iaddr.getPort() - 10);
169        else
170          newAddr =
171              new InetSocketAddress(iaddr.getAddress(), iaddr.getPort() - 10);
172        System.out.printf("Test socket: rerouting %s to %s\n", iaddr,
173            newAddr);
174        super.connect(newAddr, timeout);
175      }
176    };
177  }
178
179  /* @inheritDoc */
180  @Override
181  public boolean equals(Object obj) {
182    if (this == obj)
183      return true;
184    if (obj == null)
185      return false;
186    if (!(obj instanceof DummySocketFactory))
187      return false;
188    return true;
189  }
190
191  /* @inheritDoc */
192  @Override
193  public int hashCode() {
194    // Dummy hash code (to make find bugs happy)
195    return 53;
196  }
197}
Note: See TracBrowser for help on using the repository browser.