Skip to content
Snippets Groups Projects
Commit c40f0f21 authored by Matei Zaharia's avatar Matei Zaharia
Browse files

Merge pull request #711 from shivaram/ml-generators

Move ML lib data generator files to util/
parents 413b8417 2c9ea56d
No related branches found
No related tags found
No related merge requests found
......@@ -15,16 +15,43 @@
* limitations under the License.
*/
package spark.mllib.regression
package spark.mllib.util
import scala.util.Random
import org.jblas.DoubleMatrix
import spark.{RDD, SparkContext}
import spark.mllib.util.MLUtils
object LogisticRegressionGenerator {
object LogisticRegressionDataGenerator {
/**
* Generate an RDD containing test data for LogisticRegression. This function chooses
* positive labels with probability `probOne` and scales positive examples by `eps`.
*
* @param sc SparkContext to use for creating the RDD.
* @param nexamples Number of examples that will be contained in the RDD.
* @param nfeatures Number of features to generate for each example.
* @param eps Epsilon factor by which positive examples are scaled.
* @param nparts Number of partitions of the generated RDD. Default value is 2.
* @param probOne Probability that a label is 1 (and not 0). Default value is 0.5.
*/
def generateLogisticRDD(
sc: SparkContext,
nexamples: Int,
nfeatures: Int,
eps: Double,
nparts: Int = 2,
probOne: Double = 0.5): RDD[(Double, Array[Double])] = {
val data = sc.parallelize(0 until nexamples, nparts).map { idx =>
val rnd = new Random(42 + idx)
val y = if (idx % 2 == 0) 0.0 else 1.0
val x = Array.fill[Double](nfeatures) {
rnd.nextGaussian() + (y * eps)
}
(y, x)
}
data
}
def main(args: Array[String]) {
if (args.length != 5) {
......@@ -40,17 +67,8 @@ object LogisticRegressionGenerator {
val parts: Int = if (args.length > 4) args(4).toInt else 2
val eps = 3
val sc = new SparkContext(sparkMaster, "LogisticRegressionGenerator")
val data: RDD[(Double, Array[Double])] = sc.parallelize(0 until nexamples, parts).map { idx =>
val rnd = new Random(42 + idx)
val y = if (idx % 2 == 0) 0 else 1
val x = Array.fill[Double](nfeatures) {
rnd.nextGaussian() + (y * eps)
}
(y, x)
}
val sc = new SparkContext(sparkMaster, "LogisticRegressionDataGenerator")
val data = generateLogisticRDD(sc, nexamples, nfeatures, eps, parts)
MLUtils.saveLabeledData(data, outputPath)
sc.stop()
......
......@@ -15,43 +15,42 @@
* limitations under the License.
*/
package spark.mllib.regression
package spark.mllib.util
import scala.util.Random
import org.jblas.DoubleMatrix
import spark.{RDD, SparkContext}
import spark.mllib.util.MLUtils
object RidgeRegressionGenerator {
def main(args: Array[String]) {
if (args.length != 5) {
println("Usage: RidgeRegressionGenerator " +
"<master> <output_dir> <num_examples> <num_features> <num_partitions>")
System.exit(1)
}
val sparkMaster: String = args(0)
val outputPath: String = args(1)
val nexamples: Int = if (args.length > 2) args(2).toInt else 1000
val nfeatures: Int = if (args.length > 3) args(3).toInt else 100
val parts: Int = if (args.length > 4) args(4).toInt else 2
val eps = 10
object RidgeRegressionDataGenerator {
/**
* Generate an RDD containing test data used for RidgeRegression. This function generates
* uniformly random values for every feature and adds Gaussian noise with mean `eps` to the
* response variable `Y`.
*
* @param sc SparkContext to be used for generating the RDD.
* @param nexamples Number of examples that will be contained in the RDD.
* @param nfeatures Number of features to generate for each example.
* @param eps Epsilon factor by which examples are scaled.
* @param nparts Number of partitions in the RDD. Default value is 2.
*/
def generateRidgeRDD(
sc: SparkContext,
nexamples: Int,
nfeatures: Int,
eps: Double,
nparts: Int = 2) : RDD[(Double, Array[Double])] = {
org.jblas.util.Random.seed(42)
val sc = new SparkContext(sparkMaster, "RidgeRegressionGenerator")
// Random values distributed uniformly in [-0.5, 0.5]
val w = DoubleMatrix.rand(nfeatures, 1).subi(0.5)
w.put(0, 0, 10)
w.put(1, 0, 10)
val data: RDD[(Double, Array[Double])] = sc.parallelize(0 until parts, parts).flatMap { p =>
val data: RDD[(Double, Array[Double])] = sc.parallelize(0 until nparts, nparts).flatMap { p =>
org.jblas.util.Random.seed(42 + p)
val examplesInPartition = nexamples / parts
val examplesInPartition = nexamples / nparts
val X = DoubleMatrix.rand(examplesInPartition, nfeatures)
val y = X.mmul(w)
......@@ -65,6 +64,25 @@ object RidgeRegressionGenerator {
(yObs.get(i, 0), X.getRow(i).toArray)
}
}
data
}
def main(args: Array[String]) {
if (args.length != 5) {
println("Usage: RidgeRegressionGenerator " +
"<master> <output_dir> <num_examples> <num_features> <num_partitions>")
System.exit(1)
}
val sparkMaster: String = args(0)
val outputPath: String = args(1)
val nexamples: Int = if (args.length > 2) args(2).toInt else 1000
val nfeatures: Int = if (args.length > 3) args(3).toInt else 100
val parts: Int = if (args.length > 4) args(4).toInt else 2
val eps = 10
val sc = new SparkContext(sparkMaster, "RidgeRegressionDataGenerator")
val data = generateRidgeRDD(sc, nexamples, nfeatures, eps, parts)
MLUtils.saveLabeledData(data, outputPath)
sc.stop()
......
0% Loading or .
You are about to add 0 people to the discussion. Proceed with caution.
Finish editing this message first!
Please register or to comment