You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pig.apache.org by ch...@apache.org on 2014/03/03 19:03:02 UTC

svn commit: r1573643 - in /pig/branches/tez/test: ./ org/apache/pig/test/

Author: cheolsoo
Date: Mon Mar  3 18:03:01 2014
New Revision: 1573643

URL: http://svn.apache.org/r1573643
Log:
PIG-3791: Port more mini cluster tests to Tez - part2 (cheolsoo)

Modified:
    pig/branches/tez/test/org/apache/pig/test/TestAlgebraicEval.java
    pig/branches/tez/test/org/apache/pig/test/TestBZip.java
    pig/branches/tez/test/org/apache/pig/test/TestBestFitCast.java
    pig/branches/tez/test/org/apache/pig/test/TestBinaryExpressionOps.java
    pig/branches/tez/test/org/apache/pig/test/TestBuiltin.java
    pig/branches/tez/test/org/apache/pig/test/TestCharArrayToNumeric.java
    pig/branches/tez/test/org/apache/pig/test/TestCompressedFiles.java
    pig/branches/tez/test/tez-tests

Modified: pig/branches/tez/test/org/apache/pig/test/TestAlgebraicEval.java
URL: http://svn.apache.org/viewvc/pig/branches/tez/test/org/apache/pig/test/TestAlgebraicEval.java?rev=1573643&r1=1573642&r2=1573643&view=diff
==============================================================================
--- pig/branches/tez/test/org/apache/pig/test/TestAlgebraicEval.java (original)
+++ pig/branches/tez/test/org/apache/pig/test/TestAlgebraicEval.java Mon Mar  3 18:03:01 2014
@@ -23,26 +23,35 @@ import java.io.File;
 import java.io.FileOutputStream;
 import java.io.PrintStream;
 import java.util.Iterator;
+import java.util.Properties;
 import java.util.Random;
 
-import org.apache.pig.ExecType;
 import org.apache.pig.PigServer;
 import org.apache.pig.builtin.PigStorage;
 import org.apache.pig.data.DataType;
 import org.apache.pig.data.Tuple;
 import org.junit.AfterClass;
 import org.junit.Before;
+import org.junit.BeforeClass;
 import org.junit.Test;
 
 public class TestAlgebraicEval {
+    private static PigServer pig;
+    private static Properties properties;
+    private static MiniGenericCluster cluster;
 
     private int LOOP_COUNT = 1024;
-
-    private PigServer pig;
+    private Boolean[] nullFlags = new Boolean[]{ false, true};
 
     @Before
     public void setUp() throws Exception {
-        pig = new PigServer(ExecType.MAPREDUCE, cluster.getProperties());
+        pig = new PigServer(cluster.getExecType(), properties);
+    }
+
+    @BeforeClass
+    public static void oneTimeSetUp() throws Exception {
+        cluster = MiniGenericCluster.buildCluster();
+        properties = cluster.getProperties();
     }
 
     @AfterClass
@@ -50,9 +59,6 @@ public class TestAlgebraicEval {
         cluster.shutDown();
     }
 
-    Boolean[] nullFlags = new Boolean[]{ false, true};
-
-    static MiniCluster cluster = MiniCluster.buildCluster();
     @Test
     public void testGroupCountWithMultipleFields() throws Throwable {
         File tmpFile = File.createTempFile("test", "txt");

Modified: pig/branches/tez/test/org/apache/pig/test/TestBZip.java
URL: http://svn.apache.org/viewvc/pig/branches/tez/test/org/apache/pig/test/TestBZip.java?rev=1573643&r1=1573642&r2=1573643&view=diff
==============================================================================
--- pig/branches/tez/test/org/apache/pig/test/TestBZip.java (original)
+++ pig/branches/tez/test/org/apache/pig/test/TestBZip.java Mon Mar  3 18:03:01 2014
@@ -33,15 +33,12 @@ import java.util.Iterator;
 import java.util.Properties;
 import java.util.Map.Entry;
 
-import junit.framework.Assert;
-
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.FSDataInputStream;
 import org.apache.hadoop.fs.FileStatus;
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.mapreduce.InputSplit;
-import org.apache.pig.ExecType;
 import org.apache.pig.PigServer;
 import org.apache.pig.backend.hadoop.datastorage.ConfigurationUtil;
 import org.apache.pig.data.DataType;
@@ -50,34 +47,39 @@ import org.apache.pig.impl.PigContext;
 import org.apache.tools.bzip2r.CBZip2InputStream;
 import org.apache.tools.bzip2r.CBZip2OutputStream;
 import org.junit.AfterClass;
+import org.junit.BeforeClass;
 import org.junit.Test;
-import org.junit.runner.RunWith;
-import org.junit.runners.JUnit4;
 
-@RunWith(JUnit4.class)
 public class TestBZip {
-    static MiniCluster cluster = MiniCluster.buildCluster();
-    
+    private static Properties properties;
+    private static MiniGenericCluster cluster;
+
+    @BeforeClass
+    public static void oneTimeSetUp() throws Exception {
+        cluster = MiniGenericCluster.buildCluster();
+        properties = cluster.getProperties();
+    }
+
     @AfterClass
     public static void oneTimeTearDown() throws Exception {
         cluster.shutDown();
     }
-    
+
    /**
     * Tests the end-to-end writing and reading of a BZip file.
     */
     @Test
     public void testBzipInPig() throws Exception {
-        PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster.getProperties());
-       
+        PigServer pig = new PigServer(cluster.getExecType(), properties);
+
         File in = File.createTempFile("junit", ".bz2");
         in.deleteOnExit();
-        
+
         File out = File.createTempFile("junit", ".bz2");
         out.delete();
         String clusterOutput = Util.removeColon(out.getAbsolutePath());
-               
-        CBZip2OutputStream cos = 
+
+        CBZip2OutputStream cos =
             new CBZip2OutputStream(new FileOutputStream(in));
         for (int i = 1; i < 100; i++) {
             StringBuffer sb = new StringBuffer();
@@ -86,7 +88,7 @@ public class TestBZip {
             cos.write(bytes);
         }
         cos.close();
-                       
+
         pig.registerQuery("AA = load '"
                 + Util.generateURI(Util.encodeEscape(in.getAbsolutePath()), pig.getPigContext())
                 + "';");
@@ -94,49 +96,50 @@ public class TestBZip {
         pig.registerQuery("store A into '" + Util.encodeEscape(clusterOutput) + "';");
         FileSystem fs = FileSystem.get(ConfigurationUtil.toConfiguration(
                 pig.getPigContext().getProperties()));
-        FSDataInputStream is = fs.open(new Path(clusterOutput +
-                "/part-r-00000.bz2"));
+        FileStatus[] outputFiles = fs.listStatus(new Path(clusterOutput),
+                Util.getSuccessMarkerPathFilter());
+        FSDataInputStream is = fs.open(outputFiles[0].getPath());
         CBZip2InputStream cis = new CBZip2InputStream(is, -1, out.length());
-        
+
         // Just a sanity check, to make sure it was a bzip file; we
         // will do the value verification later
         assertEquals(100, cis.read(new byte[100]));
         cis.close();
-        
+
         pig.registerQuery("B = load '" + Util.encodeEscape(clusterOutput) + "';");
-        
+
         Iterator<Tuple> i = pig.openIterator("B");
         HashMap<Integer, Integer> map = new HashMap<Integer, Integer>();
         while (i.hasNext()) {
             Integer val = DataType.toInteger(i.next().get(0));
-            map.put(val, val);            
+            map.put(val, val);
         }
-        
+
         assertEquals(new Integer(99), new Integer(map.keySet().size()));
-        
+
         for (int j = 1; j < 100; j++) {
             assertEquals(new Integer(j), map.get(j));
         }
-        
+
         in.delete();
         Util.deleteFile(cluster, clusterOutput);
     }
-    
+
    /**
     * Tests the end-to-end writing and reading of a BZip file using absolute path with a trailing /.
     */
     @Test
     public void testBzipInPig2() throws Exception {
-        PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster.getProperties());
-       
+        PigServer pig = new PigServer(cluster.getExecType(), properties);
+
         File in = File.createTempFile("junit", ".bz2");
         in.deleteOnExit();
-        
+
         File out = File.createTempFile("junit", ".bz2");
         out.delete();
         String clusterOutput = Util.removeColon(out.getAbsolutePath());
-               
-        CBZip2OutputStream cos = 
+
+        CBZip2OutputStream cos =
             new CBZip2OutputStream(new FileOutputStream(in));
         for (int i = 1; i < 100; i++) {
             StringBuffer sb = new StringBuffer();
@@ -145,7 +148,7 @@ public class TestBZip {
             cos.write(bytes);
         }
         cos.close();
-                       
+
         pig.registerQuery("AA = load '"
                 + Util.generateURI(in.getAbsolutePath(), pig.getPigContext())
                 + "';");
@@ -153,30 +156,31 @@ public class TestBZip {
         pig.registerQuery("store A into '" + Util.encodeEscape(clusterOutput) + "/';");
         FileSystem fs = FileSystem.get(ConfigurationUtil.toConfiguration(
                 pig.getPigContext().getProperties()));
-        FSDataInputStream is = fs.open(new Path(clusterOutput +
-                "/part-r-00000.bz2"));
+        FileStatus[] outputFiles = fs.listStatus(new Path(clusterOutput),
+                Util.getSuccessMarkerPathFilter());
+        FSDataInputStream is = fs.open(outputFiles[0].getPath());
         CBZip2InputStream cis = new CBZip2InputStream(is, -1, out.length());
-        
+
         // Just a sanity check, to make sure it was a bzip file; we
         // will do the value verification later
         assertEquals(100, cis.read(new byte[100]));
         cis.close();
-        
+
         pig.registerQuery("B = load '" + Util.encodeEscape(clusterOutput) + "';");
-        
+
         Iterator<Tuple> i = pig.openIterator("B");
         HashMap<Integer, Integer> map = new HashMap<Integer, Integer>();
         while (i.hasNext()) {
             Integer val = DataType.toInteger(i.next().get(0));
-            map.put(val, val);            
+            map.put(val, val);
         }
-        
+
         assertEquals(new Integer(99), new Integer(map.keySet().size()));
-        
+
         for (int j = 1; j < 100; j++) {
             assertEquals(new Integer(j), map.get(j));
         }
-        
+
         in.delete();
         out.delete();
     }
@@ -190,15 +194,15 @@ public class TestBZip {
                 "7\t8", // '\n' case
                 "9\t10\r" // '\r\n' at the end of file
         };
-        
+
         // bzip compressed input
         File in = File.createTempFile("junit", ".bz2");
         String compressedInputFileName = in.getAbsolutePath();
         String clusterCompressedFilePath = Util.removeColon(compressedInputFileName);
         in.deleteOnExit();
-        
+
         try {
-            CBZip2OutputStream cos = 
+            CBZip2OutputStream cos =
                 new CBZip2OutputStream(new FileOutputStream(in));
             for (int i = 0; i < inputData.length; i++) {
                 StringBuffer sb = new StringBuffer();
@@ -207,31 +211,30 @@ public class TestBZip {
                 cos.write(bytes);
             }
             cos.close();
-            
+
             Util.copyFromLocalToCluster(cluster, compressedInputFileName,
-            		clusterCompressedFilePath);
-            
+                    clusterCompressedFilePath);
+
             // pig script to read compressed input
-            PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster
-                    .getProperties());
-            
+            PigServer pig = new PigServer(cluster.getExecType(), properties);
+
             // pig script to read compressed input
             String script ="a = load '" + Util.encodeEscape(clusterCompressedFilePath) +"';";
             pig.registerQuery(script);
-            
+
             pig.registerQuery("store a into 'intermediate.bz';");
             pig.registerQuery("b = load 'intermediate.bz';");
             Iterator<Tuple> it2 = pig.openIterator("b");
-			while (it2.hasNext()) {
-				it2.next();
-			}
+            while (it2.hasNext()) {
+                it2.next();
+            }
         } finally {
             in.delete();
             Util.deleteFile(cluster, "intermediate.bz");
             Util.deleteFile(cluster, "final.bz");
         }
     }
-    /** 
+    /**
      * Tests that '\n', '\r' and '\r\n' are treated as record delims when using
      * bzip just like they are when using uncompressed text
      */
@@ -243,7 +246,7 @@ public class TestBZip {
                 "7\t8", // '\n' case
                 "9\t10\r" // '\r\n' at the end of file
         };
-        
+
         // bzip compressed input
         File in = File.createTempFile("junit", ".bz2");
         String compressedInputFileName = in.getAbsolutePath();
@@ -252,9 +255,9 @@ public class TestBZip {
 
         String unCompressedInputFileName = "testRecordDelims-uncomp.txt";
         Util.createInputFile(cluster, unCompressedInputFileName, inputData);
-        
+
         try {
-            CBZip2OutputStream cos = 
+            CBZip2OutputStream cos =
                 new CBZip2OutputStream(new FileOutputStream(in));
             for (int i = 0; i < inputData.length; i++) {
                 StringBuffer sb = new StringBuffer();
@@ -263,58 +266,56 @@ public class TestBZip {
                 cos.write(bytes);
             }
             cos.close();
-            
+
             Util.copyFromLocalToCluster(cluster, compressedInputFileName,
                     clusterCompressedFilePath);
-            
+
             // pig script to read uncompressed input
             String script = "a = load '" + unCompressedInputFileName +"';";
-            PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster
-                    .getProperties());
+            PigServer pig = new PigServer(cluster.getExecType(), properties);
             pig.registerQuery(script);
             Iterator<Tuple> it1 = pig.openIterator("a");
-            
+
             // pig script to read compressed input
             script = "a = load '" + Util.encodeEscape(clusterCompressedFilePath) +"';";
             pig.registerQuery(script);
             Iterator<Tuple> it2 = pig.openIterator("a");
-            
+
             while(it1.hasNext()) {
                 Tuple t1 = it1.next();
                 Tuple t2 = it2.next();
-                Assert.assertEquals(t1, t2);
+                assertEquals(t1, t2);
             }
-            
-            Assert.assertFalse(it2.hasNext());
-        
+
+            assertFalse(it2.hasNext());
+
         } finally {
             in.delete();
             Util.deleteFile(cluster, unCompressedInputFileName);
             Util.deleteFile(cluster, clusterCompressedFilePath);
         }
-        
+
     }
-    
+
     /**
      * Tests the end-to-end writing and reading of an empty BZip file.
      */
      @Test
      public void testEmptyBzipInPig() throws Exception {
-        PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster
-                .getProperties());
- 
+        PigServer pig = new PigServer(cluster.getExecType(), properties);
+
         File in = File.createTempFile("junit", ".tmp");
         in.deleteOnExit();
 
         File out = File.createTempFile("junit", ".bz2");
         out.delete();
         String clusterOutputFilePath = Util.removeColon(out.getAbsolutePath());
-        
+
         FileOutputStream fos = new FileOutputStream(in);
         fos.write("55\n".getBytes());
         fos.close();
         System.out.println(in.getAbsolutePath());
-        
+
         pig.registerQuery("AA = load '"
                 + Util.generateURI(Util.encodeEscape(in.getAbsolutePath()), pig.getPigContext())
                 + "';");
@@ -322,21 +323,22 @@ public class TestBZip {
         pig.registerQuery("store A into '" + Util.encodeEscape(clusterOutputFilePath) + "';");
         FileSystem fs = FileSystem.get(ConfigurationUtil.toConfiguration(
                 pig.getPigContext().getProperties()));
-        FSDataInputStream is = fs.open(new Path(clusterOutputFilePath +
-                "/part-r-00000.bz2"));
+        FileStatus[] outputFiles = fs.listStatus(new Path(clusterOutputFilePath),
+                Util.getSuccessMarkerPathFilter());
+        FSDataInputStream is = fs.open(outputFiles[0].getPath());
         CBZip2InputStream cis = new CBZip2InputStream(is, -1, out.length());
-        
+
         // Just a sanity check, to make sure it was a bzip file; we
         // will do the value verification later
         assertEquals(-1, cis.read(new byte[100]));
         cis.close();
-        
+
         pig.registerQuery("B = load '" + Util.encodeEscape(clusterOutputFilePath) + "';");
         pig.openIterator("B");
-        
+
         in.delete();
         Util.deleteFile(cluster, clusterOutputFilePath);
-        
+
     }
 
     /**
@@ -357,7 +359,7 @@ public class TestBZip {
         cis.close();
         tmp.delete();
     }
-    
+
     /**
      * Tests the case where a bzip block ends exactly at the end of the {@link InputSplit}
      * with the block header ending a few bits into the last byte of current
@@ -371,11 +373,11 @@ public class TestBZip {
         // test/org/apache/pig/test/data/bzipdir1.bz2/bzipdir2.bz2/recordLossblockHeaderEndsAt136500.txt.bz2
         // In this test we will load test/org/apache/pig/test/data/bzipdir1.bz2 to also
         // test that the BZip2TextInputFormat can read subdirs recursively
-        String inputFileName = 
+        String inputFileName =
             "test/org/apache/pig/test/data/bzipdir1.bz2";
         Long expectedCount = 74999L; // number of lines in above file
-        // the first block in the above file exactly ends a few bits into the 
-        // byte at position 136500 
+        // the first block in the above file exactly ends a few bits into the
+        // byte at position 136500
         int splitSize = 136500;
         try {
             Util.copyFromLocalToCluster(cluster, inputFileName, inputFileName);
@@ -385,21 +387,21 @@ public class TestBZip {
             Util.deleteFile(cluster, inputFileName);
         }
     }
-    
+
     /**
-     *  Tests the case where a bzip block ends exactly at the end of the input 
+     *  Tests the case where a bzip block ends exactly at the end of the input
      *  split (byte aligned with the last byte) and the last byte is a carriage
      *  return.
      */
     @Test
     public void testBlockHeaderEndingWithCR() throws IOException {
-        String inputFileName = 
+        String inputFileName =
             "test/org/apache/pig/test/data/blockEndingInCR.txt.bz2";
         // number of lines in above file (the value is 1 more than bzcat | wc -l
         // since there is a '\r' which is also treated as a record delim
-        Long expectedCount = 82094L; 
-        // the first block in the above file exactly ends at the byte at 
-        // position 136498 and the last byte is a carriage return ('\r') 
+        Long expectedCount = 82094L;
+        // the first block in the above file exactly ends at the byte at
+        // position 136498 and the last byte is a carriage return ('\r')
         try {
             int splitSize = 136498;
             Util.copyFromLocalToCluster(cluster, inputFileName, inputFileName);
@@ -408,21 +410,21 @@ public class TestBZip {
             Util.deleteFile(cluster, inputFileName);
         }
     }
-    
+
     /**
      * Tests the case where a bzip block ends exactly at the end of the input
      * split and has more data which results in overcounting (record duplication)
      * in Pig 0.6
-     * 
+     *
      */
     @Test
     public void testBlockHeaderEndingAtSplitOverCounting() throws IOException {
-       
-        String inputFileName = 
+
+        String inputFileName =
             "test/org/apache/pig/test/data/blockHeaderEndsAt136500.txt.bz2";
         Long expectedCount = 1041046L; // number of lines in above file
-        // the first block in the above file exactly ends a few bits into the 
-        // byte at position 136500 
+        // the first block in the above file exactly ends a few bits into the
+        // byte at position 136500
         int splitSize = 136500;
         try {
             Util.copyFromLocalToCluster(cluster, inputFileName, inputFileName);
@@ -431,29 +433,28 @@ public class TestBZip {
             Util.deleteFile(cluster, inputFileName);
         }
     }
-    
-    private void testCount(String inputFileName, Long expectedCount, 
+
+    private void testCount(String inputFileName, Long expectedCount,
             int splitSize, String loadFuncSpec) throws IOException {
         String outputFile = "/tmp/bz-output";
         // simple load-store script to verify that the bzip input is getting
         // split
         String scriptToTestSplitting = "a = load '" +inputFileName + "' using " +
         loadFuncSpec + "; store a into '" + outputFile + "';";
-        
+
         String script = "a = load '" + inputFileName + "';" +
-        		"b = group a all;" +
-        		"c = foreach b generate COUNT_STAR(a);";
+                "b = group a all;" +
+                "c = foreach b generate COUNT_STAR(a);";
         Properties props = new Properties();
-        for (Entry<Object, Object> entry : cluster.getProperties().entrySet()) {
+        for (Entry<Object, Object> entry : properties.entrySet()) {
             props.put(entry.getKey(), entry.getValue());
         }
         props.setProperty("mapred.max.split.size", Integer.toString(splitSize));
-        PigContext pigContext = new PigContext(ExecType.MAPREDUCE, props);
-        PigServer pig = new PigServer(pigContext);
+        PigServer pig = new PigServer(cluster.getExecType(), props);
         FileSystem fs = FileSystem.get(ConfigurationUtil.toConfiguration(props));
         fs.delete(new Path(outputFile), true);
         Util.registerMultiLineQuery(pig, scriptToTestSplitting);
-        
+
         // verify that > 1 maps were launched due to splitting of the bzip input
         FileStatus[] files = fs.listStatus(new Path(outputFile));
         int numPartFiles = 0;
@@ -463,40 +464,41 @@ public class TestBZip {
             }
         }
         assertEquals(true, numPartFiles > 0);
-        
+
         // verify record count to verify we read bzip data correctly
         Util.registerMultiLineQuery(pig, script);
         Iterator<Tuple> it = pig.openIterator("c");
         Long result = (Long) it.next().get(0);
         assertEquals(expectedCount, result);
-        
+
     }
-    
+
     @Test
     public void testBzipStoreInMultiQuery() throws Exception {
         String[] inputData = new String[] {
                 "1\t2\r3\t4"
         };
-        
+
         String inputFileName = "input.txt";
         Util.createInputFile(cluster, inputFileName, inputData);
-        
-        PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster
-                .getProperties());
-        
+
+        PigServer pig = new PigServer(cluster.getExecType(), properties);
+
         pig.setBatchOn();
         pig.registerQuery("a = load '" +  inputFileName + "';");
         pig.registerQuery("store a into 'output.bz2';");
         pig.registerQuery("store a into 'output';");
         pig.executeBatch();
-        
+
         FileSystem fs = FileSystem.get(ConfigurationUtil.toConfiguration(
                 pig.getPigContext().getProperties()));
-        FileStatus stat = fs.getFileStatus(new Path("output/part-m-00000"));        
-        assertTrue(stat.getLen() > 0);     
-        
-        stat = fs.getFileStatus(new Path("output.bz2/part-m-00000.bz2"));
-        assertTrue(stat.getLen() > 0);     
+        FileStatus[] outputFiles = fs.listStatus(new Path("output"),
+                Util.getSuccessMarkerPathFilter());
+        assertTrue(outputFiles[0].getLen() > 0);
+
+        outputFiles = fs.listStatus(new Path("output.bz2"),
+                Util.getSuccessMarkerPathFilter());
+        assertTrue(outputFiles[0].getLen() > 0);
     }
 
     @Test
@@ -504,34 +506,35 @@ public class TestBZip {
         String[] inputData = new String[] {
                 "1\t2\r3\t4"
         };
-        
+
         String inputFileName = "input2.txt";
         Util.createInputFile(cluster, inputFileName, inputData);
-        
-        PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster
-                .getProperties());
+
+        PigServer pig = new PigServer(cluster.getExecType(), properties);
         PigContext pigContext = pig.getPigContext();
         pigContext.getProperties().setProperty( "output.compression.enabled", "true" );
         pigContext.getProperties().setProperty( "output.compression.codec", "org.apache.hadoop.io.compress.BZip2Codec" );
-        
+
         pig.setBatchOn();
         pig.registerQuery("a = load '" +  inputFileName + "';");
         pig.registerQuery("store a into 'output2.bz2';");
         pig.registerQuery("store a into 'output2';");
         pig.executeBatch();
-        
+
         FileSystem fs = FileSystem.get(ConfigurationUtil.toConfiguration(
                 pig.getPigContext().getProperties()));
-        FileStatus stat = fs.getFileStatus(new Path("output2/part-m-00000.bz2"));        
-        assertTrue(stat.getLen() > 0);     
-        
-        stat = fs.getFileStatus(new Path("output2.bz2/part-m-00000.bz2"));
-        assertTrue(stat.getLen() > 0);     
+        FileStatus[] outputFiles = fs.listStatus(new Path("output2"),
+                Util.getSuccessMarkerPathFilter());
+        assertTrue(outputFiles[0].getLen() > 0);
+
+        outputFiles = fs.listStatus(new Path("output2.bz2"),
+                Util.getSuccessMarkerPathFilter());
+        assertTrue(outputFiles[0].getLen() > 0);
     }
-    
-    /** 
+
+    /**
      * Tests that Pig throws an Exception when the input files to be loaded are actually
-     * a result of concatenating 2 or more bz2 files. Pig should not silently ignore part 
+     * a result of concatenating 2 or more bz2 files. Pig should not silently ignore part
      * of the input data.
      */
     @Test (expected=IOException.class)
@@ -550,12 +553,12 @@ public class TestBZip {
                 "1\tb",
                 "2\tbb"
         };
-       
+
         // bzip compressed input file1
         File in1 = File.createTempFile("junit", ".bz2");
         String compressedInputFileName1 = in1.getAbsolutePath();
         in1.deleteOnExit();
-        
+
         // file2
         File in2 = File.createTempFile("junit", ".bz2");
         String compressedInputFileName2 = in2.getAbsolutePath();
@@ -563,9 +566,9 @@ public class TestBZip {
 
         String unCompressedInputFileName = "testRecordDelims-uncomp.txt";
         Util.createInputFile(cluster, unCompressedInputFileName, inputDataMerged);
-        
+
         try {
-            CBZip2OutputStream cos = 
+            CBZip2OutputStream cos =
                 new CBZip2OutputStream(new FileOutputStream(in1));
             for (int i = 0; i < inputData1.length; i++) {
                 StringBuffer sb = new StringBuffer();
@@ -574,8 +577,8 @@ public class TestBZip {
                 cos.write(bytes);
             }
             cos.close();
-            
-            CBZip2OutputStream cos2 = 
+
+            CBZip2OutputStream cos2 =
                 new CBZip2OutputStream(new FileOutputStream(in2));
             for (int i = 0; i < inputData2.length; i++) {
                 StringBuffer sb = new StringBuffer();
@@ -589,56 +592,55 @@ public class TestBZip {
             catInto(compressedInputFileName2, compressedInputFileName1);
             Util.copyFromLocalToCluster(cluster, compressedInputFileName1,
                     compressedInputFileName1);
-            
+
             // pig script to read uncompressed input
             String script = "a = load '" + Util.encodeEscape(unCompressedInputFileName) +"';";
-            PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster
-                    .getProperties());
+            PigServer pig = new PigServer(cluster.getExecType(), properties);
             pig.registerQuery(script);
             Iterator<Tuple> it1 = pig.openIterator("a");
-            
+
             // pig script to read compressed concatenated input
             script = "a = load '" + Util.encodeEscape(compressedInputFileName1) +"';";
             pig.registerQuery(script);
             Iterator<Tuple> it2 = pig.openIterator("a");
-            
+
             while(it1.hasNext()) {
                 Tuple t1 = it1.next();
                 Tuple t2 = it2.next();
-                Assert.assertEquals(t1, t2);
+                assertEquals(t1, t2);
             }
-            
-            Assert.assertFalse(it2.hasNext());
-        
+
+            assertFalse(it2.hasNext());
+
         } finally {
             in1.delete();
             in2.delete();
             Util.deleteFile(cluster, unCompressedInputFileName);
         }
-        
+
     }
-    
+
     /*
      * Concatenate the contents of src file to the contents of dest file
      */
     private void catInto(String src, String dest) throws IOException {
-    	BufferedWriter out = new BufferedWriter(new FileWriter(dest, true));
-    	BufferedReader in = new BufferedReader(new FileReader(src));
-    	String str;
-    	while ((str = in.readLine()) != null) {
-    		out.write(str);
-    	}
-    	in.close();
-    	out.close();
+        BufferedWriter out = new BufferedWriter(new FileWriter(dest, true));
+        BufferedReader in = new BufferedReader(new FileReader(src));
+        String str;
+        while ((str = in.readLine()) != null) {
+            out.write(str);
+        }
+        in.close();
+        out.close();
     }
-    
+
     // See PIG-1714
     @Test
     public void testBzipStoreInMultiQuery3() throws Exception {
         String[] inputData = new String[] {
                 "1\t2\r3\t4"
         };
-        
+
         String inputFileName = "input3.txt";
         Util.createInputFile(cluster, inputFileName, inputData);
 
@@ -649,25 +651,27 @@ public class TestBZip {
                 "a = load '" + inputFileName + "';\n" +
                 "store a into 'output3.bz2';\n" +
                 "store a into 'output3';";
-        
+
         String inputScriptName = "script3.txt";
         PrintWriter pw = new PrintWriter(new FileWriter(inputScriptName));
         pw.println(inputScript);
         pw.close();
-        
-        PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster
-                .getProperties());
-        
+
+        PigServer pig = new PigServer(cluster.getExecType(), properties);
+
         FileInputStream fis = new FileInputStream(inputScriptName);
         pig.registerScript(fis);
-        
+
         FileSystem fs = FileSystem.get(ConfigurationUtil.toConfiguration(
                 pig.getPigContext().getProperties()));
-        FileStatus stat = fs.getFileStatus(new Path("output3/part-m-00000.bz2"));        
-        assertTrue(stat.getLen() > 0);     
-        
-        stat = fs.getFileStatus(new Path("output3.bz2/part-m-00000.bz2"));
-        assertTrue(stat.getLen() > 0);     
+        FileStatus[] outputFiles = fs.listStatus(new Path("output3"),
+                Util.getSuccessMarkerPathFilter());
+        assertTrue(outputFiles[0].getLen() > 0);
+
+        outputFiles = fs.listStatus(new Path("output3.bz2"),
+                Util.getSuccessMarkerPathFilter());
+        assertTrue(outputFiles[0].getLen() > 0);
     }
- 
+
 }
+

Modified: pig/branches/tez/test/org/apache/pig/test/TestBestFitCast.java
URL: http://svn.apache.org/viewvc/pig/branches/tez/test/org/apache/pig/test/TestBestFitCast.java?rev=1573643&r1=1573642&r2=1573643&view=diff
==============================================================================
--- pig/branches/tez/test/org/apache/pig/test/TestBestFitCast.java (original)
+++ pig/branches/tez/test/org/apache/pig/test/TestBestFitCast.java Mon Mar  3 18:03:01 2014
@@ -17,16 +17,15 @@ package org.apache.pig.test;
 
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertTrue;
-import static org.junit.Assert.fail;
 
 import java.io.IOException;
 import java.util.ArrayList;
 import java.util.Arrays;
 import java.util.Iterator;
 import java.util.List;
+import java.util.Properties;
 
 import org.apache.pig.EvalFunc;
-import org.apache.pig.ExecType;
 import org.apache.pig.FuncSpec;
 import org.apache.pig.PigException;
 import org.apache.pig.PigServer;
@@ -40,17 +39,20 @@ import org.apache.pig.impl.util.LogUtils
 import org.junit.After;
 import org.junit.AfterClass;
 import org.junit.Before;
+import org.junit.BeforeClass;
 import org.junit.Test;
 
 public class TestBestFitCast {
-    private PigServer pigServer;
-    private static MiniCluster cluster = MiniCluster.buildCluster();
-    String inputFile, inputFile2;
-    int LOOP_SIZE = 20;
+    private static PigServer pigServer;
+    private static Properties properties;
+    private static MiniGenericCluster cluster;
+
+    private String inputFile, inputFile2;
+    private int LOOP_SIZE = 20;
 
     @Before
     public void setUp() throws Exception {
-        pigServer = new PigServer(ExecType.MAPREDUCE, cluster.getProperties());
+        pigServer = new PigServer(cluster.getExecType(), properties);
         inputFile = "TestBestFitCast-input.txt";
         String[] input = new String[LOOP_SIZE];
         long l = 0;
@@ -73,6 +75,12 @@ public class TestBestFitCast {
         Util.deleteFile(cluster, inputFile2);
     }
 
+    @BeforeClass
+    public static void oneTimeSetUp() throws Exception {
+        cluster = MiniGenericCluster.buildCluster();
+        properties = cluster.getProperties();
+    }
+
     @AfterClass
     public static void oneTimeTearDown() throws Exception {
         cluster.shutDown();
@@ -233,7 +241,7 @@ public class TestBestFitCast {
         try {
             pigServer.registerQuery("A = LOAD '" + inputFile + "' as (x:float, y);");
             pigServer.registerQuery("B = FOREACH A generate x, " + UDF3.class.getName() + "(x,y);");
-            Iterator<Tuple> iter = pigServer.openIterator("B");
+            pigServer.openIterator("B");
         } catch (Exception e) {
             exceptionCaused = true;
             PigException pe = LogUtils.getPigException(e);
@@ -319,7 +327,7 @@ public class TestBestFitCast {
             pigServer.registerQuery("A = LOAD '" + inputFile + "' as (x, y:int);");
             pigServer.registerQuery("B = FOREACH A generate x, " + UDF3.class.getName()
                     + "(x,y, y);");
-            Iterator<Tuple> iter = pigServer.openIterator("B");
+            pigServer.openIterator("B");
         } catch (Exception e) {
             exceptionCaused = true;
             PigException pe = LogUtils.getPigException(e);
@@ -341,7 +349,7 @@ public class TestBestFitCast {
             pigServer.registerQuery("A = LOAD '" + inputFile + "' as (x, y:long);");
             pigServer.registerQuery("B = FOREACH A generate x, " + UDF3.class.getName()
                     + "(x,y, y);");
-            Iterator<Tuple> iter = pigServer.openIterator("B");
+            pigServer.openIterator("B");
         } catch (Exception e) {
             exceptionCaused = true;
             PigException pe = LogUtils.getPigException(e);
@@ -363,7 +371,7 @@ public class TestBestFitCast {
             pigServer.registerQuery("A = LOAD '" + inputFile + "' as (x, y:double);");
             pigServer.registerQuery("B = FOREACH A generate x, " + UDF3.class.getName()
                     + "(x,y, y);");
-            Iterator<Tuple> iter = pigServer.openIterator("B");
+            pigServer.openIterator("B");
         } catch (Exception e) {
             exceptionCaused = true;
             PigException pe = LogUtils.getPigException(e);
@@ -476,7 +484,7 @@ public class TestBestFitCast {
             pigServer.registerQuery("A = LOAD '" + inputFile2 + "' as (x:float, y, z:int);");
             pigServer.registerQuery("B = FOREACH A generate x, " + UDF3.class.getName()
                     + "(x,y, y);");
-            Iterator<Tuple> iter = pigServer.openIterator("B");
+            pigServer.openIterator("B");
         } catch (Exception e) {
             exceptionCaused = true;
             PigException pe = LogUtils.getPigException(e);

Modified: pig/branches/tez/test/org/apache/pig/test/TestBinaryExpressionOps.java
URL: http://svn.apache.org/viewvc/pig/branches/tez/test/org/apache/pig/test/TestBinaryExpressionOps.java?rev=1573643&r1=1573642&r2=1573643&view=diff
==============================================================================
--- pig/branches/tez/test/org/apache/pig/test/TestBinaryExpressionOps.java (original)
+++ pig/branches/tez/test/org/apache/pig/test/TestBinaryExpressionOps.java Mon Mar  3 18:03:01 2014
@@ -1,4 +1,3 @@
-
 /*
  * Licensed to the Apache Software Foundation (ASF) under one
  * or more contributor license agreements.  See the NOTICE file
@@ -23,8 +22,8 @@ package org.apache.pig.test;
 import static org.junit.Assert.assertEquals;
 
 import java.util.Iterator;
+import java.util.Properties;
 
-import org.apache.pig.ExecType;
 import org.apache.pig.PigServer;
 import org.apache.pig.data.Tuple;
 import org.junit.AfterClass;
@@ -32,12 +31,11 @@ import org.junit.BeforeClass;
 import org.junit.Test;
 
 public class TestBinaryExpressionOps {
-
-    private static MiniCluster cluster = MiniCluster.buildCluster();
-    
+    private static Properties properties;
+    private static MiniGenericCluster cluster;
     private static final String INPUT_1 = "input1";
     private static final String INPUT_2 = "input2";
-    
+
     @BeforeClass
     public static void setUpBeforeClass() throws Exception {
         String[] inputData1 = new String[] {
@@ -46,34 +44,40 @@ public class TestBinaryExpressionOps {
         String[] inputData2 = new String[] {
                 "id1\t2", "id2\t2"
         };
-        Util.createInputFile(cluster, INPUT_1, inputData1);        
+        Util.createInputFile(cluster, INPUT_1, inputData1);
         Util.createInputFile(cluster, INPUT_2, inputData2);
     }
 
+    @BeforeClass
+    public static void oneTimeSetUp() throws Exception {
+        cluster = MiniGenericCluster.buildCluster();
+        properties = cluster.getProperties();
+    }
+
     @AfterClass
     public static void tearDownAfterClass() throws Exception {
         cluster.shutDown();
     }
-    
+
     @Test
     public void testArithmeticOperators() throws Exception {
-        PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster.getProperties());
-        
+        PigServer pig = new PigServer(cluster.getExecType(), properties);
+
         pig.registerQuery("A = LOAD '" + INPUT_1 + "' AS (id:chararray, val:long);");
         pig.registerQuery("B = LOAD '" + INPUT_2 + "' AS (id:chararray, val:long);");
         pig.registerQuery("C = COGROUP A BY id, B BY id;");
         pig.registerQuery("D = FOREACH C GENERATE group, SUM(B.val), SUM(A.val), "
-                + "(SUM(A.val) - SUM(B.val)), (SUM(A.val) + SUM(B.val)), " 
-                + "(SUM(A.val) * SUM(B.val)), (SUM(A.val) / SUM(B.val)), " 
+                + "(SUM(A.val) - SUM(B.val)), (SUM(A.val) + SUM(B.val)), "
+                + "(SUM(A.val) * SUM(B.val)), (SUM(A.val) / SUM(B.val)), "
                 + "(SUM(A.val) % SUM(B.val)), (SUM(A.val) < 0 ? SUM(A.val) : SUM(B.val));");
-               
+
         String[] expectedResults = new String[] {"(id1,2,,,,,,,)", "(id2,2,10,8,12,20,5,0,2)"};
-        Iterator<Tuple> iter = pig.openIterator("D");             
+        Iterator<Tuple> iter = pig.openIterator("D");
         int counter = 0;
         while (iter.hasNext()) { 
-            assertEquals(expectedResults[counter++], iter.next().toString());      
+            assertEquals(expectedResults[counter++], iter.next().toString());
         }
         assertEquals(expectedResults.length, counter);
     }
-    
+
 }

Modified: pig/branches/tez/test/org/apache/pig/test/TestBuiltin.java
URL: http://svn.apache.org/viewvc/pig/branches/tez/test/org/apache/pig/test/TestBuiltin.java?rev=1573643&r1=1573642&r2=1573643&view=diff
==============================================================================
--- pig/branches/tez/test/org/apache/pig/test/TestBuiltin.java (original)
+++ pig/branches/tez/test/org/apache/pig/test/TestBuiltin.java Mon Mar  3 18:03:01 2014
@@ -130,17 +130,16 @@ import org.joda.time.DateTimeZone;
 import org.junit.AfterClass;
 import org.junit.Assert;
 import org.junit.Before;
+import org.junit.BeforeClass;
 import org.junit.Test;
 
 public class TestBuiltin {
+    private static PigServer pigServer;
+    private static Properties properties;
+    private static MiniGenericCluster cluster;
 
-    PigServer pigServer;
-
-    // This should only be used when absolutely necessary -- eg, when using ReadToEndLoader.
-    private static MiniCluster cluster = MiniCluster.buildCluster();
-
-    TupleFactory tupleFactory = TupleFactory.getInstance();
-    BagFactory bagFactory = DefaultBagFactory.getInstance();
+    private TupleFactory tupleFactory = TupleFactory.getInstance();
+    private BagFactory bagFactory = DefaultBagFactory.getInstance();
 
     // some inputs
     private static Integer[] intInput = { 3, 1, 2, 4, 5, 7, null, 6, 8, 9, 10 };
@@ -345,14 +344,17 @@ public class TestBuiltin {
         DateTimeZone.setDefault(DateTimeZone.forOffsetMillis(DateTimeZone.UTC.getOffset(null)));
     }
 
+    @BeforeClass
+    public static void oneTimeSetUp() throws Exception {
+        cluster = MiniGenericCluster.buildCluster();
+        properties = cluster.getProperties();
+    }
+
     @AfterClass
     public static void shutDown() {
         cluster.shutDown();
     }
 
-    /**
-     *
-     */
     private void setupEvalFuncMap() {
         for (String[] aggGroup : aggs) {
             for (String agg : aggGroup) {
@@ -2022,13 +2024,6 @@ public class TestBuiltin {
         assertEquals(0.582222509739582, (Double)ans.get(2) ,0.0005);
     }
 
-    private void checkItemsGT(Iterable<Tuple> tuples, int field, int limit) throws ExecException {
-        for (Tuple t : tuples) {
-            Long val = (Long) t.get(field);
-            assertTrue("Value "+ val + " exceeded the expected limit", val > limit);
-        }
-    }
-
     @Test
     public void testToBag() throws Exception {
         //TEST TOBAG
@@ -2507,7 +2502,7 @@ public class TestBuiltin {
 
         String input3 = "this:has:a:trailing:colon:\n";
         int arity3 = 6;
-        Util.createInputFile(cluster, "input.txt", new String[] {input2});
+        Util.createInputFile(cluster, "input.txt", new String[] {input3});
         LoadFunc p3 = new ReadToEndLoader(new PigStorage(":"), ConfigurationUtil.
             toConfiguration(cluster.getProperties()), "input.txt", 0);
         Tuple f3 = p3.getNext();
@@ -2545,10 +2540,9 @@ public class TestBuiltin {
         assertTrue(f3 == null);
     }
 
-    @SuppressWarnings("unchecked")
     @Test
     public void testSFPig() throws Exception {
-        PigServer mrPigServer = new PigServer(ExecType.MAPREDUCE);
+        PigServer mrPigServer = new PigServer(cluster.getExecType(), properties);
         String inputStr = "amy\tbob\tcharlene\tdavid\terin\tfrank";
         Util.createInputFile(cluster, "testSFPig-input.txt", new String[]
                                                                     {inputStr});
@@ -2578,7 +2572,6 @@ public class TestBuiltin {
      * unit tests are done in TestStringUDFs
      */
     @Test
-    @SuppressWarnings("unchecked")
     public void testStringUDFs() throws Exception {
         String inputStr = "amy smith ";
         File inputFile = Util.createInputFile("tmp", "testStrUDFsIn.txt", new String[] {inputStr});

Modified: pig/branches/tez/test/org/apache/pig/test/TestCharArrayToNumeric.java
URL: http://svn.apache.org/viewvc/pig/branches/tez/test/org/apache/pig/test/TestCharArrayToNumeric.java?rev=1573643&r1=1573642&r2=1573643&view=diff
==============================================================================
--- pig/branches/tez/test/org/apache/pig/test/TestCharArrayToNumeric.java (original)
+++ pig/branches/tez/test/org/apache/pig/test/TestCharArrayToNumeric.java Mon Mar  3 18:03:01 2014
@@ -25,9 +25,9 @@ import java.util.HashMap;
 import java.util.Iterator;
 import java.util.List;
 import java.util.Map;
+import java.util.Properties;
 import java.util.Random;
 
-import org.apache.pig.ExecType;
 import org.apache.pig.PigServer;
 import org.apache.pig.backend.executionengine.ExecException;
 import org.apache.pig.backend.hadoop.executionengine.physicalLayer.PhysicalOperator;
@@ -40,13 +40,14 @@ import org.apache.pig.impl.plan.NodeIdGe
 import org.apache.pig.impl.plan.OperatorKey;
 import org.junit.AfterClass;
 import org.junit.Before;
+import org.junit.BeforeClass;
 import org.junit.Test;
 
 public class TestCharArrayToNumeric {
-    private Double dummyDouble = null;
-    private Float dummyFloat = null;
-    private Long dummyLong = null;
-    private Integer dummyInteger = null;
+    private static PigServer pig;
+    private static Properties properties;
+    private static MiniGenericCluster cluster;
+
     private Double MaxDouble = Double.MIN_VALUE;
     private Double MinDouble = Double.MIN_VALUE;
     private Float MaxFloat = Float.MAX_VALUE;
@@ -56,12 +57,15 @@ public class TestCharArrayToNumeric {
     private Integer MaxInteger = Integer.MAX_VALUE;
     private Integer MinInteger = Integer.MIN_VALUE;
 
-    static MiniCluster cluster = MiniCluster.buildCluster();
-    PigServer pig;
-
     @Before
     public void setUp() throws Exception {
-        pig = new PigServer(ExecType.MAPREDUCE, cluster.getProperties());
+        pig = new PigServer(cluster.getExecType(), properties);
+    }
+
+    @BeforeClass
+    public static void oneTimeSetUp() throws Exception {
+        cluster = MiniGenericCluster.buildCluster();
+        properties = cluster.getProperties();
     }
 
     @AfterClass

Modified: pig/branches/tez/test/org/apache/pig/test/TestCompressedFiles.java
URL: http://svn.apache.org/viewvc/pig/branches/tez/test/org/apache/pig/test/TestCompressedFiles.java?rev=1573643&r1=1573642&r2=1573643&view=diff
==============================================================================
--- pig/branches/tez/test/org/apache/pig/test/TestCompressedFiles.java (original)
+++ pig/branches/tez/test/org/apache/pig/test/TestCompressedFiles.java Mon Mar  3 18:03:01 2014
@@ -24,10 +24,10 @@ import static org.junit.Assert.assertTru
 import java.io.File;
 import java.io.FileOutputStream;
 import java.util.Iterator;
+import java.util.Properties;
 import java.util.Random;
 import java.util.zip.GZIPOutputStream;
 
-import org.apache.pig.ExecType;
 import org.apache.pig.PigServer;
 import org.apache.pig.builtin.DIFF;
 import org.apache.pig.data.BagFactory;
@@ -37,13 +37,16 @@ import org.apache.pig.test.utils.TestHel
 import org.junit.After;
 import org.junit.AfterClass;
 import org.junit.Before;
+import org.junit.BeforeClass;
 import org.junit.Test;
 
 public class TestCompressedFiles {
-    static MiniCluster cluster = MiniCluster.buildCluster();
+    private static PigServer pig;
+    private static Properties properties;
+    private static MiniGenericCluster cluster;
 
-    File datFile;
-    File gzFile;
+    private File datFile;
+    private File gzFile;
 
     @Before
     public void setUp() throws Exception {
@@ -74,6 +77,12 @@ public class TestCompressedFiles {
         gzFile.delete();
     }
 
+    @BeforeClass
+    public static void oneTimeSetUp() throws Exception {
+        cluster = MiniGenericCluster.buildCluster();
+        properties = cluster.getProperties();
+    }
+
     @AfterClass
     public static void oneTimeTearDown() throws Exception {
         cluster.shutDown();
@@ -81,7 +90,7 @@ public class TestCompressedFiles {
 
     @Test
     public void testCompressed1() throws Throwable {
-        PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster.getProperties());
+        pig = new PigServer(cluster.getExecType(), properties);
         pig.registerQuery("A = foreach (cogroup (load '"
                 + Util.generateURI(gzFile.toString(), pig.getPigContext())
                 + "') by $1, (load '"
@@ -94,7 +103,7 @@ public class TestCompressedFiles {
 
     @Test
     public void testCompressed2() throws Throwable {
-        PigServer pig = new PigServer(ExecType.MAPREDUCE, cluster.getProperties());
+        pig = new PigServer(cluster.getExecType(), properties);
         pig.registerQuery("A = load '"
                 + Util.generateURI(gzFile.toString(), pig.getPigContext())
                 + "';");

Modified: pig/branches/tez/test/tez-tests
URL: http://svn.apache.org/viewvc/pig/branches/tez/test/tez-tests?rev=1573643&r1=1573642&r2=1573643&view=diff
==============================================================================
--- pig/branches/tez/test/tez-tests (original)
+++ pig/branches/tez/test/tez-tests Mon Mar  3 18:03:01 2014
@@ -11,5 +11,12 @@
 **/TestNestedForeach.java
 **/TestEvalPipeline.java
 **/TestPigServer.java
+**/TestAlgebraicEval.java
+**/TestBestFitCast.java
+**/TestBinaryExpressionOps.java
+**/TestBuiltin.java
+**/TestBZip.java
+**/TestCompressedFiles.java
+**/TestCharArrayToNumeric.java
 ## TODO: Runs fine individually. Hangs with file.out.index not found when run together. Likely Tez Bug
 ##**/TestSecondarySortTez.java