Return-Path: X-Original-To: archive-asf-public-internal@cust-asf2.ponee.io Delivered-To: archive-asf-public-internal@cust-asf2.ponee.io Received: from cust-asf.ponee.io (cust-asf.ponee.io [163.172.22.183]) by cust-asf2.ponee.io (Postfix) with ESMTP id 4A5EE200C8F for ; Fri, 26 May 2017 03:45:14 +0200 (CEST) Received: by cust-asf.ponee.io (Postfix) id 4942D160BD8; Fri, 26 May 2017 01:45:14 +0000 (UTC) Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by cust-asf.ponee.io (Postfix) with SMTP id 7A317160BCA for ; Fri, 26 May 2017 03:45:13 +0200 (CEST) Received: (qmail 97433 invoked by uid 500); 26 May 2017 01:45:12 -0000 Mailing-List: contact issues-help@drill.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@drill.apache.org Delivered-To: mailing list issues@drill.apache.org Received: (qmail 97406 invoked by uid 99); 26 May 2017 01:45:12 -0000 Received: from pnap-us-west-generic-nat.apache.org (HELO spamd4-us-west.apache.org) (209.188.14.142) by apache.org (qpsmtpd/0.29) with ESMTP; Fri, 26 May 2017 01:45:12 +0000 Received: from localhost (localhost [127.0.0.1]) by spamd4-us-west.apache.org (ASF Mail Server at spamd4-us-west.apache.org) with ESMTP id 9EC8BC1309 for ; Fri, 26 May 2017 01:45:11 +0000 (UTC) X-Virus-Scanned: Debian amavisd-new at spamd4-us-west.apache.org X-Spam-Flag: NO X-Spam-Score: -100.002 X-Spam-Level: X-Spam-Status: No, score=-100.002 tagged_above=-999 required=6.31 tests=[RP_MATCHES_RCVD=-0.001, SPF_PASS=-0.001, USER_IN_WHITELIST=-100] autolearn=disabled Received: from mx1-lw-us.apache.org ([10.40.0.8]) by localhost (spamd4-us-west.apache.org [10.40.0.11]) (amavisd-new, port 10024) with ESMTP id 0Vu6-XEUtV30 for ; Fri, 26 May 2017 01:45:09 +0000 (UTC) Received: from mailrelay1-us-west.apache.org (mailrelay1-us-west.apache.org [209.188.14.139]) by mx1-lw-us.apache.org (ASF Mail Server at mx1-lw-us.apache.org) with ESMTP id 8F18161066 for ; Fri, 26 May 2017 01:45:07 +0000 (UTC) Received: from jira-lw-us.apache.org (unknown [207.244.88.139]) by mailrelay1-us-west.apache.org (ASF Mail Server at mailrelay1-us-west.apache.org) with ESMTP id 8C670E0DF0 for ; Fri, 26 May 2017 01:45:06 +0000 (UTC) Received: from jira-lw-us.apache.org (localhost [127.0.0.1]) by jira-lw-us.apache.org (ASF Mail Server at jira-lw-us.apache.org) with ESMTP id A01F12409A for ; Fri, 26 May 2017 01:45:05 +0000 (UTC) Date: Fri, 26 May 2017 01:45:05 +0000 (UTC) From: "ASF GitHub Bot (JIRA)" To: issues@drill.apache.org Message-ID: In-Reply-To: References: Subject: [jira] [Commented] (DRILL-5432) Want a memory format for PCAP files MIME-Version: 1.0 Content-Type: text/plain; charset=utf-8 Content-Transfer-Encoding: 7bit X-JIRA-FingerPrint: 30527f35849b9dde25b450d4833f0394 archived-at: Fri, 26 May 2017 01:45:14 -0000 [ https://issues.apache.org/jira/browse/DRILL-5432?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16025668#comment-16025668 ] ASF GitHub Bot commented on DRILL-5432: --------------------------------------- Github user paul-rogers commented on a diff in the pull request: https://github.com/apache/drill/pull/831#discussion_r118616911 --- Diff: exec/java-exec/src/main/java/org/apache/drill/exec/store/pcap/PcapRecordReader.java --- @@ -0,0 +1,295 @@ +/* + * 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.drill.exec.store.pcap; + +import com.google.common.collect.ImmutableList; +import com.google.common.collect.ImmutableMap; +import org.apache.drill.common.exceptions.ExecutionSetupException; +import org.apache.drill.common.expression.SchemaPath; +import org.apache.drill.common.types.TypeProtos; +import org.apache.drill.common.types.TypeProtos.MajorType; +import org.apache.drill.common.types.TypeProtos.MinorType; +import org.apache.drill.common.types.Types; +import org.apache.drill.exec.exception.SchemaChangeException; +import org.apache.drill.exec.expr.TypeHelper; +import org.apache.drill.exec.ops.OperatorContext; +import org.apache.drill.exec.physical.impl.OutputMutator; +import org.apache.drill.exec.record.MaterializedField; +import org.apache.drill.exec.store.AbstractRecordReader; +import org.apache.drill.exec.store.pcap.decoder.Packet; +import org.apache.drill.exec.store.pcap.decoder.PacketDecoder; +import org.apache.drill.exec.store.pcap.dto.ColumnDto; +import org.apache.drill.exec.store.pcap.schema.PcapTypes; +import org.apache.drill.exec.store.pcap.schema.Schema; +import org.apache.drill.exec.vector.NullableBigIntVector; +import org.apache.drill.exec.vector.NullableIntVector; +import org.apache.drill.exec.vector.NullableTimeStampVector; +import org.apache.drill.exec.vector.NullableVarCharVector; +import org.apache.drill.exec.vector.ValueVector; + +import java.io.FileInputStream; +import java.io.IOException; +import java.io.InputStream; +import java.nio.ByteBuffer; +import java.util.List; +import java.util.Map; + +import static java.nio.charset.StandardCharsets.UTF_8; +import static org.apache.drill.exec.store.pcap.Utils.parseBytesToASCII; + +public class PcapRecordReader extends AbstractRecordReader { + + private OutputMutator output; + + private final PacketDecoder decoder; + private ImmutableList projectedCols; + + private byte[] buffer = new byte[100000]; + private int offset = 0; + private InputStream in; + private int validBytes; + + private static final Map TYPES; + + private static class ProjectedColumnInfo { + ValueVector vv; + ColumnDto pcapColumn; + } + + static { + TYPES = ImmutableMap.builder() + .put(PcapTypes.STRING, MinorType.VARCHAR) + .put(PcapTypes.INTEGER, MinorType.INT) + .put(PcapTypes.LONG, MinorType.BIGINT) + .put(PcapTypes.TIMESTAMP, MinorType.TIMESTAMP) + .build(); + } + + public PcapRecordReader(final String inputPath, + final List projectedColumns) { + try { + this.in = new FileInputStream(inputPath); + this.decoder = getPacketDecoder(); + validBytes = in.read(buffer); + } catch (IOException e) { + throw new RuntimeException("File " + inputPath + " not Found"); + } + setColumns(projectedColumns); + } + + @Override + public void setup(final OperatorContext context, final OutputMutator output) throws ExecutionSetupException { + this.output = output; + } + + @Override + public int next() { + projectedCols = getProjectedColsIfItNull(); + try { + return parsePcapFilesAndPutItToTable(); + } catch (IOException io) { + throw new RuntimeException("Trouble with reading packets in file!"); + } + } + + @Override + public void close() throws Exception { + } + + private PacketDecoder getPacketDecoder() { + try { + return new PacketDecoder(in); + } catch (IOException io) { + throw new RuntimeException("File Not Found or some I/O issue"); + } + } + + private ImmutableList getProjectedColsIfItNull() { + return projectedCols != null ? projectedCols : initCols(new Schema()); + } + + private ImmutableList initCols(final Schema schema) { + ImmutableList.Builder pciBuilder = ImmutableList.builder(); + ColumnDto column; + + for (int i = 0; i < schema.getNumberOfColumns(); i++) { + column = schema.getColumnByIndex(i); + + final String name = column.getColumnName(); + final PcapTypes type = column.getColumnType(); + TypeProtos.MinorType minorType = TYPES.get(type); + + ProjectedColumnInfo pci = getProjectedColumnInfo(column, name, minorType); + pciBuilder.add(pci); + } + return pciBuilder.build(); + } + + private ProjectedColumnInfo getProjectedColumnInfo(final ColumnDto column, + final String name, + final MinorType minorType) { + TypeProtos.MajorType majorType = getMajorType(minorType); + + MaterializedField field = + MaterializedField.create(name, majorType); + + ValueVector vector = + getValueVector(minorType, majorType, field); + + return getProjectedColumnInfo(column, vector); + } + + private ProjectedColumnInfo getProjectedColumnInfo(final ColumnDto column, final ValueVector vector) { + ProjectedColumnInfo pci = new ProjectedColumnInfo(); + pci.vv = vector; + pci.pcapColumn = column; + return pci; + } + + private MajorType getMajorType(final MinorType minorType) { + return Types.optional(minorType); + } + + private ValueVector getValueVector(final MinorType minorType, + final MajorType majorType, + final MaterializedField field) { + try { + + final Class clazz = TypeHelper.getValueVectorClass( + minorType, majorType.getMode()); + ValueVector vector = output.addField(field, clazz); + vector.allocateNew(); + return vector; + + } catch (SchemaChangeException sce) { + throw new NullPointerException("The addition of this field is incompatible with this OutputMutator's capabilities"); --- End diff -- `NullPointerException` --> `IllegalStateException` This is not really a null pointer, but it is an illegal state... > Want a memory format for PCAP files > ----------------------------------- > > Key: DRILL-5432 > URL: https://issues.apache.org/jira/browse/DRILL-5432 > Project: Apache Drill > Issue Type: New Feature > Reporter: Ted Dunning > > PCAP files [1] are the de facto standard for storing network capture data. In security and protocol applications, it is very common to want to extract particular packets from a capture for further analysis. > At a first level, it is desirable to query and filter by source and destination IP and port or by protocol. Beyond that, however, it would be very useful to be able to group packets by TCP session and eventually to look at packet contents. For now, however, the most critical requirement is that we should be able to scan captures at very high speed. > I previously wrote a (kind of working) proof of concept for a PCAP decoder that did lazy deserialization and could traverse hundreds of MB of PCAP data per second per core. This compares to roughly 2-3 MB/s for widely available Apache-compatible open source PCAP decoders. > This JIRA covers the integration and extension of that proof of concept as a Drill file format. > Initial work is available at https://github.com/mapr-demos/drill-pcap-format > [1] https://en.wikipedia.org/wiki/Pcap -- This message was sent by Atlassian JIRA (v6.3.15#6346)