mardi 27 mars 2018

Lessons learned building a big data solution oriented towards security logs AKA a "security data lake"

My company decided to give a go at the big data security log search, most information out there talks about implementing machine learning and other features using (cool) Jupyter notebooks and proof of concept sort of setup to accomplish that. However, trying to implement an infrastructure to accomplish that at large scale, reliably turned out to be an epic story.

The high level requirements goes like this;

  • We want to be able to search events in near real time and perform full text search on events and create dashboards for specific use cases depending on the investigation.
  • We want to store both the data in pristine and parsed format for at least a year for audit and investigation purposes.
  • We want to be able to replay older logs through the pipeline to fix/update the parser-augmenter pipeline.
At this stage some of you might say that Apache Metron or Apache Spot and you would be right to do so, the problem we found was the list of supported log parser for metron isn't that long and are linked to specific version. This is not abnormal but in our large and mixed environment (the telecom world has a lot of strange devices) the benefit were not so clear, on top of that using 2 tools means 2 different data models, interfaces and so on to work with. The full story is a lot longer and in retrospect some point were overlooked and some difficulties were over evaluated, the importance of having a tool for the analyst to manipulate the data and query them extensively was overlooked, it's called Stellar on the Metron side.

Laying out the tracks

The bird's eye overview is around 3 steps, ingestion, parsing and augmentation, storage

Ingestion is typically done through a syslog server in a Linux environment and using a Windows event collector in a Microsoft shop, this is already a problem; none of these provide high availability and scaling can only be done via a divide an conquer approach (host A&B go on that server, B&C on that other server and so on). One approach to address that issue would be to use minifi to read the log files from the syslog servers (active-active) and ship them to the nifi cluster, there we use a detect duplicate processor to ship only one copy of the log to the next layer.  For windows, you might want to use NXLog to ship your windows logs either to files and then use minifi or emit them directly towards Kafka. Once the logs have reached nifi, the 1st thing to do is to wrap them in a json envelope to add the meta data (Time of ingestion, server from which it's coming from, the type of logs ...) and then ship them to Kafka. This will allow you to be handle to back pressure nicely and do point in time recovery if the subsequent part of the pipeline ever have problems or slow down.
The above architecture result in something like this:



Parsing and augmentation is where the heavy duty processing is done.
To parse an event is to extract business data out it like IP's, username and URL from it, you can do it with various tools depending on the skills and constraint of your shop, our proof of concept used logstash to do that task, the pros are it's easy to implement and very near real time oriented (low latency) the con's are that it doesn't handle back pressure very well (one slow parser will slow down all log sources) and it will generally require fire power than others. Nifi on the other hand is micro batches oriented which means more delay, if you have a hard requirement on low latency (less than 60 seconds) this is most likely a no go, on top of that any non trivial parsing on logs that aren't record oriented (Linux auth logs or windows logs) are a massive pain IMHO, there's some work being done on that with record orient processor that can use grok expression to do the parsing but it's still sub optimal. It's a tough problem, Metron uses apache storm to do their parsing to have enough fire power and flexibility to implement complex logic but storm is primarily Java and since our team doesn't have enough Java dev's we settle for Spark since it has a Python API pySpark.
To augment an event is to add more information into it, simple example is to add the geoip information for an IP address, a more complex example would be to add the username associated with an internal IP address or a internal user and IP reputation system (there's a lot to be done here so I'll probably expand on the subject in another post) ...
Again we'll use Spark for the job, there a fair amount of boiler plate code to help you start properly and focus on the business logic rather than fight the tools to get them to work.

Storage is where we store the event for later display, reporting and again we face an amusing issue in that the tools for long term archival and near real time searches are very different, for the real time search we settle on Solr and ElasticSearch, and at the time we settled for Solr since it was included in the stack we use (Ambari) for the long term retention (HDFS) , while Solr is not a bad product, feature wise it's lagging behind, on the UI front, the only generic interface that we found is "banana" which is a fork of Kibana 3 which is old to say the least...On top of all that Elastic has been moving extremely fast with either the machine learning component in the x-pack and the vega scriptable UI, on top of that the core business of Solr seems to be the content management search, which means that the typical workload is read heavy and few writes, exactly the opposite of a log search tool where there will be massive and continuous write and much fewer reads which means that a lot of default value are not suitable for near real time search, see the link for the Solr wiki on the topic.

No matter which one you choose, keep in mind to spread your data over a large number of disks to max the number IOPS the cluster can perform and maximize the effect of the buffer cache, increase the number of shards to match the number of physical disks to enhance write & read performances, keep the replica down to a manageable size.
On the long term storage front, there's is also a couple of things to consider like the file format, there's a lot of discussions about which file format is best but basically you want one of the packed/binary file format like avro, ORC or parquet. Preferably one of the column oriented ones which allow you to change the schema (add or remove fields) over time without it being a breaking change (leaving out avro), knowing all that, we settle for parquet because it has a good support in Spark and it's compression ratio are pretty good which allow us to archive a bit more before using up all the disk space. You will also want to partition the data, that means essentially to store the files in different directories based on the time, something like /hadoop/proxy/year=2018/month=02/day=01/file.parquet this makes it easy to write a scrubber job to delete old data per type depending on the retention period of your logs, pretty much the same way we used to do it on a syslog server.

Now that we layed out the tracks, metrics, out of memory, kerberos, config management, writing a spark job for the parsing, seting up hive table as external because you want to keep the data, ...

Spark streaming with Kerberos enabled on Ambari

In the scope of a research project, I'm parsing syslog logs to stuff them into HDFS (another post on that will follow) and one of the requirement is to have Kerberos enabled on the Kafa brokers, well it turns out it's a bit more trouble than what I asked for and not so well documented.
So here's a brain dump in case you find yourself in the same situation;

My pySpark(2) job flow is as follow, it uses a kafka stream to read from one topic, parse the log line (raw field), add some fields and then emit the resulting json message to another topic, I kept the thing close to what I'm doing so that it's a somewhat relevant example ad not just a simple word count that doesn't illustrate much in my mind.

setup the jaas file

To tell the jvm we're using Kerberos and what credentials to use and all that, once again all the example I've seen use password based auth which is irrelevant in any production context (again ymmv), so here's the jaas file:
Client {
        com.sun.security.auth.module.Krb5LoginModule required
        useKeyTab=true
        keyTab="/home/user/user.keytab"
        useTicketCache=true
        serviceName="zookeeper"
        debug=true
        principal="user@DOMAIN.NET";
};
KafkaClient {
        com.sun.security.auth.module.Krb5LoginModule required
        useKeyTab=true
        keyTab="/home/user/user.keytab"
        useTicketCache=true
        serviceName="kafka"
        debug=true
        principal="user@DOMAIN.NET";
};

Create a keytab

We need to create a keytab file, as referenced by the jass file,  you can create it with those instructions:
$ktutil
ktutil: addent  -password -p your_id97@BGC.NET -k 1 -e aes256-cts-hmac-sha1-96
password for user@DOMAIN.NET: (ktutil is prompting for the password)
ktutil:  wkt your_id.keytab
ktutil:  q
$kinit –kt your_id.keytab user@DOMAIN.NET

Now that the jvm kerberos setup is ok and the keytab is initialized we can do the adjustment in the python.

Setup spark streaming


ssc = StreamingContext(sc, 1)

 kafkaStream = KafkaUtils.createStream(ssc, 'zookeeper:2181', consumer_name, {'ingest-topic': 1}, 
  kafkaParams = {'security.protocol': 'PLAINTEXTSASL',
   'sasl.kerberos.service.name': 'kafka',
   'group.id': consumer_name,
   'rebalance.backoff.ms': '5000',
   'zookeeper.session.timeout.ms': '10000' },
  storageLevel = StorageLevel.MEMORY_AND_DISK_SER)

 kafkaStream.foreachRDD(handler)

The important bit for Kerberos is the 'kafkaParams', note that the security.protocol is *really* 'PLAINTEXTSASL', some other part of the eco system uses SASL_PLAINTEXT but this one uses PLAINTEXTSASL.
The sasl service name is linked to the content of the jaas file, so make sure they match as it's the way the jvm finds the other options required.

Setup kafka producer

Now we want to emit to Kafka, using KafkaProducer:
producer = KafkaProducer(security_protocol = "SASL_PLAINTEXT",
  sasl_mechanism="GSSAPI",
  bootstrap_servers="kafkabroker:6667",
  client_id=consumer_name)
Note that this time the security protocol is "SASL_PLAINTEXT" and no longer PLAINTEXTSASL, even though it's 2 different libraries, a bit of consistency would be nice, note that you'll need GSSAPI libraries (dev) and python gssapi modules for it to support Kerberos.

Fire it up

Now we need to start our spark job with the following invocation:
Launch script:
#!/bin/bash -xp
export SPARK_MAJOR_VERSION=2
/bin/spark-submit --master=yarn --deploy-mode=cluster \
--files kafka_jaas.conf,myuser.keytab \
--repositories http://repo.hortonworks.com/content/groups/public \
--packages  org.apache.spark:spark-streaming-kafka-0-8_2.11:2.1.1.2.6.1.0-129 \
--properties-file spark-streaming.conf \
--driver-memory 5g \
--conf "spark.driver.extraJavaOptions=-Djava.security.auth.login.config=kafka_jaas.conf" \
--conf "spark.executor.extraJavaOptions=-Djava.security.auth.login.config=kafka_jaas.conf -Djavax.net.ssl.trustStore=ca_certs.jks" \
--py-files main.py

The important bits here are the extraJavaOptions to point to the jass file we want to use and the packages option that tells spark to load extra jars to allow Kafka streaming to work.
Note that running a pyspark job like this isn't really practical, I recommend using something like https://github.com/ekampf/PySpark-Boilerplate that setups a zip file with all your data file and deps, remember since the spark job will be distributed all over your workers, they all need the same exact copy of the script and dependencies.
You'll also need to bootstrap the keytab somewhere in your job, simply using
print '[+] Setting up Kerberos: \n%s\n' % (subprocess.check_output(['kinit', '-kt', 'user.keytab', 'user@DOMAIN.NET']))
did the trick for me.

One additional trick if it doesn't work is to add the following line before doing anything in your script:

Debugging

# to debug kerberos problems
import logging
logging.basicConfig(filename='/tmp/example.log',level=logging.DEBUG)
It will generate a massive amount of logs but at least it gives you some data to help you debug the thing.

Putting everything together

# Import dependencies
from __future__ import print_function
from pyspark import SparkContext, StorageLevel
from pyspark.streaming import StreamingContext
from pyspark.streaming.kafka import KafkaUtils
from kafka import KafkaProducer
import json
from pygrok import Grok
from datetime import *
import pytz
from dateutil.parser import *

consumer_name = "SyslogImporter"
syslog_pattern = "(<%{NUMBER:SyslogPriority}>)?%{TIMESTAMP_ISO8601:Timestamp}\s%{HOSTNAME:SourceHostname}\s({SYSLOGPROG:SyslogProgram}\[%{NUMBER:SyslogProgramId}\])?"

def grok_syslog(event):
    grok = Grok(syslog_pattern)
    raw = json.loads(event)
    parsed_syslog = grok.match(raw['Raw'])
    if parsed_syslog:
     parsed_syslog["tags"] = ["syslog"]
     parsed_syslog["parsers"] = [consumer_name]
    else:
     parsed_syslog = json.loads(event)
     parsed_syslog["Timestamp"] = parsed_syslog["IngestionTimestamp"]
     parsed_syslog["tags"] = ["not_syslog"]
     parsed_syslog["parsers"] = [consumer_name]    
    new_event = {key: value for (key, value) in (raw.items() + parsed_syslog.items())}
    return new_event

def parse_syslog(event):
    utc_tz = pytz.timezone('UTC')
    log = grok_syslog(event)
    ts = log['Timestamp']
    ts_dt = parse(ts)
    log['Timestamp'] = ts_dt.astimezone(utc_tz).isoformat()
    return log

def handler(message):
 records = message.collect()
 for raw_record in records:
  #saving the original to send it as such in case of exception
  syslog_record_string = raw_record[1]
  event_id = "0"
  try:
   syslog_record = parse_syslog(raw_record[1])
   event_id = syslog_record["Id"]
   syslog_record_string = json.dumps(syslog_record, ensure_ascii=False).encode('utf8')
  except Exception as e:
   print("[-] Something barfed" + str(e))
   print(raw_record)
   pass
  producer.send(b'work-topic', bytes(syslog_record_string))
  producer.flush()

def main():
 sc = SparkContext(appName=consumer_name) 
 ssc = StreamingContext(sc, 1)

 kafkaStream = KafkaUtils.createStream(ssc, 'zookeeper:2181', consumer_name, {'ingest-topic': 1}, 
  kafkaParams = {'security.protocol': 'PLAINTEXTSASL',
   'sasl.kerberos.service.name': 'kafka',
   'group.id': consumer_name,
   'rebalance.backoff.ms': '5000',
   'zookeeper.session.timeout.ms': '10000' },
  storageLevel = StorageLevel.MEMORY_AND_DISK_SER)

 kafkaStream.foreachRDD(handler)

 # Start the streaming context
 ssc.start()
 ssc.awaitTermination()

if __name__ == "__main__":
print '[+] Setting up Kerberos: \n%s\n' % (subprocess.check_output(['kinit', '-kt', 'user.keytab', 'user@DOMAIN.NET']))
 producer = KafkaProducer(security_protocol = "SASL_PLAINTEXT",
  sasl_mechanism="GSSAPI",
  bootstrap_servers="kafkabroker:6667",
  client_id=consumer_name)
 main()

Hadoop / Spark2 snippet that took way too long to figure out

This is a collection of links and snippet that took me way too long to figure out; I've copied them here with a bit of documentation in...