Skip to content

Commit e83cd8f

Browse files
committed
Changes to allow re-use of test applications
1 parent be42f35 commit e83cd8f

File tree

4 files changed

+20
-6
lines changed

4 files changed

+20
-6
lines changed

dev/audit-release/audit_release.py

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -114,6 +114,7 @@ def ensure_path_not_present(x):
114114
os.environ["SPARK_VERSION"] = RELEASE_VERSION
115115
os.environ["SCALA_VERSION"] = SCALA_VERSION
116116
os.environ["SPARK_RELEASE_REPOSITORY"] = RELEASE_REPOSITORY
117+
os.environ["SPARK_AUDIT_MASTER"] = "local"
117118
for module in modules:
118119
os.environ["SPARK_MODULE"] = module
119120
ret = run_cmd("sbt clean update", exit_on_failure=False)

dev/audit-release/sbt_app_core/src/main/scala/SparkApp.scala

Lines changed: 6 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -24,8 +24,13 @@ import org.apache.spark.SparkContext._
2424

2525
object SimpleApp {
2626
def main(args: Array[String]) {
27+
val conf = sys.env.get("SPARK_AUDIT_MASTER") match {
28+
case Some(master) => new SparkConf().setAppName("Simple Spark App").setMaster(master)
29+
case None => new SparkConf().setAppName("Simple Spark App")
30+
}
2731
val logFile = "input.txt"
28-
val sc = new SparkContext("local", "Simple App")
32+
val sc = new SparkContext(conf)
33+
SparkContext.jarOfClass(this.getClass).foreach(sc.addJar)
2934
val logData = sc.textFile(logFile, 2).cache()
3035
val numAs = logData.filter(line => line.contains("a")).count()
3136
val numBs = logData.filter(line => line.contains("b")).count()

dev/audit-release/sbt_app_graphx/src/main/scala/GraphxApp.scala

Lines changed: 8 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -17,14 +17,20 @@
1717

1818
package main.scala
1919

20-
import org.apache.spark.SparkContext
20+
import org.apache.spark.{SparkContext, SparkConf}
2121
import org.apache.spark.SparkContext._
2222
import org.apache.spark.graphx._
2323
import org.apache.spark.rdd.RDD
2424

2525
object GraphXApp {
2626
def main(args: Array[String]) {
27-
val sc = new SparkContext("local", "Simple GraphX App")
27+
val conf = sys.env.get("SPARK_AUDIT_MASTER") match {
28+
case Some(master) => new SparkConf().setAppName("Simple GraphX App").setMaster(master)
29+
case None => new SparkConf().setAppName("Simple Graphx App")
30+
}
31+
val sc = new SparkContext(conf)
32+
SparkContext.jarOfClass(this.getClass).foreach(sc.addJar)
33+
2834
val users: RDD[(VertexId, (String, String))] =
2935
sc.parallelize(Array((3L, ("rxin", "student")), (7L, ("jgonzal", "postdoc")),
3036
(5L, ("franklin", "prof")), (2L, ("istoica", "prof")),

dev/audit-release/sbt_app_streaming/src/main/scala/StreamingApp.scala

Lines changed: 5 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -27,10 +27,12 @@ import org.apache.spark.streaming._
2727
object SparkStreamingExample {
2828

2929
def main(args: Array[String]) {
30-
val conf = new SparkConf(true)
31-
.setMaster("local[2]")
32-
.setAppName("Streaming test")
30+
val conf = sys.env.get("SPARK_AUDIT_MASTER") match {
31+
case Some(master) => new SparkConf().setAppName("Simple Streaming App").setMaster(master)
32+
case None => new SparkConf().setAppName("Simple Streaming App")
33+
}
3334
val ssc = new StreamingContext(conf, Seconds(1))
35+
SparkContext.jarOfClass(this.getClass).foreach(ssc.sparkContext.addJar)
3436
val seen = ListBuffer[RDD[Int]]()
3537

3638
val rdd1 = ssc.sparkContext.makeRDD(1 to 100, 10)

0 commit comments

Comments
 (0)