You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@metron.apache.org by ot...@apache.org on 2017/09/13 02:58:46 UTC

[5/6] metron git commit: METRON-1136 Track Master in Feature Branch (ottobackwards) closes apache/metron#752

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/README.md
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/README.md b/metron-interface/metron-rest/README.md
index e568bce..823b7b6 100644
--- a/metron-interface/metron-rest/README.md
+++ b/metron-interface/metron-rest/README.md
@@ -184,6 +184,7 @@ Request and Response objects are JSON formatted.  The JSON schemas are available
 
 |            |
 | ---------- |
+| [ `POST /api/v1/alert/escalate`](#get-apiv1alertescalate)|
 | [ `POST /api/v1/ext/parsers`](#post-apiv1extparsers)|
 | [ `GET /api/v1/ext/parsers`](#get-apiv1extparsers)|
 | [ `GET /api/v1/ext/parsers/{name}`](#get-apiv1extparsersname)|
@@ -203,10 +204,11 @@ Request and Response objects are JSON formatted.  The JSON schemas are available
 | [ `GET /api/v1/kafka/topic/{name}`](#get-apiv1kafkatopicname)|
 | [ `DELETE /api/v1/kafka/topic/{name}`](#delete-apiv1kafkatopicname)|
 | [ `GET /api/v1/kafka/topic/{name}/sample`](#get-apiv1kafkatopicnamesample)|
-| [ `GET /api/v1/search/search`](#get-apiv1searchsearch)|
+| [ `POST /api/v1/search/search`](#get-apiv1searchsearch)|
+| [ `POST /api/v1/search/group`](#get-apiv1searchgroup)|
 | [ `GET /api/v1/search/findOne`](#get-apiv1searchfindone)|
-| [ `GET /api/v1/search/search`](#get-apiv1searchcolumnmetadata)|
-| [ `GET /api/v1/search/search`](#get-apiv1searchcolumnmetadatacommon)|
+| [ `GET /api/v1/search/column/metadata`](#get-apiv1searchcolumnmetadata)|
+| [ `GET /api/v1/search/column/metadata/common`](#get-apiv1searchcolumnmetadatacommon)|
 | [ `GET /api/v1/sensor/enrichment/config`](#get-apiv1sensorenrichmentconfig)|
 | [ `GET /api/v1/sensor/enrichment/config/list/available/enrichments`](#get-apiv1sensorenrichmentconfiglistavailableenrichments)|
 | [ `GET /api/v1/sensor/enrichment/config/list/available/threat/triage/aggregators`](#get-apiv1sensorenrichmentconfiglistavailablethreattriageaggregators)|
@@ -251,6 +253,13 @@ Request and Response objects are JSON formatted.  The JSON schemas are available
 | [ `PUT /api/v1/update/replace`](#patch-apiv1updatereplace)|
 | [ `GET /api/v1/user`](#get-apiv1user)|
 
+### `POST /api/v1/alert/escalate`
+  * Description: Escalates a list of alerts by producing it to the Kafka escalate topic
+  * Input:
+    * alerts - The alerts to be escalated
+  * Returns:
+    * 200 - Alerts were escalated
+
 ### `POST /api/v1/ext/parsers`
   * Description: Install a Metron Parser Extension into the system
   * Input:
@@ -320,10 +329,13 @@ Request and Response objects are JSON formatted.  The JSON schemas are available
     * 200 - JSON results
 
 ### `POST /api/v1/hdfs`
-  * Description: Writes contents to an HDFS file.  Warning: this will overwrite the contents of a file if it already exists.
+  * Description: Writes contents to an HDFS file.  Warning: this will overwrite the contents of a file if it already exists. Permissions must be set for all three groups if they are to be set. If any are missing, the default permissions will be used, and if any are invalid an exception will be thrown.
   * Input:
     * path - Path to HDFS file
     * contents - File contents
+    * userMode - [optional] symbolic permission string for user portion of the permissions to be set on the file written. For example 'rwx' or read, write, execute. The symbol '-' is used to exclude that permission such as 'rw-' for read, write, no execute
+    * groupMode - [optional] symbolic permission string for group portion of the permissions to be set on the file written. For example 'rwx' or read, write, execute. The symbol '-' is used to exclude that permission such as 'rw-' for read, write, no execute
+    * otherMode - [optional] symbolic permission string for other portion of the permissions to be set on the file written. For example 'rwx' or read, write, execute. The symbol '-' is used to exclude that permission such as 'rw-' for read, write, no execute
   * Returns:
     * 200 - Contents were written
 
@@ -386,6 +398,24 @@ Request and Response objects are JSON formatted.  The JSON schemas are available
     * 200 - Returns sample message
     * 404 - Either Kafka topic is missing or contains no messages
 
+### `POST /api/v1/search/search`
+  * Description: Searches the indexing store
+  * Input:
+      * searchRequest - Search request
+  * Returns:
+    * 200 - Search response
+    
+### `POST /api/v1/search/group`
+  * Description: Searches the indexing store and returns field groups. Groups are hierarchical and nested in the order the fields appear in the 'groups' request parameter. The default sorting within groups is by count descending.  A groupOrder type of count will sort based on then number of documents in a group while a groupType of term will sort by the groupBy term.
+  * Input:
+      * groupRequest - Group request
+        * indices - list of indices to search
+        * query - lucene query
+        * scoreField - field used to compute a total score for each group
+        * groups - List of groups (field name and sort order) 
+  * Returns:
+    * 200 - Group response
+    
 ### `GET /api/v1/search/findOne`
   * Description: Returns latest document for a guid and sensor
   * Input:
@@ -402,13 +432,6 @@ Request and Response objects are JSON formatted.  The JSON schemas are available
   * Returns:
     * 200 - Document representing the output
     * 404 - Document with UUID and sensor type not found
-
-### `GET /api/v1/search/search`
-  * Description: Searches the indexing store
-  * Input:
-      * searchRequest - Search request
-  * Returns:
-    * 200 - Search results
     
 ### `GET /api/v1/search/column/metadata`
   * Description: Get column metadata for each index in the list of indicies

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/config/rest_application.yml
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/config/rest_application.yml b/metron-interface/metron-rest/src/main/config/rest_application.yml
index afd4aeb..33e5ac6 100644
--- a/metron-interface/metron-rest/src/main/config/rest_application.yml
+++ b/metron-interface/metron-rest/src/main/config/rest_application.yml
@@ -30,6 +30,8 @@ kafka:
     url: ${BROKERLIST}
   security:
     protocol: ${KAFKA_SECURITY_PROTOCOL}
+  topics:
+    escalation: ${METRON_ESCALATION_TOPIC}
 
 hdfs:
   metron:

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/MetronRestConstants.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/MetronRestConstants.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/MetronRestConstants.java
index 55aff29..3947e17 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/MetronRestConstants.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/MetronRestConstants.java
@@ -50,6 +50,7 @@ public class MetronRestConstants {
   public static final String CURATOR_MAX_RETRIES = "curator.max.retries";
 
   public static final String KAFKA_BROKER_URL_SPRING_PROPERTY = "kafka.broker.url";
+  public static final String KAFKA_TOPICS_ESCALATION_PROPERTY = "kafka.topics.escalation";
 
   public static final String HDFS_METRON_APPS_ROOT = "hdfs.metron.apps.root";
 
@@ -58,6 +59,7 @@ public class MetronRestConstants {
   public static final String KERBEROS_KEYTAB_SPRING_PROPERTY = "kerberos.keytab";
 
   public static final String SEARCH_MAX_RESULTS = "search.max.results";
+  public static final String SEARCH_MAX_GROUPS = "search.max.groups";
   public static final String INDEX_DAO_IMPL = "index.dao.impl";
   public static final String INDEX_HBASE_TABLE_PROVIDER_IMPL = "index.hbase.provider";
 }

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/config/IndexConfig.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/config/IndexConfig.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/config/IndexConfig.java
index 6385116..b6ac5e7 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/config/IndexConfig.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/config/IndexConfig.java
@@ -17,6 +17,8 @@
  */
 package org.apache.metron.rest.config;
 
+import static org.apache.metron.rest.MetronRestConstants.INDEX_DAO_IMPL;
+
 import org.apache.metron.hbase.HTableProvider;
 import org.apache.metron.hbase.TableProvider;
 import org.apache.metron.indexing.dao.AccessConfig;
@@ -28,14 +30,8 @@ import org.apache.metron.rest.service.GlobalConfigService;
 import org.springframework.beans.factory.annotation.Autowired;
 import org.springframework.context.annotation.Bean;
 import org.springframework.context.annotation.Configuration;
-import org.springframework.context.annotation.Profile;
 import org.springframework.core.env.Environment;
 
-import java.lang.reflect.InvocationTargetException;
-
-import static org.apache.metron.rest.MetronRestConstants.INDEX_DAO_IMPL;
-import static org.apache.metron.rest.MetronRestConstants.TEST_PROFILE;
-
 @Configuration
 public class IndexConfig {
 
@@ -55,9 +51,11 @@ public class IndexConfig {
     try {
       String hbaseProviderImpl = environment.getProperty(MetronRestConstants.INDEX_HBASE_TABLE_PROVIDER_IMPL, String.class, null);
       String indexDaoImpl = environment.getProperty(MetronRestConstants.INDEX_DAO_IMPL, String.class, null);
-      int searchMaxResults = environment.getProperty(MetronRestConstants.SEARCH_MAX_RESULTS, Integer.class, -1);
+      int searchMaxResults = environment.getProperty(MetronRestConstants.SEARCH_MAX_RESULTS, Integer.class, 1000);
+      int searchMaxGroups = environment.getProperty(MetronRestConstants.SEARCH_MAX_GROUPS, Integer.class, 1000);
       AccessConfig config = new AccessConfig();
       config.setMaxSearchResults(searchMaxResults);
+      config.setMaxSearchGroups(searchMaxGroups);
       config.setGlobalConfigSupplier(() -> {
         try {
           return globalConfigService.get();

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/config/KafkaConfig.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/config/KafkaConfig.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/config/KafkaConfig.java
index 247264b..a2abbeb 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/config/KafkaConfig.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/config/KafkaConfig.java
@@ -21,6 +21,7 @@ import kafka.admin.AdminUtils$;
 import kafka.utils.ZkUtils;
 import org.I0Itec.zkclient.ZkClient;
 import org.apache.kafka.clients.consumer.KafkaConsumer;
+import org.apache.kafka.clients.producer.KafkaProducer;
 import org.apache.metron.rest.MetronRestConstants;
 import org.springframework.beans.factory.annotation.Autowired;
 import org.springframework.context.annotation.Bean;
@@ -100,6 +101,21 @@ public class KafkaConfig {
     return new DefaultKafkaConsumerFactory<>(consumerProperties());
   }
 
+  @Bean
+  public Map<String, Object> producerProperties() {
+    Map<String, Object> producerConfig = new HashMap<>();
+    producerConfig.put("bootstrap.servers", environment.getProperty(MetronRestConstants.KAFKA_BROKER_URL_SPRING_PROPERTY));
+    producerConfig.put("key.serializer", "org.apache.kafka.common.serialization.StringSerializer");
+    producerConfig.put("value.serializer", "org.apache.kafka.common.serialization.StringSerializer");
+    producerConfig.put("request.required.acks", 1);
+    return producerConfig;
+  }
+
+  @Bean
+  public KafkaProducer kafkaProducer() {
+    return new KafkaProducer<>(producerProperties());
+  }
+
   /**
    * Create a bean for {@link AdminUtils$}. This is primarily done to make testing a bit easier.
    *

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/AlertsController.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/AlertsController.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/AlertsController.java
new file mode 100644
index 0000000..6f028a1
--- /dev/null
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/AlertsController.java
@@ -0,0 +1,55 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.metron.rest.controller;
+
+import io.swagger.annotations.ApiOperation;
+import io.swagger.annotations.ApiParam;
+import io.swagger.annotations.ApiResponse;
+import java.util.List;
+import java.util.Map;
+import org.apache.metron.rest.RestException;
+import org.apache.metron.rest.service.AlertService;
+import org.springframework.beans.factory.annotation.Autowired;
+import org.springframework.http.HttpStatus;
+import org.springframework.http.ResponseEntity;
+import org.springframework.web.bind.annotation.RequestBody;
+import org.springframework.web.bind.annotation.RequestMapping;
+import org.springframework.web.bind.annotation.RequestMethod;
+import org.springframework.web.bind.annotation.RestController;
+
+/**
+ * The API resource that is used for alert-related operations.
+ */
+@RestController
+@RequestMapping("/api/v1/alert")
+public class AlertsController {
+
+  /**
+   * Service used to interact with alerts.
+   */
+  @Autowired
+  private AlertService alertService;
+
+  @ApiOperation(value = "Escalates a list of alerts by producing it to the Kafka escalate topic")
+  @ApiResponse(message = "Alerts were escalated", code = 200)
+  @RequestMapping(value = "/escalate", method = RequestMethod.POST)
+  ResponseEntity<Void> escalate(final @ApiParam(name = "alerts", value = "The alerts to be escalated", required = true) @RequestBody List<Map<String, Object>> alerts) throws RestException {
+    alertService.escalateAlerts(alerts);
+    return new ResponseEntity<>(HttpStatus.OK);
+  }
+}

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/HdfsController.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/HdfsController.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/HdfsController.java
index b8957a9..80b5942 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/HdfsController.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/HdfsController.java
@@ -67,9 +67,14 @@ public class HdfsController {
   @ApiOperation(value = "Writes contents to an HDFS file.  Warning: this will overwrite the contents of a file if it already exists.")
   @ApiResponse(message = "Contents were written", code = 200)
   @RequestMapping(method = RequestMethod.POST)
-  ResponseEntity<Void> write(@ApiParam(name="path", value="Path to HDFS file", required=true) @RequestParam String path,
-                                              @ApiParam(name="contents", value="File contents", required=true) @RequestBody String contents) throws RestException {
-    hdfsService.write(new Path(path), contents.getBytes(UTF_8));
+  ResponseEntity<Void> write(
+      @ApiParam(name = "path", value = "Path to HDFS file", required = true) @RequestParam String path,
+      @ApiParam(name = "contents", value = "File contents", required = true) @RequestBody String contents,
+      @ApiParam(name = "userMode", value = "requested user permissions") @RequestParam(required = false, defaultValue = "") String userMode,
+      @ApiParam(name = "groupMode", value = "requested group permissions") @RequestParam(required = false, defaultValue = "") String groupMode,
+      @ApiParam(name = "otherMode", value = "requested other permissions") @RequestParam(required = false, defaultValue = "") String otherMode
+      ) throws RestException {
+    hdfsService.write(new Path(path), contents.getBytes(UTF_8), userMode, groupMode, otherMode);
     return new ResponseEntity<>(HttpStatus.OK);
 
   }

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/KafkaController.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/KafkaController.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/KafkaController.java
index 2787504..d057ac4 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/KafkaController.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/KafkaController.java
@@ -21,6 +21,7 @@ import io.swagger.annotations.ApiOperation;
 import io.swagger.annotations.ApiParam;
 import io.swagger.annotations.ApiResponse;
 import io.swagger.annotations.ApiResponses;
+import java.util.Set;
 import org.apache.metron.rest.RestException;
 import org.apache.metron.rest.model.KafkaTopic;
 import org.apache.metron.rest.service.KafkaService;
@@ -33,8 +34,6 @@ import org.springframework.web.bind.annotation.RequestMapping;
 import org.springframework.web.bind.annotation.RequestMethod;
 import org.springframework.web.bind.annotation.RestController;
 
-import java.util.Set;
-
 /**
  * The API resource that is use to interact with Kafka.
  */
@@ -109,4 +108,15 @@ public class KafkaController {
       return new ResponseEntity<>(HttpStatus.NOT_FOUND);
     }
   }
+
+  @ApiOperation(value = "Produces a message to a Kafka topic")
+  @ApiResponses(value = {
+      @ApiResponse(message = "Message produced successfully", code = 200)
+  })
+  @RequestMapping(value = "/topic/{name}/produce", method = RequestMethod.POST)
+  ResponseEntity<String> produce(final @ApiParam(name = "name", value = "Kafka topic name", required = true) @PathVariable String name,
+      final @ApiParam(name = "message", value = "Message", required = true) @RequestBody String message) throws RestException {
+    kafkaService.produceMessage(name, message);
+    return new ResponseEntity<>(HttpStatus.OK);
+  }
 }

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/SearchController.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/SearchController.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/SearchController.java
index dea628c..e215413 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/SearchController.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/controller/SearchController.java
@@ -21,7 +21,8 @@ import io.swagger.annotations.ApiOperation;
 import io.swagger.annotations.ApiParam;
 import io.swagger.annotations.ApiResponse;
 import org.apache.metron.indexing.dao.search.GetRequest;
-import org.apache.metron.indexing.dao.update.Document;
+import org.apache.metron.indexing.dao.search.GroupRequest;
+import org.apache.metron.indexing.dao.search.GroupResponse;
 import org.apache.metron.indexing.dao.search.FieldType;
 import org.apache.metron.rest.RestException;
 import org.apache.metron.indexing.dao.search.SearchRequest;
@@ -38,7 +39,6 @@ import org.springframework.web.bind.annotation.RestController;
 import java.util.Map;
 import java.util.Optional;
 import java.util.List;
-import java.util.Map;
 
 @RestController
 @RequestMapping("/api/v1/search")
@@ -54,6 +54,15 @@ public class SearchController {
     return new ResponseEntity<>(searchService.search(searchRequest), HttpStatus.OK);
   }
 
+  @ApiOperation(value = "Searches the indexing store and returns field groups. "
+      + "Groups are hierarchical and nested in the order the fields appear in the 'groups' request parameter. "
+      + "The default sorting within groups is by count descending.")
+  @ApiResponse(message = "Group response", code = 200)
+  @RequestMapping(value = "/group", method = RequestMethod.POST)
+  ResponseEntity<GroupResponse> group(final @ApiParam(name = "groupRequest", value = "Group request", required = true) @RequestBody GroupRequest groupRequest) throws RestException {
+    return new ResponseEntity<>(searchService.group(groupRequest), HttpStatus.OK);
+  }
+
   @ApiOperation(value = "Returns latest document for a guid and sensor")
   @ApiResponse(message = "Document representing the output", code = 200)
   @RequestMapping(value = "/findOne", method = RequestMethod.POST)

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/AlertService.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/AlertService.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/AlertService.java
new file mode 100644
index 0000000..9668b7c
--- /dev/null
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/AlertService.java
@@ -0,0 +1,30 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.metron.rest.service;
+
+import java.util.List;
+import java.util.Map;
+import org.apache.metron.rest.RestException;
+
+/**
+ * This is a set of operations created to interact with alerts.
+ */
+public interface AlertService {
+
+  void escalateAlerts(List<Map<String, Object>> alerts) throws RestException;
+}

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/GrokService.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/GrokService.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/GrokService.java
index 8c3562e..ada5c0e 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/GrokService.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/GrokService.java
@@ -17,10 +17,10 @@
  */
 package org.apache.metron.rest.service;
 
+import org.apache.hadoop.fs.Path;
 import org.apache.metron.rest.RestException;
 import org.apache.metron.rest.model.GrokValidation;
 
-import java.io.File;
 import java.util.Map;
 
 public interface GrokService {
@@ -29,8 +29,6 @@ public interface GrokService {
 
     GrokValidation validateGrokStatement(GrokValidation grokValidation) throws RestException;
 
-    File saveTemporary(String statement, String name) throws RestException;
-
     String getStatement(String path) throws RestException;
 
     void saveStatement(String path, byte[] contents) throws RestException;

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/HdfsService.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/HdfsService.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/HdfsService.java
index 846f19b..55f7863 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/HdfsService.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/HdfsService.java
@@ -17,8 +17,6 @@
  */
 package org.apache.metron.rest.service;
 
-import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
 import org.apache.metron.rest.RestException;
 
@@ -28,11 +26,13 @@ public interface HdfsService {
 
     String read(Path path) throws RestException;
 
-    void write(Path path, byte[] contents) throws RestException;
+    void write(Path path, byte[] contents,String userMode, String groupMode, String otherMode) throws RestException;
 
     List<String> list(Path path) throws RestException;
 
     boolean delete(Path path, boolean recursive) throws RestException;
 
+    boolean mkdirs(Path path) throws RestException;
+
     boolean ensureDirectory(Path path) throws RestException;
  }

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/KafkaService.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/KafkaService.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/KafkaService.java
index bee00f2..da3b226 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/KafkaService.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/KafkaService.java
@@ -17,11 +17,10 @@
  */
 package org.apache.metron.rest.service;
 
+import java.util.Set;
 import org.apache.metron.rest.RestException;
 import org.apache.metron.rest.model.KafkaTopic;
 
-import java.util.Set;
-
 /**
  * This is a set of operations created to interact with Kafka.
  */
@@ -67,4 +66,6 @@ public interface KafkaService {
    * @return A string representation of the sample message retrieved. If topic doesn't exist null will be returned.
    */
   String getSampleMessage(String topic);
+
+  void produceMessage(String topic, String message) throws RestException;
 }

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/SearchService.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/SearchService.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/SearchService.java
index ea0ae81..5899765 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/SearchService.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/SearchService.java
@@ -18,7 +18,8 @@
 package org.apache.metron.rest.service;
 
 import org.apache.metron.indexing.dao.search.GetRequest;
-import org.apache.metron.indexing.dao.update.Document;
+import org.apache.metron.indexing.dao.search.GroupRequest;
+import org.apache.metron.indexing.dao.search.GroupResponse;
 import org.apache.metron.indexing.dao.search.FieldType;
 import org.apache.metron.rest.RestException;
 import org.apache.metron.indexing.dao.search.SearchRequest;
@@ -27,11 +28,11 @@ import org.apache.metron.indexing.dao.search.SearchResponse;
 import java.util.Map;
 import java.util.Optional;
 import java.util.List;
-import java.util.Map;
 
 public interface SearchService {
 
   SearchResponse search(SearchRequest searchRequest) throws RestException;
+  GroupResponse group(GroupRequest groupRequest) throws RestException;
   Optional<Map<String, Object>> getLatest(GetRequest request) throws RestException;
   Map<String, Map<String, FieldType>> getColumnMetadata(List<String> indices) throws RestException;
   Map<String, FieldType> getCommonColumnMetadata(List<String> indices) throws RestException;

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/AlertServiceImpl.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/AlertServiceImpl.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/AlertServiceImpl.java
new file mode 100644
index 0000000..46370eb
--- /dev/null
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/AlertServiceImpl.java
@@ -0,0 +1,62 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.metron.rest.service.impl;
+
+import com.fasterxml.jackson.core.JsonProcessingException;
+import java.util.List;
+import java.util.Map;
+import org.apache.metron.common.utils.JSONUtils;
+import org.apache.metron.rest.MetronRestConstants;
+import org.apache.metron.rest.RestException;
+import org.apache.metron.rest.service.AlertService;
+import org.apache.metron.rest.service.KafkaService;
+import org.springframework.beans.factory.annotation.Autowired;
+import org.springframework.core.env.Environment;
+import org.springframework.stereotype.Service;
+
+/**
+ * The default service layer implementation of {@link AlertService}.
+ *
+ * @see AlertService
+ */
+@Service
+public class AlertServiceImpl implements AlertService {
+
+  private Environment environment;
+  private final KafkaService kafkaService;
+
+  @Autowired
+  public AlertServiceImpl(final KafkaService kafkaService,
+                          final Environment environment) {
+    this.kafkaService = kafkaService;
+    this.environment = environment;
+  }
+
+  @Override
+  public void escalateAlerts(List<Map<String, Object>> alerts) throws RestException {
+    try {
+      for (Map<String, Object> alert : alerts) {
+        kafkaService.produceMessage(
+            environment.getProperty(MetronRestConstants.KAFKA_TOPICS_ESCALATION_PROPERTY),
+            JSONUtils.INSTANCE.toJSON(alert, false));
+      }
+    } catch (JsonProcessingException e) {
+      throw new RestException(e);
+    }
+  }
+}

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/ExtensionServiceImpl.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/ExtensionServiceImpl.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/ExtensionServiceImpl.java
index 3cbed70..971a5a3 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/ExtensionServiceImpl.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/ExtensionServiceImpl.java
@@ -445,7 +445,7 @@ public class ExtensionServiceImpl implements ExtensionService{
     BundleProperties props = context.bundleProperties.get();
     org.apache.hadoop.fs.Path altPath = new org.apache.hadoop.fs.Path(props.getProperty("bundle.library.directory.alt"));
     org.apache.hadoop.fs.Path targetPath = new org.apache.hadoop.fs.Path(altPath, bundlePath.toFile().getName());
-    hdfsService.write(targetPath,FileUtils.readFileToByteArray(bundlePath.toFile()));
+    hdfsService.write(targetPath,FileUtils.readFileToByteArray(bundlePath.toFile()),"rwx","r-x","r-x");
   }
 
   private void deleteBundleFromHdfs(String bundleName) throws Exception{
@@ -480,7 +480,7 @@ public class ExtensionServiceImpl implements ExtensionService{
       paths.add(parserRulePath);
       for(Path thisRule : grokRulePaths){
         org.apache.hadoop.fs.Path targetPath = new org.apache.hadoop.fs.Path(parserRulePath,thisRule.toFile().getName());
-        hdfsService.write(targetPath,FileUtils.readFileToByteArray(thisRule.toFile()));
+        hdfsService.write(targetPath,FileUtils.readFileToByteArray(thisRule.toFile()),"rwx","r-x","r-x");
       }
     }
     context.hdfsPathContext.put(Paths.GROK_HDFS_DIRS,paths);

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/GrokServiceImpl.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/GrokServiceImpl.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/GrokServiceImpl.java
index 9d48203..d420955 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/GrokServiceImpl.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/GrokServiceImpl.java
@@ -20,6 +20,7 @@ package org.apache.metron.rest.service.impl;
 import java.io.InputStream;
 import java.nio.file.Paths;
 import java.util.HashMap;
+import java.nio.charset.Charset;
 import oi.thekraken.grok.api.Grok;
 import oi.thekraken.grok.api.Match;
 import org.apache.commons.io.IOUtils;
@@ -40,9 +41,6 @@ import org.springframework.security.core.Authentication;
 import org.springframework.security.core.context.SecurityContextHolder;
 import org.springframework.stereotype.Service;
 
-import java.io.File;
-import java.io.FileWriter;
-import java.io.IOException;
 import java.io.InputStreamReader;
 import java.io.StringReader;
 import java.util.Map;
@@ -104,42 +102,13 @@ public class GrokServiceImpl implements GrokService {
     }
 
     @Override
-    public File saveTemporary(String statement, String name) throws RestException {
-        if (statement != null) {
-            try {
-                File grokDirectory = new File(getTemporaryGrokRootPath());
-                if (!grokDirectory.exists()) {
-                  grokDirectory.mkdirs();
-                }
-                File path = new File(grokDirectory, name);
-                FileWriter fileWriter = new FileWriter(new File(grokDirectory, name));
-                fileWriter.write(statement);
-                fileWriter.close();
-                return path;
-            } catch (IOException e) {
-                throw new RestException(e);
-            }
-        } else {
-            throw new RestException("A grokStatement must be provided");
-        }
-    }
-
-    @Override
     public void saveStatement(String path, byte[] contents) throws RestException {
         String root = (String)configurationMap.get("metron.apps.hdfs.dir");
         if(!root.endsWith("/") && !path.startsWith("/")) {
             root = root + "/";
         }
         Path rootedPath = new Path(root + path);
-        hdfsService.write(rootedPath, contents);
-    }
-
-    private String getTemporaryGrokRootPath() {
-        String javaTmp = System.getProperty("java.io.tmpdir");
-        String grokTempPath = Paths
-            .get(javaTmp, environment.getProperty(GROK_TEMP_PATH_SPRING_PROPERTY)).toString();
-        Authentication authentication = SecurityContextHolder.getContext().getAuthentication();
-        return new Path(grokTempPath, authentication.getName()).toString();
+        hdfsService.write(rootedPath, contents, null, null, null);
     }
 
     public String getStatement(String path) throws RestException {

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/HdfsServiceImpl.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/HdfsServiceImpl.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/HdfsServiceImpl.java
index 82565eb..eb0d705 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/HdfsServiceImpl.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/HdfsServiceImpl.java
@@ -17,10 +17,13 @@
  */
 package org.apache.metron.rest.service.impl;
 
+import org.apache.commons.lang.StringUtils;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.FSDataOutputStream;
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
+import org.apache.hadoop.fs.permission.FsAction;
+import org.apache.hadoop.fs.permission.FsPermission;
 import org.apache.hadoop.io.IOUtils;
 import org.apache.metron.rest.RestException;
 import org.apache.metron.rest.service.HdfsService;
@@ -68,22 +71,49 @@ public class HdfsServiceImpl implements HdfsService {
       return new String(byteArrayOutputStream.toByteArray(), UTF_8);
     }
 
+  @Override
+  public void write(Path path, byte[] contents, String userMode, String groupMode, String otherMode)
+      throws RestException {
+    FSDataOutputStream fsDataOutputStream;
+    try {
+      FsPermission permission = null;
+      boolean setPermissions = false;
+      if(StringUtils.isNotEmpty(userMode) && StringUtils.isNotEmpty(groupMode) && StringUtils.isNotEmpty(otherMode)) {
+        // invalid actions will return null
+        FsAction userAction = FsAction.getFsAction(userMode);
+        FsAction groupAction = FsAction.getFsAction(groupMode);
+        FsAction otherAction = FsAction.getFsAction(otherMode);
+        if(userAction == null || groupAction == null || otherAction == null){
+          throw new RestException(String.format("Invalid permission set: user[%s] " +
+              "group[%s] other[%s]", userAction, groupAction, otherAction));
+        }
+        permission = new FsPermission(userAction, groupAction, otherAction);
+        setPermissions = true;
+      }
+      fsDataOutputStream = FileSystem.get(configuration).create(path, true);
+      fsDataOutputStream.write(contents);
+      fsDataOutputStream.close();
+      if(setPermissions) {
+        FileSystem.get(configuration).setPermission(path, permission);
+      }
+    } catch (IOException e) {
+      throw new RestException(e);
+    }
+  }
+
     @Override
-    public void write(Path path, byte[] contents) throws RestException {
-      FSDataOutputStream fsDataOutputStream;
+    public boolean delete(Path path, boolean recursive) throws RestException {
       try {
-        fsDataOutputStream = FileSystem.get(configuration).create(path, true);
-        fsDataOutputStream.write(contents);
-        fsDataOutputStream.close();
+        return FileSystem.get(configuration).delete(path, recursive);
       } catch (IOException e) {
         throw new RestException(e);
       }
     }
 
     @Override
-    public boolean delete(Path path, boolean recursive) throws RestException {
+    public boolean mkdirs(Path path) throws RestException {
       try {
-        return FileSystem.get(configuration).delete(path, recursive);
+        return FileSystem.get(configuration).mkdirs(path);
       } catch (IOException e) {
         throw new RestException(e);
       }

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/KafkaServiceImpl.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/KafkaServiceImpl.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/KafkaServiceImpl.java
index 61e2618..4f232fb 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/KafkaServiceImpl.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/KafkaServiceImpl.java
@@ -17,6 +17,11 @@
  */
 package org.apache.metron.rest.service.impl;
 
+import java.util.HashSet;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.util.stream.Collectors;
 import kafka.admin.AdminOperationException;
 import kafka.admin.AdminUtils$;
 import kafka.admin.RackAwareMode;
@@ -24,21 +29,18 @@ import kafka.utils.ZkUtils;
 import org.apache.kafka.clients.consumer.Consumer;
 import org.apache.kafka.clients.consumer.ConsumerRecords;
 import org.apache.kafka.clients.consumer.KafkaConsumer;
+import org.apache.kafka.clients.producer.KafkaProducer;
+import org.apache.kafka.clients.producer.ProducerRecord;
 import org.apache.kafka.common.PartitionInfo;
 import org.apache.kafka.common.TopicPartition;
 import org.apache.metron.rest.RestException;
 import org.apache.metron.rest.model.KafkaTopic;
 import org.apache.metron.rest.service.KafkaService;
 import org.springframework.beans.factory.annotation.Autowired;
+import org.springframework.core.env.Environment;
 import org.springframework.kafka.core.ConsumerFactory;
 import org.springframework.stereotype.Service;
 
-import java.util.HashSet;
-import java.util.List;
-import java.util.Map;
-import java.util.Set;
-import java.util.stream.Collectors;
-
 /**
  * The default service layer implementation of {@link KafkaService}.
  *
@@ -53,19 +55,26 @@ public class KafkaServiceImpl implements KafkaService {
 
   private final ZkUtils zkUtils;
   private final ConsumerFactory<String, String> kafkaConsumerFactory;
+  private final KafkaProducer<String, String> kafkaProducer;
   private final AdminUtils$ adminUtils;
 
+  @Autowired
+  private Environment environment;
+
   /**
    * @param zkUtils              A utility class used to interact with ZooKeeper.
    * @param kafkaConsumerFactory A class used to create {@link KafkaConsumer} in order to interact with Kafka.
+   * @param kafkaProducer        A class used to produce messages to Kafka.
    * @param adminUtils           A utility class used to do administration operations on Kafka.
    */
   @Autowired
   public KafkaServiceImpl(final ZkUtils zkUtils,
                           final ConsumerFactory<String, String> kafkaConsumerFactory,
+                          final KafkaProducer<String, String> kafkaProducer,
                           final AdminUtils$ adminUtils) {
     this.zkUtils = zkUtils;
     this.kafkaConsumerFactory = kafkaConsumerFactory;
+    this.kafkaProducer = kafkaProducer;
     this.adminUtils = adminUtils;
   }
 
@@ -141,4 +150,8 @@ public class KafkaServiceImpl implements KafkaService {
     return message;
   }
 
+  @Override
+  public void produceMessage(String topic, String message) throws RestException {
+    kafkaProducer.send(new ProducerRecord<>(topic, message));
+  }
 }

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/SearchServiceImpl.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/SearchServiceImpl.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/SearchServiceImpl.java
index bdf6037..d865e0e 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/SearchServiceImpl.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/SearchServiceImpl.java
@@ -19,10 +19,11 @@ package org.apache.metron.rest.service.impl;
 
 import org.apache.metron.indexing.dao.IndexDao;
 import org.apache.metron.indexing.dao.search.GetRequest;
+import org.apache.metron.indexing.dao.search.GroupRequest;
+import org.apache.metron.indexing.dao.search.GroupResponse;
 import org.apache.metron.indexing.dao.search.InvalidSearchException;
 import org.apache.metron.indexing.dao.search.SearchRequest;
 import org.apache.metron.indexing.dao.search.SearchResponse;
-import org.apache.metron.indexing.dao.update.Document;
 import org.apache.metron.indexing.dao.search.FieldType;
 import org.apache.metron.rest.RestException;
 import org.apache.metron.rest.service.SearchService;
@@ -34,7 +35,6 @@ import java.io.IOException;
 import java.util.Map;
 import java.util.Optional;
 import java.util.List;
-import java.util.Map;
 
 @Service
 public class SearchServiceImpl implements SearchService {
@@ -58,6 +58,16 @@ public class SearchServiceImpl implements SearchService {
   }
 
   @Override
+  public GroupResponse group(GroupRequest groupRequest) throws RestException {
+    try {
+      return dao.group(groupRequest);
+    }
+    catch(InvalidSearchException ise) {
+      throw new RestException(ise.getMessage(), ise);
+    }
+  }
+
+  @Override
   public Optional<Map<String, Object>> getLatest(GetRequest request) throws RestException {
     try {
       return dao.getLatestResult(request);

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/SensorParserConfigServiceImpl.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/SensorParserConfigServiceImpl.java b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/SensorParserConfigServiceImpl.java
index 5d6f264..619f2b2 100644
--- a/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/SensorParserConfigServiceImpl.java
+++ b/metron-interface/metron-rest/src/main/java/org/apache/metron/rest/service/impl/SensorParserConfigServiceImpl.java
@@ -17,7 +17,15 @@
  */
 package org.apache.metron.rest.service.impl;
 
+import static org.apache.metron.rest.MetronRestConstants.GROK_CLASS_NAME;
+
 import com.fasterxml.jackson.databind.ObjectMapper;
+import java.util.ArrayList;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.io.File;
 import java.io.InputStreamReader;
 import java.io.Reader;
 import java.io.StringReader;
@@ -47,15 +55,6 @@ import org.springframework.beans.factory.annotation.Autowired;
 import org.springframework.core.env.Environment;
 import org.springframework.stereotype.Service;
 
-import java.io.File;
-import java.util.ArrayList;
-import java.util.HashMap;
-import java.util.List;
-import java.util.Map;
-import java.util.Set;
-
-import static org.apache.metron.rest.MetronRestConstants.GROK_CLASS_NAME;
-
 @Service
 public class SensorParserConfigServiceImpl implements SensorParserConfigService {
   private static final Logger LOG = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());
@@ -101,7 +100,8 @@ public class SensorParserConfigServiceImpl implements SensorParserConfigService
   @Override
   public SensorParserConfig save(SensorParserConfig sensorParserConfig) throws RestException {
     try {
-      ConfigurationsUtils.writeSensorParserConfigToZookeeper(sensorParserConfig.getSensorTopic(), objectMapper.writeValueAsString(sensorParserConfig).getBytes(), client);
+      ConfigurationsUtils.writeSensorParserConfigToZookeeper(sensorParserConfig.getSensorTopic(),
+          objectMapper.writeValueAsString(sensorParserConfig).getBytes(), client);
     } catch (Exception e) {
       throw new RestException(e);
     }

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/resources/application-docker.yml
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/resources/application-docker.yml b/metron-interface/metron-rest/src/main/resources/application-docker.yml
index 15fa293..426a0da 100644
--- a/metron-interface/metron-rest/src/main/resources/application-docker.yml
+++ b/metron-interface/metron-rest/src/main/resources/application-docker.yml
@@ -46,7 +46,6 @@ hdfs:
 grok:
   path:
     temp: target/patterns/temp
-    default: target/patterns
 
 storm:
   ui:

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/resources/application-vagrant.yml
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/resources/application-vagrant.yml b/metron-interface/metron-rest/src/main/resources/application-vagrant.yml
index 47d1750..29ff02d 100644
--- a/metron-interface/metron-rest/src/main/resources/application-vagrant.yml
+++ b/metron-interface/metron-rest/src/main/resources/application-vagrant.yml
@@ -40,8 +40,7 @@ hdfs:
     apps.root: /apps/metron
 grok:
   path:
-    temp: ./patterns/temp
-    default: /apps/metron/patterns
+    temp: /apps/metron/patterns/tmp
 
 storm:
   ui:

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/main/resources/application.yml
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/main/resources/application.yml b/metron-interface/metron-rest/src/main/resources/application.yml
index 4aff769..3aa5fd9 100644
--- a/metron-interface/metron-rest/src/main/resources/application.yml
+++ b/metron-interface/metron-rest/src/main/resources/application.yml
@@ -26,8 +26,7 @@ spring:
 
 grok:
   path:
-    temp: ./
-    default: /apps/metron/patterns
+    temp: /apps/metron/patterns/tmp
 
 zookeeper:
   client:
@@ -35,6 +34,10 @@ zookeeper:
       session: 10000
       connection: 10000
 
+kafka:
+  topics:
+    escalation: escalation
+
 curator:
   sleep:
     time: 1000
@@ -44,8 +47,10 @@ curator:
 search:
   max:
     results: 1000
+    groups: 1000
 
 index:
   dao:
   # By default, we use the ElasticsearchDao and HBaseDao for backing updates.
-     impl: org.apache.metron.elasticsearch.dao.ElasticsearchDao,org.apache.metron.indexing.dao.HBaseDao
\ No newline at end of file
+     impl: org.apache.metron.elasticsearch.dao.ElasticsearchDao,org.apache.metron.indexing.dao.HBaseDao
+

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/config/TestConfig.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/config/TestConfig.java b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/config/TestConfig.java
index dbf4aa4..7095d11 100644
--- a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/config/TestConfig.java
+++ b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/config/TestConfig.java
@@ -36,6 +36,7 @@ import org.apache.curator.RetryPolicy;
 import org.apache.curator.framework.CuratorFramework;
 import org.apache.curator.framework.CuratorFrameworkFactory;
 import org.apache.curator.retry.ExponentialBackoffRetry;
+import org.apache.kafka.clients.producer.KafkaProducer;
 import org.apache.metron.common.configuration.ConfigurationsUtils;
 import org.apache.metron.hbase.mock.MockHBaseTableProvider;
 import org.apache.metron.integration.ComponentRunner;
@@ -142,6 +143,21 @@ public class TestConfig {
   }
 
   @Bean
+  public Map<String, Object> producerProperties(KafkaComponent kafkaWithZKComponent) {
+    Map<String, Object> producerConfig = new HashMap<>();
+    producerConfig.put("bootstrap.servers", kafkaWithZKComponent.getBrokerList());
+    producerConfig.put("key.serializer", "org.apache.kafka.common.serialization.StringSerializer");
+    producerConfig.put("value.serializer", "org.apache.kafka.common.serialization.StringSerializer");
+    producerConfig.put("request.required.acks", 1);
+    return producerConfig;
+  }
+
+  @Bean
+  public KafkaProducer kafkaProducer(KafkaComponent kafkaWithZKComponent) {
+    return new KafkaProducer<>(producerProperties(kafkaWithZKComponent));
+  }
+
+  @Bean
   public StormCLIWrapper stormCLIClientWrapper() {
     return new MockStormCLIClientWrapper();
   }

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/AlertControllerIntegrationTest.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/AlertControllerIntegrationTest.java b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/AlertControllerIntegrationTest.java
new file mode 100644
index 0000000..be320fc
--- /dev/null
+++ b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/AlertControllerIntegrationTest.java
@@ -0,0 +1,86 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.metron.rest.controller;
+
+import static org.apache.metron.rest.MetronRestConstants.TEST_PROFILE;
+import static org.springframework.security.test.web.servlet.request.SecurityMockMvcRequestPostProcessors.csrf;
+import static org.springframework.security.test.web.servlet.request.SecurityMockMvcRequestPostProcessors.httpBasic;
+import static org.springframework.security.test.web.servlet.setup.SecurityMockMvcConfigurers.springSecurity;
+import static org.springframework.test.web.servlet.request.MockMvcRequestBuilders.post;
+import static org.springframework.test.web.servlet.result.MockMvcResultMatchers.status;
+
+import org.adrianwalker.multilinestring.Multiline;
+import org.junit.Before;
+import org.junit.Test;
+import org.junit.runner.RunWith;
+import org.springframework.beans.factory.annotation.Autowired;
+import org.springframework.boot.test.context.SpringBootTest;
+import org.springframework.http.MediaType;
+import org.springframework.test.context.ActiveProfiles;
+import org.springframework.test.context.junit4.SpringRunner;
+import org.springframework.test.web.servlet.MockMvc;
+import org.springframework.test.web.servlet.setup.MockMvcBuilders;
+import org.springframework.web.context.WebApplicationContext;
+
+@RunWith(SpringRunner.class)
+@SpringBootTest(webEnvironment= SpringBootTest.WebEnvironment.RANDOM_PORT)
+@ActiveProfiles(TEST_PROFILE)
+public class AlertControllerIntegrationTest {
+
+    /**
+     [
+     {
+     "is_alert": true,
+     "field": "value1"
+     },
+     {
+     "is_alert": true,
+     "field": "value2"
+     }
+     ]
+     */
+    @Multiline
+    public static String alerts;
+
+    @Autowired
+    private WebApplicationContext wac;
+
+    private MockMvc mockMvc;
+
+    private String alertUrl = "/api/v1/alert";
+    private String user = "user";
+    private String password = "password";
+
+    @Before
+    public void setup() throws Exception {
+        this.mockMvc = MockMvcBuilders.webAppContextSetup(this.wac).apply(springSecurity()).build();
+    }
+
+    @Test
+    public void testSecurity() throws Exception {
+        this.mockMvc.perform(post(alertUrl + "/escalate").with(csrf()).contentType(MediaType.parseMediaType("application/json;charset=UTF-8")).content(alerts))
+                .andExpect(status().isUnauthorized());
+    }
+
+    @Test
+    public void test() throws Exception {
+        this.mockMvc.perform(post(alertUrl + "/escalate").with(httpBasic(user,password)).with(csrf()).contentType(MediaType.parseMediaType("application/json;charset=UTF-8")).content(alerts))
+                .andExpect(status().isOk());
+
+    }
+}

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/KafkaControllerIntegrationTest.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/KafkaControllerIntegrationTest.java b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/KafkaControllerIntegrationTest.java
index 5c00cd2..5719a41 100644
--- a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/KafkaControllerIntegrationTest.java
+++ b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/KafkaControllerIntegrationTest.java
@@ -17,15 +17,24 @@
  */
 package org.apache.metron.rest.controller;
 
+import static org.apache.metron.rest.MetronRestConstants.TEST_PROFILE;
+import static org.springframework.security.test.web.servlet.request.SecurityMockMvcRequestPostProcessors.csrf;
+import static org.springframework.security.test.web.servlet.request.SecurityMockMvcRequestPostProcessors.httpBasic;
+import static org.springframework.security.test.web.servlet.setup.SecurityMockMvcConfigurers.springSecurity;
+import static org.springframework.test.web.servlet.request.MockMvcRequestBuilders.delete;
+import static org.springframework.test.web.servlet.request.MockMvcRequestBuilders.get;
+import static org.springframework.test.web.servlet.request.MockMvcRequestBuilders.post;
+import static org.springframework.test.web.servlet.result.MockMvcResultMatchers.content;
+import static org.springframework.test.web.servlet.result.MockMvcResultMatchers.jsonPath;
+import static org.springframework.test.web.servlet.result.MockMvcResultMatchers.status;
+
 import kafka.common.TopicAlreadyMarkedForDeletionException;
 import org.adrianwalker.multilinestring.Multiline;
 import org.apache.metron.integration.ComponentRunner;
 import org.apache.metron.integration.UnableToStartException;
 import org.apache.metron.integration.components.KafkaComponent;
-import org.apache.metron.rest.generator.SampleDataGenerator;
 import org.apache.metron.rest.service.KafkaService;
 import org.hamcrest.Matchers;
-import org.json.simple.parser.ParseException;
 import org.junit.After;
 import org.junit.Before;
 import org.junit.Test;
@@ -41,19 +50,6 @@ import org.springframework.test.web.servlet.setup.MockMvcBuilders;
 import org.springframework.web.context.WebApplicationContext;
 import org.springframework.web.util.NestedServletException;
 
-import java.io.IOException;
-
-import static org.apache.metron.rest.MetronRestConstants.TEST_PROFILE;
-import static org.springframework.security.test.web.servlet.request.SecurityMockMvcRequestPostProcessors.csrf;
-import static org.springframework.security.test.web.servlet.request.SecurityMockMvcRequestPostProcessors.httpBasic;
-import static org.springframework.security.test.web.servlet.setup.SecurityMockMvcConfigurers.springSecurity;
-import static org.springframework.test.web.servlet.request.MockMvcRequestBuilders.delete;
-import static org.springframework.test.web.servlet.request.MockMvcRequestBuilders.get;
-import static org.springframework.test.web.servlet.request.MockMvcRequestBuilders.post;
-import static org.springframework.test.web.servlet.result.MockMvcResultMatchers.content;
-import static org.springframework.test.web.servlet.result.MockMvcResultMatchers.jsonPath;
-import static org.springframework.test.web.servlet.result.MockMvcResultMatchers.status;
-
 @RunWith(SpringRunner.class)
 @SpringBootTest(webEnvironment= SpringBootTest.WebEnvironment.RANDOM_PORT)
 @ActiveProfiles(TEST_PROFILE)
@@ -99,35 +95,6 @@ public class KafkaControllerIntegrationTest {
     }
   }
 
-  class SampleDataRunner implements Runnable {
-
-        private boolean stop = false;
-        private String path = "../../metron-platform/metron-extensions/metron-parser-extensions/metron-parser-bro-extension/metron-parser-bro/src/test/resources/data/raw/test.raw";
-
-    @Override
-    public void run() {
-      SampleDataGenerator broSampleDataGenerator = new SampleDataGenerator();
-      broSampleDataGenerator.setBrokerUrl(kafkaWithZKComponent.getBrokerList());
-      broSampleDataGenerator.setNum(1);
-      broSampleDataGenerator.setSelectedSensorType("bro");
-      broSampleDataGenerator.setDelay(0);
-      try {
-        while(!stop) {
-          broSampleDataGenerator.generateSampleData(path);
-        }
-      } catch (ParseException|IOException e) {
-        throw new IllegalStateException("Caught an error generating sample data", e);
-      }
-    }
-
-    public void stop() {
-      stop = true;
-    }
-  }
-
-  private SampleDataRunner sampleDataRunner =  new SampleDataRunner();
-  private Thread sampleDataThread = new Thread(sampleDataRunner);
-
   /**
    {
    "name": "bro",
@@ -139,6 +106,30 @@ public class KafkaControllerIntegrationTest {
   @Multiline
   public static String broTopic;
 
+  /**
+   * {
+   *   "type":"message1"
+   * }
+   */
+  @Multiline
+  public static String message1;
+
+  /**
+   * {
+   *   "type":"message2"
+   * }
+   */
+  @Multiline
+  public static String message2;
+
+  /**
+   * {
+   *   "type":"message3"
+   * }
+   */
+  @Multiline
+  public static String message3;
+
   @Autowired
   private WebApplicationContext wac;
 
@@ -179,6 +170,9 @@ public class KafkaControllerIntegrationTest {
     this.mockMvc.perform(get(kafkaUrl + "/topic/bro/sample"))
             .andExpect(status().isUnauthorized());
 
+    this.mockMvc.perform(get(kafkaUrl + "/topic/bro/produce"))
+        .andExpect(status().isUnauthorized());
+
     this.mockMvc.perform(delete(kafkaUrl + "/topic/bro").with(csrf()))
             .andExpect(status().isUnauthorized());
   }
@@ -200,7 +194,7 @@ public class KafkaControllerIntegrationTest {
             .andExpect(jsonPath("$.numPartitions").value(1))
             .andExpect(jsonPath("$.replicationFactor").value(1))
     );
-    sampleDataThread.start();
+
     Thread.sleep(1000);
     testAndRetry(() ->
     this.mockMvc.perform(get(kafkaUrl + "/topic/bro").with(httpBasic(user,password)))
@@ -222,24 +216,42 @@ public class KafkaControllerIntegrationTest {
             .andExpect(jsonPath("$", Matchers.hasItem("bro")))
     );
 
-    for(int i = 0;i < KAFKA_RETRY;++i) {
-      MvcResult result = this.mockMvc.perform(get(kafkaUrl + "/topic/bro/sample").with(httpBasic(user, password)))
-              .andReturn();
-      if(result.getResponse().getStatus() == 200) {
-        break;
-      }
-      Thread.sleep(1000);
-    }
+    testAndRetry(() ->
+        this.mockMvc.perform(post(kafkaUrl + "/topic/bro/produce").with(httpBasic(user,password)).with(csrf()).contentType(MediaType.parseMediaType("application/json;charset=UTF-8")).content(message1))
+            .andExpect(status().isOk())
+    );
+    testAndRetry(() ->
+        this.mockMvc.perform(get(kafkaUrl + "/topic/bro/sample").with(httpBasic(user,password)))
+            .andExpect(status().isOk())
+            .andExpect(content().contentType(MediaType.parseMediaType("text/plain;charset=UTF-8")))
+            .andExpect(jsonPath("$.type").value("message1"))
+    );
 
     testAndRetry(() ->
-    this.mockMvc.perform(get(kafkaUrl + "/topic/bro/sample").with(httpBasic(user,password)))
+        this.mockMvc.perform(post(kafkaUrl + "/topic/bro/produce").with(httpBasic(user,password)).with(csrf()).contentType(MediaType.parseMediaType("application/json;charset=UTF-8")).content(message2))
+            .andExpect(status().isOk())
+    );
+    testAndRetry(() ->
+        this.mockMvc.perform(get(kafkaUrl + "/topic/bro/sample").with(httpBasic(user,password)))
             .andExpect(status().isOk())
             .andExpect(content().contentType(MediaType.parseMediaType("text/plain;charset=UTF-8")))
-            .andExpect(jsonPath("$").isNotEmpty())
+            .andExpect(jsonPath("$.type").value("message2"))
+    );
+
+    testAndRetry(() ->
+        this.mockMvc.perform(post(kafkaUrl + "/topic/bro/produce").with(httpBasic(user,password)).with(csrf()).contentType(MediaType.parseMediaType("application/json;charset=UTF-8")).content(message3))
+            .andExpect(status().isOk())
+    );
+    testAndRetry(() ->
+        this.mockMvc.perform(get(kafkaUrl + "/topic/bro/sample").with(httpBasic(user,password)))
+            .andExpect(status().isOk())
+            .andExpect(content().contentType(MediaType.parseMediaType("text/plain;charset=UTF-8")))
+            .andExpect(jsonPath("$.type").value("message3"))
     );
 
     this.mockMvc.perform(get(kafkaUrl + "/topic/someTopic/sample").with(httpBasic(user,password)))
             .andExpect(status().isNotFound());
+
     boolean deleted = false;
     for(int i = 0;i < KAFKA_RETRY;++i) {
       try {
@@ -270,7 +282,6 @@ public class KafkaControllerIntegrationTest {
 
   @After
   public void tearDown() {
-    sampleDataRunner.stop();
     runner.stop();
   }
 }

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/SearchControllerIntegrationTest.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/SearchControllerIntegrationTest.java b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/SearchControllerIntegrationTest.java
index e75c356..645e525 100644
--- a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/SearchControllerIntegrationTest.java
+++ b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/SearchControllerIntegrationTest.java
@@ -24,7 +24,6 @@ import org.apache.metron.rest.service.SearchService;
 import org.json.simple.parser.ParseException;
 import org.junit.After;
 import org.junit.Before;
-import org.junit.BeforeClass;
 import org.junit.Test;
 import org.junit.runner.RunWith;
 import org.springframework.beans.factory.annotation.Autowired;
@@ -44,7 +43,6 @@ import static org.hamcrest.Matchers.hasSize;
 import static org.springframework.security.test.web.servlet.request.SecurityMockMvcRequestPostProcessors.csrf;
 import static org.springframework.security.test.web.servlet.request.SecurityMockMvcRequestPostProcessors.httpBasic;
 import static org.springframework.security.test.web.servlet.setup.SecurityMockMvcConfigurers.springSecurity;
-import static org.springframework.test.web.servlet.request.MockMvcRequestBuilders.get;
 import static org.springframework.test.web.servlet.request.MockMvcRequestBuilders.post;
 import static org.springframework.test.web.servlet.result.MockMvcResultMatchers.content;
 import static org.springframework.test.web.servlet.result.MockMvcResultMatchers.jsonPath;
@@ -174,6 +172,22 @@ public class SearchControllerIntegrationTest extends DaoControllerTest {
             .andExpect(jsonPath("$.responseCode").value(500))
             .andExpect(jsonPath("$.message").value("Search result size must be less than 100"));
 
+    this.mockMvc.perform(post(searchUrl + "/group").with(httpBasic(user, password)).with(csrf()).contentType(MediaType.parseMediaType("application/json;charset=UTF-8")).content(SearchIntegrationTest.groupByQuery))
+            .andExpect(status().isOk())
+            .andExpect(content().contentType(MediaType.parseMediaType("application/json;charset=UTF-8")))
+            .andExpect(jsonPath("$.*", hasSize(2)))
+            .andExpect(jsonPath("$.groupedBy").value("is_alert"))
+            .andExpect(jsonPath("$.groupResults.*", hasSize(1)))
+            .andExpect(jsonPath("$.groupResults[0].*", hasSize(5)))
+            .andExpect(jsonPath("$.groupResults[0].key").value("is_alert_value"))
+            .andExpect(jsonPath("$.groupResults[0].total").value(10))
+            .andExpect(jsonPath("$.groupResults[0].groupedBy").value("latitude"))
+            .andExpect(jsonPath("$.groupResults[0].groupResults.*", hasSize(1)))
+            .andExpect(jsonPath("$.groupResults[0].groupResults[0].*", hasSize(3)))
+            .andExpect(jsonPath("$.groupResults[0].groupResults[0].key").value("latitude_value"))
+            .andExpect(jsonPath("$.groupResults[0].groupResults[0].total").value(10))
+            .andExpect(jsonPath("$.groupResults[0].groupResults[0].score").value(50));
+
     this.mockMvc.perform(post(searchUrl + "/column/metadata").with(httpBasic(user, password)).with(csrf()).contentType(MediaType.parseMediaType("application/json;charset=UTF-8")).content("[\"bro\",\"snort\"]"))
             .andExpect(status().isOk())
             .andExpect(content().contentType(MediaType.parseMediaType("application/json;charset=UTF-8")))

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/SensorParserConfigControllerIntegrationTest.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/SensorParserConfigControllerIntegrationTest.java b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/SensorParserConfigControllerIntegrationTest.java
index ffe6b9a..5cb0ba5 100644
--- a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/SensorParserConfigControllerIntegrationTest.java
+++ b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/controller/SensorParserConfigControllerIntegrationTest.java
@@ -21,6 +21,7 @@ import java.io.FileInputStream;
 import java.util.HashMap;
 import org.adrianwalker.multilinestring.Multiline;
 import org.apache.commons.io.FileUtils;
+import org.apache.metron.common.configuration.SensorParserConfig;
 import org.apache.metron.bundles.BundleSystem;
 import org.apache.metron.bundles.util.BundleProperties;
 import org.apache.metron.rest.MetronRestConstants;
@@ -41,6 +42,7 @@ import org.springframework.web.context.WebApplicationContext;
 
 import java.io.File;
 import java.io.IOException;
+import java.lang.reflect.Method;
 
 import static org.apache.metron.rest.MetronRestConstants.TEST_PROFILE;
 import static org.hamcrest.Matchers.hasSize;
@@ -212,11 +214,17 @@ public class SensorParserConfigControllerIntegrationTest {
     cleanFileSystem();
     this.sensorParserConfigService.delete("jsonTest");
     this.sensorParserConfigService.delete("squidTest");
-
+    Method[] method = SensorParserConfig.class.getMethods();
+    int numFields = 0;
+    for(Method m : method) {
+      if(m.getName().startsWith("set")) {
+        numFields++;
+      }
+    }
     this.mockMvc.perform(post(sensorParserConfigUrl).with(httpBasic(user, password)).with(csrf()).contentType(MediaType.parseMediaType("application/json;charset=UTF-8")).content(squidJson))
             .andExpect(status().isCreated())
             .andExpect(content().contentType(MediaType.parseMediaType("application/json;charset=UTF-8")))
-            .andExpect(jsonPath("$.*", hasSize(10)))
+            .andExpect(jsonPath("$.*", hasSize(numFields)))
             .andExpect(jsonPath("$.parserClassName").value("org.apache.metron.parsers.grok.GrokParser"))
             .andExpect(jsonPath("$.sensorTopic").value("squidTest"))
             .andExpect(jsonPath("$.parserConfig.grokPath").value("target/patterns/squidTest"))
@@ -231,7 +239,7 @@ public class SensorParserConfigControllerIntegrationTest {
     this.mockMvc.perform(get(sensorParserConfigUrl + "/squidTest").with(httpBasic(user,password)))
             .andExpect(status().isOk())
             .andExpect(content().contentType(MediaType.parseMediaType("application/json;charset=UTF-8")))
-            .andExpect(jsonPath("$.*", hasSize(10)))
+            .andExpect(jsonPath("$.*", hasSize(numFields)))
             .andExpect(jsonPath("$.parserClassName").value("org.apache.metron.parsers.grok.GrokParser"))
             .andExpect(jsonPath("$.sensorTopic").value("squidTest"))
             .andExpect(jsonPath("$.parserConfig.grokPath").value("target/patterns/squidTest"))
@@ -260,7 +268,7 @@ public class SensorParserConfigControllerIntegrationTest {
     this.mockMvc.perform(post(sensorParserConfigUrl).with(httpBasic(user, password)).with(csrf()).contentType(MediaType.parseMediaType("application/json;charset=UTF-8")).content(jsonMapJson))
             .andExpect(status().isCreated())
             .andExpect(content().contentType(MediaType.parseMediaType("application/json;charset=UTF-8")))
-            .andExpect(jsonPath("$.*", hasSize(10)))
+            .andExpect(jsonPath("$.*", hasSize(numFields)))
             .andExpect(jsonPath("$.parserClassName").value("org.apache.metron.parsers.json.JSONMapParser"))
             .andExpect(jsonPath("$.sensorTopic").value("jsonTest"))
             .andExpect(jsonPath("$.readMetadata").value("true"))
@@ -270,7 +278,7 @@ public class SensorParserConfigControllerIntegrationTest {
     this.mockMvc.perform(post(sensorParserConfigUrl).with(httpBasic(user, password)).with(csrf()).contentType(MediaType.parseMediaType("application/json;charset=UTF-8")).content(jsonMapJson))
             .andExpect(status().isOk())
             .andExpect(content().contentType(MediaType.parseMediaType("application/json;charset=UTF-8")))
-            .andExpect(jsonPath("$.*", hasSize(10)))
+            .andExpect(jsonPath("$.*", hasSize(numFields)))
             .andExpect(jsonPath("$.parserClassName").value("org.apache.metron.parsers.json.JSONMapParser"))
             .andExpect(jsonPath("$.sensorTopic").value("jsonTest"))
             .andExpect(jsonPath("$.readMetadata").value("true"))
@@ -373,11 +381,6 @@ public class SensorParserConfigControllerIntegrationTest {
       FileUtils.cleanDirectory(grokTempPath);
       FileUtils.deleteDirectory(grokTempPath);
     }
-    File grokPath = new File(environment.getProperty(MetronRestConstants.GROK_DEFAULT_PATH_SPRING_PROPERTY));
-    if (grokPath.exists()) {
-      FileUtils.cleanDirectory(grokPath);
-      FileUtils.deleteDirectory(grokPath);
-    }
   }
 }
 

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/AlertServiceImplTest.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/AlertServiceImplTest.java b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/AlertServiceImplTest.java
new file mode 100644
index 0000000..c55e0a5
--- /dev/null
+++ b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/AlertServiceImplTest.java
@@ -0,0 +1,69 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.metron.rest.service.impl;
+
+import static org.mockito.Mockito.verify;
+import static org.mockito.Mockito.verifyZeroInteractions;
+import static org.mockito.Mockito.when;
+import static org.powermock.api.mockito.PowerMockito.mock;
+
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import org.apache.metron.rest.MetronRestConstants;
+import org.apache.metron.rest.service.AlertService;
+import org.apache.metron.rest.service.KafkaService;
+import org.junit.Before;
+import org.junit.Test;
+import org.springframework.core.env.Environment;
+
+@SuppressWarnings("unchecked")
+public class AlertServiceImplTest {
+
+  private KafkaService kafkaService;
+  private Environment environment;
+  private AlertService alertService;
+
+  @SuppressWarnings("unchecked")
+  @Before
+  public void setUp() throws Exception {
+    kafkaService = mock(KafkaService.class);
+    environment = mock(Environment.class);
+    alertService = new AlertServiceImpl(kafkaService, environment);
+  }
+
+  @Test
+  public void produceMessageShouldProperlyProduceMessage() throws Exception {
+    String escalationTopic = "escalation";
+    final Map<String, Object> message1 = new HashMap<>();
+    message1.put("field", "value1");
+    final Map<String, Object> message2 = new HashMap<>();
+    message2.put("field", "value2");
+    List<Map<String, Object>> messages = Arrays.asList(message1, message2);
+    when(environment.getProperty(MetronRestConstants.KAFKA_TOPICS_ESCALATION_PROPERTY)).thenReturn(escalationTopic);
+
+    alertService.escalateAlerts(messages);
+
+    String expectedMessage1 = "{\"field\":\"value1\"}";
+    String expectedMessage2 = "{\"field\":\"value2\"}";
+    verify(kafkaService).produceMessage("escalation", expectedMessage1);
+    verify(kafkaService).produceMessage("escalation", expectedMessage2);
+    verifyZeroInteractions(kafkaService);
+  }
+}

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/GrokServiceImplTest.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/GrokServiceImplTest.java b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/GrokServiceImplTest.java
index e6fdbd0..603a0ca 100644
--- a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/GrokServiceImplTest.java
+++ b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/GrokServiceImplTest.java
@@ -17,12 +17,11 @@
  */
 package org.apache.metron.rest.service.impl;
 
-import javax.security.auth.Subject;
+import java.nio.charset.Charset;
 import oi.thekraken.grok.api.Grok;
 import org.apache.commons.io.FileUtils;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.FileSystem;
-import org.apache.hadoop.security.UserGroupInformation;
 import org.apache.metron.rest.RestException;
 import org.apache.metron.rest.model.GrokValidation;
 import org.apache.metron.rest.service.GrokService;
@@ -32,7 +31,6 @@ import org.junit.Rule;
 import org.junit.Test;
 import org.junit.rules.ExpectedException;
 import org.junit.runner.RunWith;
-import org.powermock.core.classloader.annotations.PowerMockIgnore;
 import org.powermock.core.classloader.annotations.PrepareForTest;
 import org.powermock.modules.junit4.PowerMockRunner;
 import org.springframework.core.env.Environment;
@@ -200,43 +198,4 @@ public class GrokServiceImplTest {
 
     grokService.validateGrokStatement(grokValidation);
   }
-
-  @Test
-  public void saveTemporaryShouldProperlySaveFile() throws Exception {
-    new File("./target/user1").delete();
-    String statement = "grok statement";
-
-    Authentication authentication = mock(Authentication.class);
-    when(authentication.getName()).thenReturn("user1");
-    SecurityContextHolder.getContext().setAuthentication(authentication);
-    when(environment.getProperty(GROK_TEMP_PATH_SPRING_PROPERTY)).thenReturn("./target");
-
-    File testFile = grokService.saveTemporary(statement, "squid");
-
-    assertEquals(statement, FileUtils.readFileToString(testFile));
-    testFile.delete();
-  }
-
-  @Test
-  public void saveTemporaryShouldWrapExceptionInRestException() throws Exception {
-    exception.expect(RestException.class);
-
-    String statement = "grok statement";
-
-    Authentication authentication = mock(Authentication.class);
-    when(authentication.getName()).thenReturn("user1");
-    SecurityContextHolder.getContext().setAuthentication(authentication);
-    when(environment.getProperty(GROK_TEMP_PATH_SPRING_PROPERTY)).thenReturn("./target");
-    whenNew(FileWriter.class).withParameterTypes(File.class).withArguments(any()).thenThrow(new IOException());
-
-    grokService.saveTemporary(statement, "squid");
-  }
-
-  @Test
-  public void missingGrokStatementShouldThrowRestException() throws Exception {
-    exception.expect(RestException.class);
-    exception.expectMessage("A grokStatement must be provided");
-
-    grokService.saveTemporary(null, "squid");
-  }
 }

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/HdfsServiceImplExceptionTest.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/HdfsServiceImplExceptionTest.java b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/HdfsServiceImplExceptionTest.java
index d11f5a4..7282859 100644
--- a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/HdfsServiceImplExceptionTest.java
+++ b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/HdfsServiceImplExceptionTest.java
@@ -85,7 +85,17 @@ public class HdfsServiceImplExceptionTest {
         when(FileSystem.get(configuration)).thenReturn(fileSystem);
         when(fileSystem.create(new Path(testDir), true)).thenThrow(new IOException());
 
-        hdfsService.write(new Path(testDir), "contents".getBytes(UTF_8));
+        hdfsService.write(new Path(testDir), "contents".getBytes(UTF_8),null, null,null);
+    }
+
+    @Test
+    public void writeShouldThrowIfInvalidPermissions() throws Exception {
+        exception.expect(RestException.class);
+
+        FileSystem fileSystem = mock(FileSystem.class);
+        when(FileSystem.get(configuration)).thenReturn(fileSystem);
+
+        hdfsService.write(new Path(testDir,"test"),"oops".getBytes(UTF_8), "foo", "r-x","r--");
     }
 
     @Test

http://git-wip-us.apache.org/repos/asf/metron/blob/24b668b0/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/HdfsServiceImplTest.java
----------------------------------------------------------------------
diff --git a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/HdfsServiceImplTest.java b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/HdfsServiceImplTest.java
index 005a427..280be63 100644
--- a/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/HdfsServiceImplTest.java
+++ b/metron-interface/metron-rest/src/test/java/org/apache/metron/rest/service/impl/HdfsServiceImplTest.java
@@ -19,7 +19,10 @@ package org.apache.metron.rest.service.impl;
 
 import org.apache.commons.io.FileUtils;
 import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
+import org.apache.hadoop.fs.permission.FsPermission;
+import org.apache.hadoop.fs.permission.FsAction;
 import org.apache.metron.rest.service.HdfsService;
 import org.junit.After;
 import org.junit.Before;
@@ -84,12 +87,23 @@ public class HdfsServiceImplTest {
     @Test
     public void writeShouldProperlyWriteContents() throws Exception {
         String contents = "contents";
-        hdfsService.write(new Path(testDir, "writeTest.txt"), contents.getBytes(UTF_8));
+        hdfsService.write(new Path(testDir, "writeTest.txt"), contents.getBytes(UTF_8),null,null,null);
 
         assertEquals("contents", FileUtils.readFileToString(new File(testDir, "writeTest.txt")));
     }
 
     @Test
+    public void writeShouldProperlyWriteContentsWithPermissions() throws Exception {
+        String contents = "contents";
+        Path thePath = new Path(testDir,"writeTest2.txt");
+        hdfsService.write(thePath, contents.getBytes(UTF_8),"rw-","r-x","r--");
+
+        assertEquals("contents", FileUtils.readFileToString(new File(testDir, "writeTest2.txt")));
+        assertEquals(FileSystem.get(configuration).listStatus(thePath)[0].getPermission().toShort(),
+        new FsPermission(FsAction.READ_WRITE,FsAction.READ_EXECUTE,FsAction.READ).toShort());
+    }
+
+    @Test
     public void deleteShouldProperlyDeleteFile() throws Exception {
         String contents = "contents";
         FileUtils.writeStringToFile(new File(testDir, "deleteTest.txt"), contents);