Skip to content

Commit e246b2f

Browse files
jiaoqingboyaooqinn
authored andcommitted
[KYUUBI #2012] rename EventLoggingService and creates an correct directory
### _Why are the changes needed?_ Fix #2012 #1180 In `EventLoggingService`, when `EventLoggerType` is set to `JSON` Kyuubi creates an error directory, it should not be the `ENGINE_EVENT_JSON_LOG_PATH` but the `SERVER_EVENT_JSON_LOG_PATH`. <img width="557" alt="微信图片_20220304165614" src="https://user-images.githubusercontent.com/14961757/156771039-e18fcdbe-8b99-4190-9b14-6953daa55e5f.png"> <img width="605" alt="微信图片_20220304165633" src="https://user-images.githubusercontent.com/14961757/156771053-defdee62-67f9-4f16-9d4a-8ef170304221.png"> rename `EventLoggingService` in `kyuubi-server` to `KyuubiEventLoggingService` rename `EventLoggingService` in `kyuubi-spark-sql-engine` to `SparkEventLoggingService` ### _How was this patch tested?_ - [ ] Add some test cases that check the changes thoroughly including negative and positive cases if possible - [ ] Add screenshots for manual tests if appropriate - [ ] [Run test](https://kyuubi.apache.org/docs/latest/develop_tools/testing.html#running-tests) locally before make a pull request Closes #2037 from jiaoqingbo/kyuubi2012. Closes #2012 714e5ea [jiaoqingbo] fix ut failed 10d1a15 [jiaoqingbo] execute dev/format cb99a45 [jiaoqingbo] [KYUUBI #2012] rename EventLoggingService and creates an correct directory Authored-by: jiaoqingbo <1178404354@qq.com> Signed-off-by: Kent Yao <yao@apache.org>
1 parent 7bf2460 commit e246b2f

File tree

7 files changed

+10
-11
lines changed

7 files changed

+10
-11
lines changed

externals/kyuubi-spark-sql-engine/src/main/scala/org/apache/kyuubi/engine/spark/SparkSQLEngine.scala

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -33,7 +33,7 @@ import org.apache.kyuubi.config.KyuubiConf
3333
import org.apache.kyuubi.config.KyuubiConf._
3434
import org.apache.kyuubi.config.KyuubiReservedKeys.KYUUBI_ENGINE_SUBMIT_TIME_KEY
3535
import org.apache.kyuubi.engine.spark.SparkSQLEngine.{countDownLatch, currentEngine}
36-
import org.apache.kyuubi.engine.spark.events.{EngineEvent, EngineEventsStore, EventLoggingService}
36+
import org.apache.kyuubi.engine.spark.events.{EngineEvent, EngineEventsStore, SparkEventLoggingService}
3737
import org.apache.kyuubi.events.EventLogging
3838
import org.apache.kyuubi.ha.HighAvailabilityConf._
3939
import org.apache.kyuubi.ha.client.RetryPolicies
@@ -141,7 +141,7 @@ object SparkSQLEngine extends Logging {
141141
currentEngine = Some(new SparkSQLEngine(spark))
142142
currentEngine.foreach { engine =>
143143
// start event logging ahead so that we can capture all statuses
144-
val eventLogging = new EventLoggingService(spark.sparkContext)
144+
val eventLogging = new SparkEventLoggingService(spark.sparkContext)
145145
try {
146146
eventLogging.initialize(kyuubiConf)
147147
eventLogging.start()
Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -24,7 +24,7 @@ import org.apache.kyuubi.config.KyuubiConf
2424
import org.apache.kyuubi.config.KyuubiConf.{ENGINE_EVENT_JSON_LOG_PATH, ENGINE_EVENT_LOGGERS}
2525
import org.apache.kyuubi.events.{AbstractEventLoggingService, EventLoggerType, JsonEventLogger}
2626

27-
class EventLoggingService(spark: SparkContext)
27+
class SparkEventLoggingService(spark: SparkContext)
2828
extends AbstractEventLoggingService {
2929

3030
override def initialize(conf: KyuubiConf): Unit = synchronized {
@@ -38,6 +38,7 @@ class EventLoggingService(spark: SparkContext)
3838
spark.applicationAttemptId.getOrElse(spark.applicationId),
3939
ENGINE_EVENT_JSON_LOG_PATH,
4040
spark.hadoopConfiguration)
41+
jsonEventLogger.createEventLogRootDir(conf, spark.hadoopConfiguration)
4142
addService(jsonEventLogger)
4243
addEventLogger(jsonEventLogger)
4344
case logger =>
Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -32,7 +32,7 @@ import org.apache.kyuubi.events.EventLoggerType._
3232
import org.apache.kyuubi.events.JsonProtocol
3333
import org.apache.kyuubi.operation.{HiveJDBCTestHelper, OperationHandle}
3434

35-
class EventLoggingServiceSuite extends WithSparkSQLEngine with HiveJDBCTestHelper {
35+
class SparkEventLoggingServiceSuite extends WithSparkSQLEngine with HiveJDBCTestHelper {
3636

3737
private val logRoot = "file://" + Utils.createTempDir().toString
3838
private val currentDate = Utils.getDateFromTimestamp(System.currentTimeMillis())

kyuubi-common/src/main/scala/org/apache/kyuubi/events/JsonEventLogger.scala

Lines changed: 1 addition & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -29,7 +29,6 @@ import org.apache.hadoop.fs.permission.FsPermission
2929

3030
import org.apache.kyuubi.Logging
3131
import org.apache.kyuubi.config.{ConfigEntry, KyuubiConf}
32-
import org.apache.kyuubi.config.KyuubiConf.ENGINE_EVENT_JSON_LOG_PATH
3332
import org.apache.kyuubi.events.JsonEventLogger._
3433
import org.apache.kyuubi.service.AbstractService
3534

@@ -114,9 +113,8 @@ class JsonEventLogger(
114113
stream.foreach(_.hflush())
115114
}
116115

117-
// This method is only called by kyuubiServer
118116
def createEventLogRootDir(conf: KyuubiConf, hadoopConf: Configuration): Unit = {
119-
val logRoot: URI = URI.create(conf.get(ENGINE_EVENT_JSON_LOG_PATH))
117+
val logRoot: URI = URI.create(conf.get(logPath))
120118
val fs: FileSystem = FileSystem.get(logRoot, hadoopConf)
121119
val success: Boolean = FileSystem.mkdirs(fs, new Path(logRoot), JSON_LOG_DIR_PERM)
122120
if (!success) {

kyuubi-server/src/main/scala/org/apache/kyuubi/server/EventLoggingService.scala renamed to kyuubi-server/src/main/scala/org/apache/kyuubi/server/KyuubiEventLoggingService.scala

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -24,7 +24,7 @@ import org.apache.kyuubi.config.KyuubiConf.{SERVER_EVENT_JSON_LOG_PATH, SERVER_E
2424
import org.apache.kyuubi.events.{AbstractEventLoggingService, EventLoggerType, JsonEventLogger}
2525
import org.apache.kyuubi.util.KyuubiHadoopUtils
2626

27-
class EventLoggingService extends AbstractEventLoggingService {
27+
class KyuubiEventLoggingService extends AbstractEventLoggingService {
2828

2929
override def initialize(conf: KyuubiConf): Unit = {
3030
val hadoopConf = KyuubiHadoopUtils.newHadoopConf(conf)
@@ -37,7 +37,7 @@ class EventLoggingService extends AbstractEventLoggingService {
3737
s"server-$hostName",
3838
SERVER_EVENT_JSON_LOG_PATH,
3939
hadoopConf)
40-
// TODO: #1180 kyuubiServerEvent need create logRoot automatically
40+
4141
jsonEventLogger.createEventLogRootDir(conf, hadoopConf)
4242
addService(jsonEventLogger)
4343
addEventLogger(jsonEventLogger)

kyuubi-server/src/main/scala/org/apache/kyuubi/server/KyuubiServer.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -154,7 +154,7 @@ class KyuubiServer(name: String) extends Serverable(name) {
154154
throw new UnsupportedOperationException(s"Frontend protocol $other is not supported yet.")
155155
}
156156

157-
private val eventLoggingService = new EventLoggingService()
157+
private val eventLoggingService = new KyuubiEventLoggingService()
158158

159159
override def initialize(conf: KyuubiConf): Unit = synchronized {
160160
val kinit = new KinitAuxiliaryService()

kyuubi-server/src/test/scala/org/apache/kyuubi/events/EventLoggingServiceSuite.scala renamed to kyuubi-server/src/test/scala/org/apache/kyuubi/events/KyuubiEventLoggingServiceSuite.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -34,7 +34,7 @@ import org.apache.kyuubi.operation.OperationState._
3434
import org.apache.kyuubi.server.KyuubiServer
3535
import org.apache.kyuubi.service.ServiceState
3636

37-
class EventLoggingServiceSuite extends WithKyuubiServer with HiveJDBCTestHelper {
37+
class KyuubiEventLoggingServiceSuite extends WithKyuubiServer with HiveJDBCTestHelper {
3838

3939
private val engineLogRoot = "file://" + Utils.createTempDir().toString
4040
private val serverLogRoot = "file://" + Utils.createTempDir().toString

0 commit comments

Comments
 (0)