Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -238,7 +238,7 @@ package object config {
"each event log file to the configured size.")
.version("3.0.0")
.booleanConf
.createWithDefault(false)
.createWithDefault(true)

private[spark] val EVENT_LOG_ROLLING_MAX_FILE_SIZE =
ConfigBuilder("spark.eventLog.rolling.maxFileSize")
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -731,6 +731,7 @@ class SparkSubmitSuite
"--conf", "spark.master.rest.enabled=false",
"--conf", "spark.executorEnv.HADOOP_CREDSTORE_PASSWORD=secret_password",
"--conf", "spark.eventLog.enabled=true",
"--conf", "spark.eventLog.rolling.enabled=false",
"--conf", "spark.eventLog.testing=true",
"--conf", s"spark.eventLog.dir=${testDirPath.toUri.toString}",
"--conf", "spark.hadoop.fs.defaultFS=unsupported://example.com",
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -66,7 +66,7 @@ abstract class EventLogFileWritersSuite extends SparkFunSuite with LocalSparkCon
conf.set(EVENT_LOG_DIR, testDir.toString)

// default config
buildWriterAndVerify(conf, classOf[SingleEventLogFileWriter])
buildWriterAndVerify(conf, classOf[RollingEventLogFilesWriter])

conf.set(EVENT_LOG_ENABLE_ROLLING, true)
Copy link
Member

@viirya viirya Nov 2, 2023

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Is it redundant then?

Suggested change
conf.set(EVENT_LOG_ENABLE_ROLLING, true)

Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Or we want to:

Suggested change
conf.set(EVENT_LOG_ENABLE_ROLLING, true)
conf.set(EVENT_LOG_ENABLE_ROLLING, false)
buildWriterAndVerify(conf, classOf[SingleEventLogFileWriter])

buildWriterAndVerify(conf, classOf[RollingEventLogFilesWriter])
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Suggested change
buildWriterAndVerify(conf, classOf[RollingEventLogFilesWriter])

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -38,6 +38,7 @@ object EventLogTestHelper {
def getLoggingConf(logDir: Path, compressionCodec: Option[String] = None): SparkConf = {
val conf = new SparkConf
conf.set(EVENT_LOG_ENABLED, true)
conf.set(EVENT_LOG_ENABLE_ROLLING, false)
Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This is consistent with the existing function description.

* Get a SparkConf with event logging enabled. It doesn't enable rolling event logs, so caller

conf.set(EVENT_LOG_BLOCK_UPDATES, true)
conf.set(EVENT_LOG_TESTING, true)
conf.set(EVENT_LOG_DIR, logDir.toString)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -33,7 +33,7 @@ import org.apache.spark.deploy.SparkHadoopUtil
import org.apache.spark.deploy.history.{EventLogFileReader, SingleEventLogFileWriter}
import org.apache.spark.deploy.history.EventLogTestHelper._
import org.apache.spark.executor.{ExecutorMetrics, TaskMetrics}
import org.apache.spark.internal.config.{EVENT_LOG_COMPRESS, EVENT_LOG_DIR, EVENT_LOG_ENABLED}
import org.apache.spark.internal.config.{EVENT_LOG_COMPRESS, EVENT_LOG_DIR, EVENT_LOG_ENABLE_ROLLING, EVENT_LOG_ENABLED}
import org.apache.spark.io._
import org.apache.spark.metrics.{ExecutorMetricType, MetricsSystem}
import org.apache.spark.resource.ResourceProfile
Expand Down Expand Up @@ -163,6 +163,7 @@ class EventLoggingListenerSuite extends SparkFunSuite with LocalSparkContext wit
test("SPARK-31764: isBarrier should be logged in event log") {
val conf = new SparkConf()
conf.set(EVENT_LOG_ENABLED, true)
conf.set(EVENT_LOG_ENABLE_ROLLING, false)
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Is it failed without setting to false?

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yes, this test case tries to read the event log file.

conf.set(EVENT_LOG_COMPRESS, false)
conf.set(EVENT_LOG_DIR, testDirPath.toString)
val sc = new SparkContext("local", "test-SPARK-31764", conf)
Expand Down
2 changes: 2 additions & 0 deletions docs/core-migration-guide.md
Original file line number Diff line number Diff line change
Expand Up @@ -24,6 +24,8 @@ license: |

## Upgrading from Core 3.5 to 4.0

- Since Spark 4.0, Spark will roll event logs to archive them incrementally. To restore the behavior before Spark 4.0, you can set `spark.eventLog.rolling.enabled` to `false`.

- Since Spark 4.0, Spark will compress event logs. To restore the behavior before Spark 4.0, you can set `spark.eventLog.compress` to `false`.

- Since Spark 4.0, `spark.shuffle.service.db.backend` is set to `ROCKSDB` by default which means Spark will use RocksDB store for shuffle service. To restore the behavior before Spark 4.0, you can set `spark.shuffle.service.db.backend` to `LEVELDB`.
Expand Down