diff --git a/core/src/main/scala/kafka/producer/KafkaLog4jAppender.scala b/core/src/main/scala/kafka/producer/KafkaLog4jAppender.scala index b379439d20996bf5c158a6acbe54b35474cf4cda..66330f1e45b49f9beba7f42dbbbf1bbc9bb7a900 100644 --- a/core/src/main/scala/kafka/producer/KafkaLog4jAppender.scala +++ b/core/src/main/scala/kafka/producer/KafkaLog4jAppender.scala @@ -29,12 +29,16 @@ class KafkaLog4jAppender extends AppenderSkeleton with Logging { var port:Int = 0 var host:String = null var topic:String = null + var producerType:String = "sync" var serializerClass:String = null var zkConnect:String = null var brokerList:String = null private var producer: Producer[String, String] = null + def getProducerType:String = producerType + def setProducerType(producerType: String) { this.producerType = producerType } + def getTopic:String = topic def setTopic(topic: String) { this.topic = topic } @@ -65,6 +69,7 @@ class KafkaLog4jAppender extends AppenderSkeleton with Logging { serializerClass = "kafka.serializer.StringEncoder" LogLog.debug("Using default encoder - kafka.serializer.StringEncoder") } + props.put("producer.type", producerType) props.put("serializer.class", serializerClass) val config : ProducerConfig = new ProducerConfig(props) producer = new Producer[String, String](config)