Skip to content

Commit

Permalink
Switch JDBC driver to log4j 2.17.1 from log4j 1.x
Browse files Browse the repository at this point in the history
- removed hard-coded compile time dependency on log4j 1.x from entire SnappyData code-base
  (the underlying Spark still depends on log4j 1.x)
- removed code dependencies on log4j and instead use slf4j
- update store link that removes log4j 1.x dependency rather supports both 2.x and 1.x
- JDBC driver packaging now includes log4j 2.x using a special "shadowInclude" configuration
  that includes the dependencies only in the shadow jar
- also explicitly exlude log4j 1.x resources and Log4j1Configurator implementation from the
  JDBC driver's shadow jar to completely eliminate all traces for log4j 1.x
- code that required explicit log4j references (e.g. to change log-level temporarily) now
  uses the methods from ClientSharedUtils that has adapters for both log4j 2.x and 1.x
- ensure log4j init from store layer rather than default spark route
- updated version to 1.3.0.1 from 1.3.0
- core/cluster and other components still continue to use log4j 1.x due to underlying Spark
- increased gradle max-memory to 3G from 2G
- updated license header templates to year 2022 from 2021

Fixes for some test failures
- allow JAVA_HOME to be inherited across VMs including the independent Spark/SnappyData
  clusters launched by the tests by writing it to spark-env.sh; this allows one to run the
  SnappyData test suites by setting JAVA_HOME to Java8 while the system JDK can be different
- added explicit "bind-address=localhost" in many tests that were failing due to
  automatically determined bind address being unusable (e.g. the one used by docker/VM/VPN)
- fixed URL for cassandra download in CassandraSnappyDUnitTest
  • Loading branch information
sumwale committed Feb 4, 2022
1 parent 05ebab4 commit 85b9010
Show file tree
Hide file tree
Showing 41 changed files with 204 additions and 192 deletions.
2 changes: 1 addition & 1 deletion aqp
Submodule aqp updated 1 files
+2 −2 README.md
11 changes: 7 additions & 4 deletions build.gradle
Original file line number Diff line number Diff line change
Expand Up @@ -70,7 +70,7 @@ allprojects {
apply plugin: "build-time-tracker"

group = 'io.snappydata'
version = '1.3.0'
version = '1.3.0.1'

// apply compiler options
tasks.withType(JavaCompile) {
Expand Down Expand Up @@ -109,6 +109,7 @@ allprojects {
sparkJobServerVersion = '0.6.2.12'
snappySparkMetricsLibVersion = '2.0.0.1'
log4jVersion = '1.2.17'
log4j2Version = '2.17.1'
slf4jVersion = '1.7.30'
junitVersion = '4.12'
mockitoVersion = '1.10.19'
Expand Down Expand Up @@ -732,6 +733,11 @@ subprojects {
extendsFrom testCompile
description 'a dependency that exposes test artifacts'
}
shadowInclude {
description = 'a dependency that is included only in the shadow jar'
canBeResolved = true
canBeConsumed = false
}
/*
all {
resolutionStrategy {
Expand Down Expand Up @@ -782,10 +788,7 @@ subprojects {
}

dependencies {
compile 'log4j:log4j:' + log4jVersion
compile 'org.slf4j:slf4j-api:' + slf4jVersion
compile 'org.slf4j:slf4j-log4j12:' + slf4jVersion

testCompile "junit:junit:${junitVersion}"
}
}
Expand Down
2 changes: 1 addition & 1 deletion cluster/bin/snappy
Original file line number Diff line number Diff line change
Expand Up @@ -121,7 +121,7 @@ elif [ -z "$SNAPPY_NO_QUICK_LAUNCH" -a $# -ge 2 \
fi
fi

JARS="`echo "${SPARK_HOME}"/jars/snappydata-launcher* "${SPARK_HOME}"/jars/gemfire-shared* "${SPARK_HOME}"/jars/jna-4.* | sed 's/ /:/g'`"
JARS="`echo "${SPARK_HOME}"/jars/snappydata-launcher* "${SPARK_HOME}"/jars/gemfire-shared* "${SPARK_HOME}"/jars/log4j-* "${SPARK_HOME}"/jars/jna-4.* | sed 's/ /:/g'`"
exec $RUNNER $JAVA_ARGS -Xverify:none -cp "$JARS" io.snappydata.tools.QuickLauncher "$@" $HOSTNAME_FOR_CLIENTS $IMPLICIT_AWS_CLIENT_BIND_ADDRESS
IMPLICIT_CLIENT_BIND_ADDRESS=
EXPLICIT_CLIENT_BIND_ADDRESS=
Expand Down
2 changes: 1 addition & 1 deletion cluster/conf/log4j.properties.template
Original file line number Diff line number Diff line change
@@ -1,5 +1,5 @@
#
# Copyright (c) 2017-2019 TIBCO Software Inc. All rights reserved.
# Copyright (c) 2017-2022 TIBCO Software Inc. All rights reserved.
#
# Licensed under the Apache License, Version 2.0 (the "License"); you
# may not use this file except in compliance with the License. You
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -133,7 +133,7 @@ abstract class ClusterManagerTestBase(s: String)
val sysProps = this.sysProps
DistributedTestBase.invokeInLocator(new SerializableRunnable() {
override def run(): Unit = {
// force initialize test-log4j.properties to enable override by SnappyData
// force initialize console-log4j(2).properties to enable override by SnappyData
getStaticLogWriter.info("[SnappyData] starting locator...")
ClusterManagerTestBase.setSystemProperties(sysProps)
val loc: Locator = ServiceManager.getLocatorInstance
Expand All @@ -153,7 +153,7 @@ abstract class ClusterManagerTestBase(s: String)
val nodeProps = bootProps
val startNode = new SerializableRunnable() {
override def run(): Unit = {
// force initialize test-log4j.properties to enable override by SnappyData
// force initialize console-log4j(2).properties to enable override by SnappyData
getStaticLogWriter.info("[SnappyData] starting server...")
ClusterManagerTestBase.setSystemProperties(sysProps)
val node = ServiceManager.currentFabricServiceInstance
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -32,7 +32,7 @@ import com.pivotal.gemfirexd.internal.snappy.InterpreterExecute
import com.pivotal.gemfirexd.{Attribute, Constants}
import io.snappydata.Constant
import io.snappydata.gemxd.SnappySessionPerConnection
import org.apache.log4j.Logger
import org.slf4j.LoggerFactory

import org.apache.spark.Logging
import org.apache.spark.sql.execution.columnar.ExternalStoreUtils
Expand Down Expand Up @@ -279,7 +279,7 @@ object SnappyInterpreterExecute {

object PermissionChecker {

private val logger = Logger.getLogger(
private[this] val logger = LoggerFactory.getLogger(
"io.snappydata.remote.interpreter.SnappyInterpreterExecute")

def isAllowed(key: String, currentUser: String, tableSchema: String): Boolean = {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,9 +19,9 @@ package org.apache.spark.sql.hive.thriftserver

import java.net.InetAddress

import com.gemstone.gemfire.internal.shared.ClientSharedUtils
import org.apache.hadoop.hive.ql.metadata.Hive
import org.apache.hive.service.cli.thrift.ThriftCLIService
import org.apache.log4j.{Level, LogManager}

import org.apache.spark.Logging
import org.apache.spark.sql.hive.client.HiveClientImpl
Expand Down Expand Up @@ -55,12 +55,11 @@ object SnappyHiveThriftServer2 extends Logging {
// full conf used is from the internal hive client from SnappySharedState.

// avoid meta-store init warnings
val rootLogger = LogManager.getRootLogger
val metaLogger = LogManager.getLogger("org.apache.hadoop.hive.metastore.MetaStoreDirectSql")
val currentRootLevel = rootLogger.getLevel
val currentMetaLevel = metaLogger.getLevel
rootLogger.setLevel(Level.ERROR)
metaLogger.setLevel(Level.ERROR)
val metaLogger = "org.apache.hadoop.hive.metastore.MetaStoreDirectSql"
val currentRootLevel = ClientSharedUtils.getLog4jLevel(null)
val currentMetaLevel = ClientSharedUtils.getLog4jLevel(metaLogger)
ClientSharedUtils.setLog4jLevel(null, "ERROR")
ClientSharedUtils.setLog4jLevel(metaLogger, "ERROR")
val externalCatalog = SnappyHiveExternalCatalog.getExistingInstance
val hiveConf = try {
val executionHive = HiveUtils.newClientForExecution(conf,
Expand All @@ -87,8 +86,8 @@ object SnappyHiveThriftServer2 extends Logging {
conf
}
} finally {
rootLogger.setLevel(currentRootLevel)
metaLogger.setLevel(currentMetaLevel)
ClientSharedUtils.setLog4jLevel(null, currentRootLevel)
ClientSharedUtils.setLog4jLevel(metaLogger, currentMetaLevel)
}

val server = new HiveThriftServer2(SparkSQLEnv.sqlContext)
Expand Down
2 changes: 2 additions & 0 deletions core/build.gradle
Original file line number Diff line number Diff line change
Expand Up @@ -30,10 +30,12 @@ dependencies {
compileOnly 'org.scala-lang:scala-library:' + scalaVersion
compileOnly 'org.scala-lang:scala-reflect:' + scalaVersion

compile 'log4j:log4j:' + log4jVersion
compile 'org.slf4j:slf4j-api:' + slf4jVersion
compile 'org.slf4j:slf4j-log4j12:' + slf4jVersion
compile 'org.slf4j:jcl-over-slf4j:' + slf4jVersion
compile 'org.slf4j:jul-to-slf4j:' + slf4jVersion

compile group: 'org.codehaus.janino', name: 'janino', version: janinoVersion
compile("org.apache.thrift:libthrift:${thriftVersion}") {
exclude(group: 'org.slf4j', module: 'slf4j-api')
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -112,7 +112,7 @@ class CassandraSnappyDUnitTest(val s: String)
}
if (downloadFiles) {
val cassandraTarball = s"apache-cassandra-$cassandraVersion-bin.tar.gz"
s"curl -OL http://www-us.apache.org/dist/cassandra/$cassandraVersion/$cassandraTarball".!!
s"curl -OL http://www.apache.org/dist/cassandra/$cassandraVersion/$cassandraTarball".!!
("curl -OL https://repo1.maven.org/maven2/com/datastax/spark/" +
s"spark-cassandra-connector_2.11/$cassandraConnVersion/$cassandraConnectorJar").!!
("tar xf " + cassandraTarball).!!
Expand Down
42 changes: 8 additions & 34 deletions core/src/dunit/scala/io/snappydata/cluster/ClusterUtils.scala
Original file line number Diff line number Diff line change
Expand Up @@ -17,18 +17,15 @@

package io.snappydata.cluster

import java.io.{BufferedWriter, FileWriter, PrintWriter}
import java.nio.file.{Files, Path, Paths, StandardCopyOption}
import java.util.concurrent.atomic.AtomicBoolean

import scala.collection.JavaConverters._
import scala.io.{Codec, Source}
import scala.sys.process._
import scala.util.control.NonFatal

import io.snappydata.SnappyTestRunner
import io.snappydata.test.dunit.VM
import io.snappydata.test.util.TestException
import io.snappydata.util.TestUtils
import org.apache.commons.io.FileUtils

import org.apache.spark.Logging
Expand Down Expand Up @@ -135,7 +132,7 @@ object ClusterUtils extends Serializable with Logging {

def getEnvironmentVariable(name: String): String = {
val value = System.getenv(name)
if (name eq null) {
if (value eq null) {
throw new TestException(s"Environment variable $name is not defined")
}
value
Expand All @@ -144,39 +141,16 @@ object ClusterUtils extends Serializable with Logging {
def getSparkClusterDirectory(productDir: String): String =
getClusterDirectory(Paths.get(productDir), isSnappy = false)

def writeToFile(str: String, filePath: String, append: Boolean): Unit = {
val fileWriter = new FileWriter(filePath, append)
val bufferedWriter = new BufferedWriter(fileWriter)
val pw = new PrintWriter(bufferedWriter)
try {
pw.write(str)
pw.flush()
} finally {
pw.close()
}
// wait until file becomes available (e.g. running on NFS)
var matched = append
while (!matched) {
Thread.sleep(100)
try {
val source = Source.fromFile(filePath)(Codec.UTF8)
val lines = try {
source.mkString
} finally {
source.close()
}
matched = lines == str
} catch {
case NonFatal(_) =>
}
}
}
def writeToFile(str: String, filePath: String, append: Boolean): Unit =
SnappyTestRunner.writeToFile(str, filePath, append)

def startSparkCluster(clusterDir: String): String = {
logInfo(s"Starting spark cluster in $clusterDir/work")
writeToFile(
s"\nSPARK_WORKER_CORES=${TestUtils.defaultCores * 2}",
s"$clusterDir/conf/spark-env.sh", append = true)
s"""
|JAVA_HOME=${SnappyTestRunner.javaHome}
|SPARK_WORKER_CORES=${SnappyTestRunner.defaultCores * 2}
|""".stripMargin, s"$clusterDir/conf/spark-env.sh", append = true)
val output = s"$clusterDir/sbin/start-all.sh".!!
logInfo(output)
output
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -372,8 +372,8 @@ trait SplitClusterDUnitTestObject extends Logging {
* Returns the SnappyContext for external(connector) Spark cluster connected to
* SnappyData cluster
*/
def getSnappyContextForConnector(locatorClientPort: Int, props: Properties = null):
SnappyContext = {
def getSnappyContextForConnector(locatorClientPort: Int,
props: Properties = null): SnappyContext = {
val hostName = InetAddress.getLocalHost.getHostName
// val connectionURL = "jdbc:snappydata://localhost:" + locatorClientPort + "/"
val connectionURL = s"localhost:$locatorClientPort"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -27,7 +27,7 @@ import scala.util.control.NonFatal
import com.gemstone.gemfire.cache.RegionDestroyedException
import com.gemstone.gemfire.internal.LogWriterImpl
import com.gemstone.gemfire.internal.cache.{ExternalTableMetaData, GemfireCacheHelper, LocalRegion, PolicyTableData}
import com.gemstone.gemfire.internal.shared.SystemProperties
import com.gemstone.gemfire.internal.shared.{ClientSharedUtils, SystemProperties}
import com.pivotal.gemfirexd.Attribute.{PASSWORD_ATTR, USERNAME_ATTR}
import com.pivotal.gemfirexd.internal.catalog.ExternalCatalog
import com.pivotal.gemfirexd.internal.engine.Misc
Expand All @@ -40,7 +40,6 @@ import io.snappydata.Constant.{SPARK_STORE_PREFIX, STORE_PROPERTY_PREFIX}
import io.snappydata.sql.catalog.SnappyExternalCatalog.checkSchemaPermission
import io.snappydata.sql.catalog.{CatalogObjectType, ConnectorExternalCatalog, SnappyExternalCatalog}
import io.snappydata.thrift._
import org.apache.log4j.{Level, LogManager}

import org.apache.spark.sql.catalyst.TableIdentifier
import org.apache.spark.sql.catalyst.catalog.CatalogTypes.TablePartitionSpec
Expand Down Expand Up @@ -237,9 +236,8 @@ class StoreHiveCatalog extends ExternalCatalog with Logging {
lockTaken = lockService.readLock(hiveClientObject, lockOwner,
GfxdLockSet.MAX_LOCKWAIT_VAL)
// reduce log4j level to avoid "function exists" warnings
val log4jLogger = LogManager.getRootLogger
if (log4jLogger.getEffectiveLevel == Level.WARN) {
log4jLogger.setLevel(Level.ERROR)
if ("WARN".equalsIgnoreCase(ClientSharedUtils.getLog4jLevel(null))) {
ClientSharedUtils.setLog4jLevel(null, "ERROR")
}
} else {
lockTaken = lockService.writeLock(hiveClientObject, lockOwner,
Expand Down
8 changes: 5 additions & 3 deletions core/src/main/scala/io/snappydata/util/ServiceUtils.scala
Original file line number Diff line number Diff line change
Expand Up @@ -29,6 +29,7 @@ import _root_.com.gemstone.gemfire.internal.shared.ClientSharedUtils
import _root_.com.pivotal.gemfirexd.internal.engine.GfxdConstants
import _root_.com.pivotal.gemfirexd.internal.engine.distributed.utils.GemFireXDUtils
import io.snappydata.{Constant, Property, ServerManager, SnappyTableStatsProviderService}
import org.slf4j.{Logger, LoggerFactory}

import org.apache.spark.memory.MemoryMode
import org.apache.spark.sql.collection.Utils
Expand Down Expand Up @@ -118,10 +119,11 @@ object ServiceUtils {
}
// set the log-level from initialized SparkContext's level if set to higher level than default
if (!properties.containsKey("log-level")) {
val level = org.apache.log4j.Logger.getRootLogger.getLevel
if ((level ne null) && level.isGreaterOrEqual(org.apache.log4j.Level.WARN)) {
// preserve existing sl4j level if it is higher
val rootLogger = LoggerFactory.getLogger(Logger.ROOT_LOGGER_NAME)
if (rootLogger.isWarnEnabled()) {
properties.setProperty("log-level",
ClientSharedUtils.convertToJavaLogLevel(level).getName.toLowerCase)
ClientSharedUtils.getJavaLogLevel(rootLogger).getName.toLowerCase)
}
}
ServerManager.getServerInstance.start(properties)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -106,6 +106,10 @@ object ExternalStoreUtils extends Logging {

registerBuiltinDrivers()

def init(): Unit = {
// empty method just to ease initialization by registerBuiltinDrivers()
}

def registerBuiltinDrivers(): Unit = {
DriverRegistry.register(Constant.JDBC_EMBEDDED_DRIVER)
DriverRegistry.register(Constant.JDBC_CLIENT_DRIVER)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -51,8 +51,6 @@ object HiveClientUtil extends Logging {
ConfVars.HIVEHISTORYFILELOC -> hivePath("query_logs"),
ConfVars.HIVE_SERVER2_LOGGING_OPERATION_LOG_LOCATION -> hivePath("operation_logs"))

ExternalStoreUtils.registerBuiltinDrivers()

private def hivePath(name: String): String =
Paths.get(s"$HIVE_TMPDIR/$name").toAbsolutePath.toString

Expand All @@ -63,6 +61,7 @@ object HiveClientUtil extends Logging {
*/
def getOrCreateExternalCatalog(sparkContext: SparkContext,
conf: SparkConf): SnappyHiveExternalCatalog = synchronized {
ExternalStoreUtils.init()
val (dbURL, dbDriver) = resolveMetaStoreDBProps(SnappyContext.getClusterMode(sparkContext))
val metadataConf = new SnappyHiveConf
// make a copy of SparkConf since it is to be updated later
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -28,6 +28,7 @@ import scala.concurrent.ExecutionException
import com.gemstone.gemfire.cache.CacheClosedException
import com.gemstone.gemfire.internal.LogWriterImpl
import com.gemstone.gemfire.internal.cache.{LocalRegion, PartitionedRegion}
import com.gemstone.gemfire.internal.shared.ClientSharedUtils
import com.google.common.cache.{Cache, CacheBuilder, CacheLoader, LoadingCache}
import com.pivotal.gemfirexd.Constants
import com.pivotal.gemfirexd.internal.engine.Misc
Expand All @@ -41,7 +42,6 @@ import io.snappydata.sql.catalog.{CatalogObjectType, ConnectorExternalCatalog, R
import org.apache.hadoop.conf.Configuration
import org.apache.hadoop.hive.metastore.api.NoSuchObjectException
import org.apache.hadoop.hive.ql.metadata.Hive
import org.apache.log4j.{Level, LogManager}

import org.apache.spark.SparkConf
import org.apache.spark.jdbc.{ConnectionConf, ConnectionUtil}
Expand All @@ -62,7 +62,7 @@ import org.apache.spark.sql.sources.JdbcExtendedUtils
import org.apache.spark.sql.sources.JdbcExtendedUtils.normalizeSchema
import org.apache.spark.sql.store.CodeGeneration
import org.apache.spark.sql.types.LongType
import org.apache.spark.sql.{AnalysisException, _}
import org.apache.spark.sql._

class SnappyHiveExternalCatalog private[hive](val conf: SparkConf,
val hadoopConf: Configuration, val createTime: Long)
Expand Down Expand Up @@ -872,20 +872,19 @@ object SnappyHiveExternalCatalog {
// Once the initialization is done, restore the logging level.
val logger = Misc.getI18NLogWriter.asInstanceOf[LogWriterImpl]
val previousLevel = logger.getLevel
val log4jLogger = LogManager.getRootLogger
val log4jLevel = log4jLogger.getEffectiveLevel
val log4jLevel = ClientSharedUtils.getLog4jLevel(null)
logger.info("Starting hive meta-store initialization")
val reduceLog = previousLevel == LogWriterImpl.CONFIG_LEVEL ||
previousLevel == LogWriterImpl.INFO_LEVEL
if (reduceLog) {
logger.setLevel(LogWriterImpl.ERROR_LEVEL)
log4jLogger.setLevel(Level.ERROR)
ClientSharedUtils.setLog4jLevel(null, "ERROR")
}
try {
instance = new SnappyHiveExternalCatalog(sparkConf, hadoopConf, createTime)
} finally {
logger.setLevel(previousLevel)
log4jLogger.setLevel(log4jLevel)
ClientSharedUtils.setLog4jLevel(null, log4jLevel)
logger.info("Done hive meta-store initialization")
}
instance
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -397,8 +397,7 @@ trait NativeTableRowLevelSecurityRelation
def isRowLevelSecurityEnabled: Boolean = {
val conn = connFactory()
try {
JdbcExtendedUtils.isRowLevelSecurityEnabled(resolvedName,
conn, dialect, sqlContext)
JdbcExtendedUtils.isRowLevelSecurityEnabled(resolvedName, conn)
} catch {
case se: java.sql.SQLException =>
if (se.getMessage.contains("No suitable driver found")) {
Expand Down
Loading

0 comments on commit 85b9010

Please sign in to comment.