Skip to content

Commit 181ec50

Browse files
ScrapCodespwendell
authored andcommitted
[java8API] SPARK-964 Investigate the potential for using JDK 8 lambda expressions for the Java/Scala APIs
Author: Prashant Sharma <prashant.s@imaginea.com> Author: Patrick Wendell <pwendell@gmail.com> Closes #17 from ScrapCodes/java8-lambdas and squashes the following commits: 95850e6 [Patrick Wendell] Some doc improvements and build changes to the Java 8 patch. 85a954e [Prashant Sharma] Nit. import orderings. 673f7ac [Prashant Sharma] Added support for -java-home as well 80a13e8 [Prashant Sharma] Used fake class tag syntax 26eb3f6 [Prashant Sharma] Patrick's comments on PR. 35d8d79 [Prashant Sharma] Specified java 8 building in the docs 31d4cd6 [Prashant Sharma] Maven build to support -Pjava8-tests flag. 4ab87d3 [Prashant Sharma] Review feedback on the pr c33dc2c [Prashant Sharma] SPARK-964, Java 8 API Support.
1 parent b14ede7 commit 181ec50

Some content is hidden

Large Commits have some content hidden by default. Use the searchbox below for content that may be hidden.

52 files changed

+1946
-551
lines changed

core/src/main/scala/org/apache/spark/api/java/function/DoubleFlatMapFunction.scala renamed to core/src/main/java/org/apache/spark/api/java/function/DoubleFlatMapFunction.java

Lines changed: 4 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -15,16 +15,13 @@
1515
* limitations under the License.
1616
*/
1717

18-
package org.apache.spark.api.java.function
18+
package org.apache.spark.api.java.function;
1919

20-
import java.lang.{Double => JDouble, Iterable => JIterable}
20+
import java.io.Serializable;
2121

2222
/**
2323
* A function that returns zero or more records of type Double from each input record.
2424
*/
25-
// DoubleFlatMapFunction does not extend FlatMapFunction because flatMap is
26-
// overloaded for both FlatMapFunction and DoubleFlatMapFunction.
27-
abstract class DoubleFlatMapFunction[T] extends WrappedFunction1[T, JIterable[JDouble]]
28-
with Serializable {
29-
// Intentionally left blank
25+
public interface DoubleFlatMapFunction<T> extends Serializable {
26+
public Iterable<Double> call(T t) throws Exception;
3027
}
Lines changed: 27 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,27 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
18+
package org.apache.spark.api.java.function;
19+
20+
import java.io.Serializable;
21+
22+
/**
23+
* A function that returns Doubles, and can be used to construct DoubleRDDs.
24+
*/
25+
public interface DoubleFunction<T> extends Serializable {
26+
public double call(T t) throws Exception;
27+
}

core/src/main/scala/org/apache/spark/api/java/function/FlatMapFunction.scala renamed to core/src/main/java/org/apache/spark/api/java/function/FlatMapFunction.java

Lines changed: 5 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -15,13 +15,13 @@
1515
* limitations under the License.
1616
*/
1717

18-
package org.apache.spark.api.java.function
18+
package org.apache.spark.api.java.function;
1919

20-
import scala.reflect.ClassTag
20+
import java.io.Serializable;
2121

2222
/**
2323
* A function that returns zero or more output records from each input record.
2424
*/
25-
abstract class FlatMapFunction[T, R] extends Function[T, java.lang.Iterable[R]] {
26-
def elementType(): ClassTag[R] = ClassTag.Any.asInstanceOf[ClassTag[R]]
27-
}
25+
public interface FlatMapFunction<T, R> extends Serializable {
26+
public Iterable<R> call(T t) throws Exception;
27+
}

core/src/main/scala/org/apache/spark/api/java/function/FlatMapFunction2.scala renamed to core/src/main/java/org/apache/spark/api/java/function/FlatMapFunction2.java

Lines changed: 5 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -15,13 +15,13 @@
1515
* limitations under the License.
1616
*/
1717

18-
package org.apache.spark.api.java.function
18+
package org.apache.spark.api.java.function;
1919

20-
import scala.reflect.ClassTag
20+
import java.io.Serializable;
2121

2222
/**
2323
* A function that takes two inputs and returns zero or more output records.
2424
*/
25-
abstract class FlatMapFunction2[A, B, C] extends Function2[A, B, java.lang.Iterable[C]] {
26-
def elementType() : ClassTag[C] = ClassTag.Any.asInstanceOf[ClassTag[C]]
27-
}
25+
public interface FlatMapFunction2<T1, T2, R> extends Serializable {
26+
public Iterable<R> call(T1 t1, T2 t2) throws Exception;
27+
}

core/src/main/scala/org/apache/spark/api/java/function/Function.scala renamed to core/src/main/java/org/apache/spark/api/java/function/Function.java

Lines changed: 5 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -15,17 +15,15 @@
1515
* limitations under the License.
1616
*/
1717

18-
package org.apache.spark.api.java.function
18+
package org.apache.spark.api.java.function;
1919

20-
import scala.reflect.ClassTag
21-
import org.apache.spark.api.java.JavaSparkContext
20+
import java.io.Serializable;
2221

2322
/**
24-
* Base class for functions whose return types do not create special RDDs. PairFunction and
23+
* Base interface for functions whose return types do not create special RDDs. PairFunction and
2524
* DoubleFunction are handled separately, to allow PairRDDs and DoubleRDDs to be constructed
2625
* when mapping RDDs of other types.
2726
*/
28-
abstract class Function[T, R] extends WrappedFunction1[T, R] with Serializable {
29-
def returnType(): ClassTag[R] = JavaSparkContext.fakeClassTag
27+
public interface Function<T1, R> extends Serializable {
28+
public R call(T1 v1) throws Exception;
3029
}
31-

core/src/main/scala/org/apache/spark/api/java/function/Function2.scala renamed to core/src/main/java/org/apache/spark/api/java/function/Function2.java

Lines changed: 4 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -15,15 +15,13 @@
1515
* limitations under the License.
1616
*/
1717

18-
package org.apache.spark.api.java.function
18+
package org.apache.spark.api.java.function;
1919

20-
import scala.reflect.ClassTag
21-
import org.apache.spark.api.java.JavaSparkContext
20+
import java.io.Serializable;
2221

2322
/**
2423
* A two-argument function that takes arguments of type T1 and T2 and returns an R.
2524
*/
26-
abstract class Function2[T1, T2, R] extends WrappedFunction2[T1, T2, R] with Serializable {
27-
def returnType(): ClassTag[R] = JavaSparkContext.fakeClassTag
25+
public interface Function2<T1, T2, R> extends Serializable {
26+
public R call(T1 v1, T2 v2) throws Exception;
2827
}
29-

core/src/main/scala/org/apache/spark/api/java/function/Function3.scala renamed to core/src/main/java/org/apache/spark/api/java/function/Function3.java

Lines changed: 4 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -15,14 +15,13 @@
1515
* limitations under the License.
1616
*/
1717

18-
package org.apache.spark.api.java.function
18+
package org.apache.spark.api.java.function;
1919

20-
import org.apache.spark.api.java.JavaSparkContext
21-
import scala.reflect.ClassTag
20+
import java.io.Serializable;
2221

2322
/**
2423
* A three-argument function that takes arguments of type T1, T2 and T3 and returns an R.
2524
*/
26-
abstract class Function3[T1, T2, T3, R] extends WrappedFunction3[T1, T2, T3, R] with Serializable {
27-
def returnType(): ClassTag[R] = JavaSparkContext.fakeClassTag
25+
public interface Function3<T1, T2, T3, R> extends Serializable {
26+
public R call(T1 v1, T2 v2, T3 v3) throws Exception;
2827
}
Lines changed: 30 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,30 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
18+
package org.apache.spark.api.java.function;
19+
20+
import java.io.Serializable;
21+
22+
import scala.Tuple2;
23+
24+
/**
25+
* A function that returns zero or more key-value pair records from each input record. The
26+
* key-value pairs are represented as scala.Tuple2 objects.
27+
*/
28+
public interface PairFlatMapFunction<T, K, V> extends Serializable {
29+
public Iterable<Tuple2<K, V>> call(T t) throws Exception;
30+
}
Lines changed: 29 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,29 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
18+
package org.apache.spark.api.java.function;
19+
20+
import java.io.Serializable;
21+
22+
import scala.Tuple2;
23+
24+
/**
25+
* A function that returns key-value pairs (Tuple2<K, V>), and can be used to construct PairRDDs.
26+
*/
27+
public interface PairFunction<T, K, V> extends Serializable {
28+
public Tuple2<K, V> call(T t) throws Exception;
29+
}
Lines changed: 27 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,27 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
18+
package org.apache.spark.api.java.function;
19+
20+
import java.io.Serializable;
21+
22+
/**
23+
* A function with no return value.
24+
*/
25+
public interface VoidFunction<T> extends Serializable {
26+
public void call(T t) throws Exception;
27+
}

0 commit comments

Comments
 (0)