Skip to content

Commit 36e0753

Browse files
author
Andrew Or
committed
Merge branch 'master' of github.com:apache/spark into yarn-settings
2 parents dcd1316 + 2b9b726 commit 36e0753

File tree

151 files changed

+3697
-1920
lines changed

Some content is hidden

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

151 files changed

+3697
-1920
lines changed

.gitignore

Lines changed: 3 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -5,6 +5,7 @@
55
*.ipr
66
*.iml
77
*.iws
8+
*.pyc
89
.idea/
910
.idea_modules/
1011
sbt/*.jar
@@ -49,6 +50,8 @@ dependency-reduced-pom.xml
4950
checkpoint
5051
derby.log
5152
dist/
53+
dev/create-release/*txt
54+
dev/create-release/*new
5255
spark-*-bin-*.tgz
5356
unit-tests.log
5457
/lib/

LICENSE

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -646,7 +646,8 @@ THE SOFTWARE.
646646

647647
========================================================================
648648
For Scala Interpreter classes (all .scala files in repl/src/main/scala
649-
except for Main.Scala, SparkHelper.scala and ExecutorClassLoader.scala):
649+
except for Main.Scala, SparkHelper.scala and ExecutorClassLoader.scala),
650+
and for SerializableMapWrapper in JavaUtils.scala:
650651
========================================================================
651652

652653
Copyright (c) 2002-2013 EPFL

assembly/pom.xml

Lines changed: 0 additions & 10 deletions
Original file line numberDiff line numberDiff line change
@@ -169,16 +169,6 @@
169169
</build>
170170

171171
<profiles>
172-
<profile>
173-
<id>yarn-alpha</id>
174-
<dependencies>
175-
<dependency>
176-
<groupId>org.apache.spark</groupId>
177-
<artifactId>spark-yarn-alpha_${scala.binary.version}</artifactId>
178-
<version>${project.version}</version>
179-
</dependency>
180-
</dependencies>
181-
</profile>
182172
<profile>
183173
<id>yarn</id>
184174
<dependencies>

bin/beeline.cmd

Lines changed: 21 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,21 @@
1+
@echo off
2+
3+
rem
4+
rem Licensed to the Apache Software Foundation (ASF) under one or more
5+
rem contributor license agreements. See the NOTICE file distributed with
6+
rem this work for additional information regarding copyright ownership.
7+
rem The ASF licenses this file to You under the Apache License, Version 2.0
8+
rem (the "License"); you may not use this file except in compliance with
9+
rem the License. You may obtain a copy of the License at
10+
rem
11+
rem http://www.apache.org/licenses/LICENSE-2.0
12+
rem
13+
rem Unless required by applicable law or agreed to in writing, software
14+
rem distributed under the License is distributed on an "AS IS" BASIS,
15+
rem WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16+
rem See the License for the specific language governing permissions and
17+
rem limitations under the License.
18+
rem
19+
20+
set SPARK_HOME=%~dp0..
21+
cmd /V /E /C %SPARK_HOME%\bin\spark-class.cmd org.apache.hive.beeline.BeeLine %*

core/src/main/java/org/apache/spark/SparkJobInfo.java

Lines changed: 3 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -17,13 +17,15 @@
1717

1818
package org.apache.spark;
1919

20+
import java.io.Serializable;
21+
2022
/**
2123
* Exposes information about Spark Jobs.
2224
*
2325
* This interface is not designed to be implemented outside of Spark. We may add additional methods
2426
* which may break binary compatibility with outside implementations.
2527
*/
26-
public interface SparkJobInfo {
28+
public interface SparkJobInfo extends Serializable {
2729
int jobId();
2830
int[] stageIds();
2931
JobExecutionStatus status();

core/src/main/java/org/apache/spark/SparkStageInfo.java

Lines changed: 3 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -17,13 +17,15 @@
1717

1818
package org.apache.spark;
1919

20+
import java.io.Serializable;
21+
2022
/**
2123
* Exposes information about Spark Stages.
2224
*
2325
* This interface is not designed to be implemented outside of Spark. We may add additional methods
2426
* which may break binary compatibility with outside implementations.
2527
*/
26-
public interface SparkStageInfo {
28+
public interface SparkStageInfo extends Serializable {
2729
int stageId();
2830
int currentAttemptId();
2931
long submissionTime();

core/src/main/resources/org/apache/spark/ui/static/webui.css

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -171,6 +171,6 @@ span.additional-metric-title {
171171

172172
/* Hide all additional metrics by default. This is done here rather than using JavaScript to
173173
* avoid slow page loads for stage pages with large numbers (e.g., thousands) of tasks. */
174-
.scheduler_delay, .gc_time, .deserialization_time, .serialization_time, .getting_result_time {
174+
.scheduler_delay, .deserialization_time, .serialization_time, .getting_result_time {
175175
display: none;
176176
}

core/src/main/scala/org/apache/spark/SparkContext.scala

Lines changed: 4 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -1758,7 +1758,7 @@ object SparkContext extends Logging {
17581758

17591759
@deprecated("Replaced by implicit functions in WritableConverter. This is kept here only for " +
17601760
"backward compatibility.", "1.3.0")
1761-
def writableWritableConverter[T <: Writable]() =
1761+
def writableWritableConverter[T <: Writable](): WritableConverter[T] =
17621762
WritableConverter.writableWritableConverter()
17631763

17641764
/**
@@ -2017,15 +2017,15 @@ object WritableConverter {
20172017
simpleWritableConverter[Boolean, BooleanWritable](_.get)
20182018

20192019
implicit def bytesWritableConverter(): WritableConverter[Array[Byte]] = {
2020-
simpleWritableConverter[Array[Byte], BytesWritable](bw =>
2020+
simpleWritableConverter[Array[Byte], BytesWritable] { bw =>
20212021
// getBytes method returns array which is longer then data to be returned
20222022
Arrays.copyOfRange(bw.getBytes, 0, bw.getLength)
2023-
)
2023+
}
20242024
}
20252025

20262026
implicit def stringWritableConverter(): WritableConverter[String] =
20272027
simpleWritableConverter[String, Text](_.toString)
20282028

2029-
implicit def writableWritableConverter[T <: Writable]() =
2029+
implicit def writableWritableConverter[T <: Writable](): WritableConverter[T] =
20302030
new WritableConverter[T](_.runtimeClass.asInstanceOf[Class[T]], _.asInstanceOf[T])
20312031
}

core/src/main/scala/org/apache/spark/api/java/JavaRDDLike.scala

Lines changed: 10 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -211,8 +211,9 @@ trait JavaRDDLike[T, This <: JavaRDDLike[T, This]] extends Serializable {
211211
* Return an RDD of grouped elements. Each group consists of a key and a sequence of elements
212212
* mapping to that key.
213213
*/
214-
def groupBy[K](f: JFunction[T, K]): JavaPairRDD[K, JIterable[T]] = {
215-
implicit val ctagK: ClassTag[K] = fakeClassTag
214+
def groupBy[U](f: JFunction[T, U]): JavaPairRDD[U, JIterable[T]] = {
215+
// The type parameter is U instead of K in order to work around a compiler bug; see SPARK-4459
216+
implicit val ctagK: ClassTag[U] = fakeClassTag
216217
implicit val ctagV: ClassTag[JList[T]] = fakeClassTag
217218
JavaPairRDD.fromRDD(groupByResultToJava(rdd.groupBy(f)(fakeClassTag)))
218219
}
@@ -221,10 +222,11 @@ trait JavaRDDLike[T, This <: JavaRDDLike[T, This]] extends Serializable {
221222
* Return an RDD of grouped elements. Each group consists of a key and a sequence of elements
222223
* mapping to that key.
223224
*/
224-
def groupBy[K](f: JFunction[T, K], numPartitions: Int): JavaPairRDD[K, JIterable[T]] = {
225-
implicit val ctagK: ClassTag[K] = fakeClassTag
225+
def groupBy[U](f: JFunction[T, U], numPartitions: Int): JavaPairRDD[U, JIterable[T]] = {
226+
// The type parameter is U instead of K in order to work around a compiler bug; see SPARK-4459
227+
implicit val ctagK: ClassTag[U] = fakeClassTag
226228
implicit val ctagV: ClassTag[JList[T]] = fakeClassTag
227-
JavaPairRDD.fromRDD(groupByResultToJava(rdd.groupBy(f, numPartitions)(fakeClassTag[K])))
229+
JavaPairRDD.fromRDD(groupByResultToJava(rdd.groupBy(f, numPartitions)(fakeClassTag[U])))
228230
}
229231

230232
/**
@@ -458,8 +460,9 @@ trait JavaRDDLike[T, This <: JavaRDDLike[T, This]] extends Serializable {
458460
/**
459461
* Creates tuples of the elements in this RDD by applying `f`.
460462
*/
461-
def keyBy[K](f: JFunction[T, K]): JavaPairRDD[K, T] = {
462-
implicit val ctag: ClassTag[K] = fakeClassTag
463+
def keyBy[U](f: JFunction[T, U]): JavaPairRDD[U, T] = {
464+
// The type parameter is U instead of K in order to work around a compiler bug; see SPARK-4459
465+
implicit val ctag: ClassTag[U] = fakeClassTag
463466
JavaPairRDD.fromRDD(rdd.keyBy(f))
464467
}
465468

core/src/main/scala/org/apache/spark/api/java/JavaUtils.scala

Lines changed: 60 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -19,7 +19,8 @@ package org.apache.spark.api.java
1919

2020
import com.google.common.base.Optional
2121

22-
import scala.collection.convert.Wrappers.MapWrapper
22+
import java.{util => ju}
23+
import scala.collection.mutable
2324

2425
private[spark] object JavaUtils {
2526
def optionToOptional[T](option: Option[T]): Optional[T] =
@@ -32,7 +33,64 @@ private[spark] object JavaUtils {
3233
def mapAsSerializableJavaMap[A, B](underlying: collection.Map[A, B]) =
3334
new SerializableMapWrapper(underlying)
3435

36+
// Implementation is copied from scala.collection.convert.Wrappers.MapWrapper,
37+
// but implements java.io.Serializable. It can't just be subclassed to make it
38+
// Serializable since the MapWrapper class has no no-arg constructor. This class
39+
// doesn't need a no-arg constructor though.
3540
class SerializableMapWrapper[A, B](underlying: collection.Map[A, B])
36-
extends MapWrapper(underlying) with java.io.Serializable
41+
extends ju.AbstractMap[A, B] with java.io.Serializable { self =>
3742

43+
override def size = underlying.size
44+
45+
override def get(key: AnyRef): B = try {
46+
underlying get key.asInstanceOf[A] match {
47+
case None => null.asInstanceOf[B]
48+
case Some(v) => v
49+
}
50+
} catch {
51+
case ex: ClassCastException => null.asInstanceOf[B]
52+
}
53+
54+
override def entrySet: ju.Set[ju.Map.Entry[A, B]] = new ju.AbstractSet[ju.Map.Entry[A, B]] {
55+
def size = self.size
56+
57+
def iterator = new ju.Iterator[ju.Map.Entry[A, B]] {
58+
val ui = underlying.iterator
59+
var prev : Option[A] = None
60+
61+
def hasNext = ui.hasNext
62+
63+
def next() = {
64+
val (k, v) = ui.next
65+
prev = Some(k)
66+
new ju.Map.Entry[A, B] {
67+
import scala.util.hashing.byteswap32
68+
def getKey = k
69+
def getValue = v
70+
def setValue(v1 : B) = self.put(k, v1)
71+
override def hashCode = byteswap32(k.hashCode) + (byteswap32(v.hashCode) << 16)
72+
override def equals(other: Any) = other match {
73+
case e: ju.Map.Entry[_, _] => k == e.getKey && v == e.getValue
74+
case _ => false
75+
}
76+
}
77+
}
78+
79+
def remove() {
80+
prev match {
81+
case Some(k) =>
82+
underlying match {
83+
case mm: mutable.Map[a, _] =>
84+
mm remove k
85+
prev = None
86+
case _ =>
87+
throw new UnsupportedOperationException("remove")
88+
}
89+
case _ =>
90+
throw new IllegalStateException("next must be called at least once before remove")
91+
}
92+
}
93+
}
94+
}
95+
}
3896
}

0 commit comments

Comments
 (0)