Skip to content
Open
Show file tree
Hide file tree
Changes from all commits
Commits
Show all changes
67 commits
Select commit Hold shift + click to select a range
1ad898b
Use the official aws-sdk instead of jet3t (#5382)
jihoonson Mar 21, 2018
0851f22
Expanded documentation for DataSketches aggregators. (#5513)
gianm Mar 22, 2018
ef21ce5
Add graceful shutdown timeout for Jetty (#5429)
drcrallen Mar 23, 2018
ec17a44
Add result level caching to Brokers (#5028)
a2l007 Mar 24, 2018
ea30c05
Adding ParserSpec for Influx Line Protocol (#5440)
njhartwell Mar 26, 2018
50e0e7f
Correct lookup documentation (#5537)
clintropolis Mar 27, 2018
db508cf
[docs] fix invalid example json (#5547)
dyanarose Mar 28, 2018
024e0a9
Respect forceHashAggregation in queryContext (#5533)
jihoonson Mar 28, 2018
912adcc
ArrayAggregation: Use long to avoid overflow (#5544)
niketh Mar 28, 2018
81be1b3
this will fix it (#5549)
clintropolis Mar 29, 2018
8878a7f
Replace guava Charsets with native java StandardCharsets (#5545)
yuppie-flu Mar 29, 2018
30fc4d3
Coordinator balancer move then drop fix (#5528)
clintropolis Mar 29, 2018
0c4598c
Fix typo in avatica java client code documenation (#5553)
amalakar Mar 29, 2018
05547e2
Fix SQLMetadataSegmentManager to allow succesive start and stop (#5554)
jihoonson Mar 30, 2018
6feac20
Coordinator primary segment assignment fix (#5532)
clintropolis Apr 2, 2018
723f7ac
Add support for task reports, upload reports to deep storage (#5524)
jon-wei Apr 2, 2018
a81ae99
add 'stopped' check and handling to HttpLoadQueuePeon load and drop s…
clintropolis Apr 3, 2018
f31dba6
Coordinator drop segment selection through cost balancer (#5529)
clintropolis Apr 3, 2018
f0a94f5
Remove unused config (#5564)
niketh Apr 3, 2018
7239f56
Fix NPE in RemoteTaskRunner when some tasks in ZooKeeper but not in O…
jihoonson Apr 4, 2018
818091e
Add overlord unsecured paths to coordinator when using combined servi…
jon-wei Apr 5, 2018
969342c
More error reporting and stats for ingestion tasks (#5418)
jon-wei Apr 6, 2018
270fd1e
Allow getDomain to return disjointed intervals (#5570)
niketh Apr 6, 2018
371c672
Adding feature thetaSketchConstant to do some set operation in PostAg…
lssenthilkumar Apr 6, 2018
ddd23a1
Fix taskDuration docs for KafkaIndexingService (#5572)
dylwylie Apr 6, 2018
7238576
Add doc for automatic pendingSegments (#5565)
jihoonson Apr 6, 2018
298ed17
Fix indexTask to respect forceExtendableShardSpecs (#5509)
jihoonson Apr 6, 2018
c228eed
Update sql.md (#5519)
aleksi75 Apr 6, 2018
b86ed99
Deprecate spark2 profile in pom.xml (#5581)
drcrallen Apr 6, 2018
5ab1766
CompressionUtils: Add support for decompressing xz, bz2, zip. (#5586)
gianm Apr 6, 2018
ea4f854
revert lambda conversion to fix occasional jvm error (#5591)
clintropolis Apr 6, 2018
ad6f234
Update lookups-cached-global.md (#5525)
aleksi75 Apr 6, 2018
685f406
DoublesSketchModule: Fix serde for DoublesSketchMergeAggregatorFactor…
gianm Apr 7, 2018
3a5d516
Add missing type for MapVirtualColumn (#5598)
jihoonson Apr 10, 2018
ee37ff2
CompressionUtils: Make gzipInputStream public once again. (#5590)
gianm Apr 10, 2018
80fa509
Fix Kerberos Authentication failing requests without cookies and excl…
nishantmonu51 Apr 10, 2018
ff27c54
SQL: Remove useless boolean CASTs in filters. (#5619)
gianm Apr 10, 2018
b32aad9
Fix some broken links in druid docs (#5622)
nishantmonu51 Apr 11, 2018
afa75e0
change header in overlord console; minor querydoc change (#5625)
Apr 11, 2018
e6efd75
Add config to allow setting up custom unsecured paths for druid nodes…
nishantmonu51 Apr 12, 2018
72d6dcd
ParallelCombiner: Fix buffer leak on exception in "combine". (#5630)
gianm Apr 12, 2018
d709d1a
correct overlord console header. Because it's not the coordinator con…
Apr 12, 2018
48c1a1e
change header from Data Schema to Ingestion Spec (#5631)
Apr 12, 2018
19f3599
Add getters for AlertEvent (#5522)
a2l007 Apr 12, 2018
dbea5cb
Load rules should honor partial overlap (#5595)
palanieppan-m Apr 12, 2018
d0400a0
SegmentWithState: Add toString method. (#5635)
gianm Apr 12, 2018
df51a7b
Fix HTTP OPTIONS request auth handling (#5615)
jon-wei Apr 12, 2018
e91add6
Fix coordinator loadStatus performance (#5632)
jon-wei Apr 12, 2018
882b172
Revert "Fix HTTP OPTIONS request auth handling (#5615)" (#5637)
jon-wei Apr 12, 2018
124c89e
Replace EmittedBatchCounter and UpdateCounter with ConcurrentAwaitabl…
leventov Apr 13, 2018
f349e03
Fix NPE in compactionTask (#5613)
jihoonson Apr 13, 2018
6b3bde0
Fix granularitySpec doc (#5647)
jihoonson Apr 16, 2018
d0b66a6
Fix HTTP OPTIONS request auth handling (#5638)
jon-wei Apr 17, 2018
fbf3fc1
Timeseries: Add "grandTotal" option. (#5640)
gianm Apr 17, 2018
8e441cd
Fix cache bug in stats module (#5650)
drcrallen Apr 17, 2018
a7ba2bf
Detailed error message when unable to create temp dir (#5648)
yuppie-flu Apr 17, 2018
5d09f76
topN: Fix caching of Float dimension values. (#5653)
gianm Apr 17, 2018
15f4ab2
Expose noop filter to users (#5597)
scrawfor Apr 18, 2018
ca3f833
Fix coordinator's dataSource api with full parameter (#5662)
jihoonson Apr 20, 2018
a3a9ada
Add GenericWhitespace checkstyle check (#5668)
leventov Apr 23, 2018
55b003e
Fix loadstatus?full double counting expected segments (#5667)
dclim Apr 23, 2018
73da742
Timeseries results are incoherent for case interval is out of range a…
b-slim Apr 23, 2018
fd76af9
remove old prod cluster config link (#5676)
Caroline1000 Apr 24, 2018
2e76012
Allow GCS data segment killer to delete if present (#5675)
drcrallen Apr 24, 2018
034a0aa
Fix wrong null check in TaskStatusPlus (#5678)
jihoonson Apr 24, 2018
dc786eb
SQL: Remove some unused code. (#5690)
gianm Apr 24, 2018
c42ac08
Support conversion of pair returned by first/last aggregators to numb…
kskalski Apr 25, 2018
File filter

Filter by extension

Filter by extension


Conversations
Failed to load comments.
Loading
Jump to
The table of contents is too big for display.
Diff view
Diff view
  •  
  •  
  •  
1 change: 1 addition & 0 deletions .travis.yml
Original file line number Diff line number Diff line change
Expand Up @@ -47,6 +47,7 @@ matrix:
- sudo: false
env:
- NAME="other modules test"
- AWS_REGION=us-east-1 # set a aws region for unit tests
install: echo "MAVEN_OPTS='-Xmx3000m'" > ~/.mavenrc && mvn install -q -ff -DskipTests -B
before_script:
- unset _JAVA_OPTIONS
Expand Down
3 changes: 3 additions & 0 deletions api/src/main/java/io/druid/data/input/Rows.java
Original file line number Diff line number Diff line change
Expand Up @@ -23,6 +23,7 @@
import com.google.common.collect.ImmutableSortedSet;
import com.google.common.collect.Maps;
import com.google.common.primitives.Longs;
import io.druid.java.util.common.Pair;
import io.druid.java.util.common.StringUtils;
import io.druid.java.util.common.parsers.ParseException;

Expand Down Expand Up @@ -116,6 +117,8 @@ public static Number objectToNumber(final String name, final Object inputValue)
catch (Exception e) {
throw new ParseException(e, "Unable to parse value[%s] for field[%s]", inputValue, name);
}
} else if (inputValue instanceof Pair) {
return objectToNumber(name, ((Pair) inputValue).rhs);
} else {
throw new ParseException("Unknown type[%s] for field", inputValue.getClass(), inputValue);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -72,7 +72,7 @@ public List<InputRow> parseBatch(Map<String, Object> theMap)
}
}
catch (Exception e) {
throw new ParseException(e, "Unparseable timestamp found!");
throw new ParseException(e, "Unparseable timestamp found! Event: %s", theMap);
}

return ImmutableList.of(new MapBasedInputRow(timestamp.getMillis(), dimensions, theMap));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,7 +21,6 @@

import com.fasterxml.jackson.annotation.JsonCreator;
import com.fasterxml.jackson.annotation.JsonProperty;
import com.google.common.base.Charsets;
import com.google.common.base.Preconditions;
import com.google.common.collect.Iterators;
import io.druid.data.input.ByteBufferInputRowParser;
Expand All @@ -36,14 +35,15 @@
import java.nio.charset.Charset;
import java.nio.charset.CoderResult;
import java.nio.charset.CodingErrorAction;
import java.nio.charset.StandardCharsets;
import java.util.List;
import java.util.Map;

/**
*/
public class StringInputRowParser implements ByteBufferInputRowParser
{
private static final Charset DEFAULT_CHARSET = Charsets.UTF_8;
private static final Charset DEFAULT_CHARSET = StandardCharsets.UTF_8;

private final ParseSpec parseSpec;
private final MapInputRowParser mapParser;
Expand Down
4 changes: 2 additions & 2 deletions api/src/main/java/io/druid/guice/JsonConfigProvider.java
Original file line number Diff line number Diff line change
Expand Up @@ -130,8 +130,8 @@ public static <T> void bind(
Key<Supplier<T>> supplierKey
)
{
binder.bind(supplierKey).toProvider((Provider) of(propertyBase, clazz)).in(LazySingleton.class);
binder.bind(instanceKey).toProvider(new SupplierProvider<T>(supplierKey));
binder.bind(supplierKey).toProvider(of(propertyBase, clazz)).in(LazySingleton.class);
binder.bind(instanceKey).toProvider(new SupplierProvider<>(supplierKey));
}

@SuppressWarnings("unchecked")
Expand Down
28 changes: 28 additions & 0 deletions api/src/main/java/io/druid/indexer/IngestionState.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,28 @@
/*
* Licensed to Metamarkets Group Inc. (Metamarkets) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Metamarkets 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 io.druid.indexer;

public enum IngestionState
{
NOT_STARTED,
DETERMINE_PARTITIONS,
BUILD_SEGMENTS,
COMPLETED
}
29 changes: 29 additions & 0 deletions api/src/main/java/io/druid/indexer/TaskMetricsGetter.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,29 @@
/*
* Licensed to Metamarkets Group Inc. (Metamarkets) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Metamarkets 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 io.druid.indexer;

import java.util.List;
import java.util.Map;

public interface TaskMetricsGetter
{
List<String> getKeys();
Map<String, Number> getTotalMetrics();
}
47 changes: 47 additions & 0 deletions api/src/main/java/io/druid/indexer/TaskMetricsUtils.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,47 @@
/*
* Licensed to Metamarkets Group Inc. (Metamarkets) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Metamarkets 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 io.druid.indexer;

import com.google.common.collect.Maps;

import java.util.Map;

public class TaskMetricsUtils
{
public static final String ROWS_PROCESSED = "rowsProcessed";
public static final String ROWS_PROCESSED_WITH_ERRORS = "rowsProcessedWithErrors";
public static final String ROWS_UNPARSEABLE = "rowsUnparseable";
public static final String ROWS_THROWN_AWAY = "rowsThrownAway";

public static Map<String, Object> makeIngestionRowMetrics(
long rowsProcessed,
long rowsProcessedWithErrors,
long rowsUnparseable,
long rowsThrownAway
)
{
Map<String, Object> metricsMap = Maps.newHashMap();
metricsMap.put(ROWS_PROCESSED, rowsProcessed);
metricsMap.put(ROWS_PROCESSED_WITH_ERRORS, rowsProcessedWithErrors);
metricsMap.put(ROWS_UNPARSEABLE, rowsUnparseable);
metricsMap.put(ROWS_THROWN_AWAY, rowsThrownAway);
return metricsMap;
}
}
73 changes: 41 additions & 32 deletions api/src/main/java/io/druid/indexer/TaskStatusPlus.java
Original file line number Diff line number Diff line change
Expand Up @@ -38,6 +38,9 @@ public class TaskStatusPlus
private final TaskLocation location;
private final String dataSource;

@Nullable
private final String errorMsg;

@JsonCreator
public TaskStatusPlus(
@JsonProperty("id") String id,
Expand All @@ -47,20 +50,22 @@ public TaskStatusPlus(
@JsonProperty("statusCode") @Nullable TaskState state,
@JsonProperty("duration") @Nullable Long duration,
@JsonProperty("location") TaskLocation location,
@JsonProperty("dataSource") String dataSource
@JsonProperty("dataSource") @Nullable String dataSource, // nullable for backward compatibility
@JsonProperty("errorMsg") @Nullable String errorMsg
)
{
if (state != null && state.isComplete()) {
Preconditions.checkNotNull(duration, "duration");
}
this.id = Preconditions.checkNotNull(id, "id");
this.type = Preconditions.checkNotNull(type, "type");
this.type = type;
this.createdTime = Preconditions.checkNotNull(createdTime, "createdTime");
this.queueInsertionTime = Preconditions.checkNotNull(queueInsertionTime, "queueInsertionTime");
this.state = state;
this.duration = duration;
this.location = Preconditions.checkNotNull(location, "location");
this.dataSource = dataSource;
this.errorMsg = errorMsg;
}

@JsonProperty
Expand Down Expand Up @@ -108,49 +113,53 @@ public TaskLocation getLocation()
return location;
}

@JsonProperty
public String getDataSource()
{
return dataSource;
}

@Nullable
@JsonProperty("errorMsg")
public String getErrorMsg()
{
return errorMsg;
}

@Override
public boolean equals(Object o)
{
if (this == o) {
return true;
}

if (o == null || getClass() != o.getClass()) {
return false;
}

final TaskStatusPlus that = (TaskStatusPlus) o;
if (!id.equals(that.id)) {
return false;
}
if (!type.equals(that.type)) {
return false;
}
if (!createdTime.equals(that.createdTime)) {
return false;
}
if (!queueInsertionTime.equals(that.queueInsertionTime)) {
return false;
}
if (!Objects.equals(state, that.state)) {
return false;
}
if (!Objects.equals(duration, that.duration)) {
return false;
}
return location.equals(that.location);
TaskStatusPlus that = (TaskStatusPlus) o;
return Objects.equals(getId(), that.getId()) &&
Objects.equals(getType(), that.getType()) &&
Objects.equals(getCreatedTime(), that.getCreatedTime()) &&
Objects.equals(getQueueInsertionTime(), that.getQueueInsertionTime()) &&
getState() == that.getState() &&
Objects.equals(getDuration(), that.getDuration()) &&
Objects.equals(getLocation(), that.getLocation()) &&
Objects.equals(getDataSource(), that.getDataSource()) &&
Objects.equals(getErrorMsg(), that.getErrorMsg());
}

@Override
public int hashCode()
{
return Objects.hash(id, type, createdTime, queueInsertionTime, state, duration, location);
}

@JsonProperty
public String getDataSource()
{
return dataSource;
return Objects.hash(
getId(),
getType(),
getCreatedTime(),
getQueueInsertionTime(),
getState(),
getDuration(),
getLocation(),
getDataSource(),
getErrorMsg()
);
}

}
7 changes: 7 additions & 0 deletions api/src/main/java/io/druid/tasklogs/NoopTaskLogs.java
Original file line number Diff line number Diff line change
Expand Up @@ -24,6 +24,7 @@
import io.druid.java.util.common.logger.Logger;

import java.io.File;
import java.io.IOException;

public class NoopTaskLogs implements TaskLogs
{
Expand All @@ -41,6 +42,12 @@ public void pushTaskLog(String taskid, File logFile)
log.info("Not pushing logs for task: %s", taskid);
}

@Override
public void pushTaskReports(String taskid, File reportFile) throws IOException
{
log.info("Not pushing reports for task: %s", taskid);
}

@Override
public void killAll()
{
Expand Down
4 changes: 4 additions & 0 deletions api/src/main/java/io/druid/tasklogs/TaskLogPusher.java
Original file line number Diff line number Diff line change
Expand Up @@ -31,4 +31,8 @@
public interface TaskLogPusher
{
void pushTaskLog(String taskid, File logFile) throws IOException;

default void pushTaskReports(String taskid, File reportFile) throws IOException
{
}
}
5 changes: 5 additions & 0 deletions api/src/main/java/io/druid/tasklogs/TaskLogStreamer.java
Original file line number Diff line number Diff line change
Expand Up @@ -40,4 +40,9 @@ public interface TaskLogStreamer
* @return input supplier for this log, if available from this provider
*/
Optional<ByteSource> streamTaskLog(String taskid, long offset) throws IOException;

default Optional<ByteSource> streamTaskReports(final String taskid) throws IOException
{
return Optional.absent();
}
}
14 changes: 3 additions & 11 deletions api/src/main/java/io/druid/timeline/partition/NoneShardSpec.java
Original file line number Diff line number Diff line change
Expand Up @@ -22,7 +22,7 @@
import com.fasterxml.jackson.annotation.JsonCreator;
import com.fasterxml.jackson.annotation.JsonIgnore;
import com.google.common.collect.ImmutableMap;
import com.google.common.collect.Range;
import com.google.common.collect.RangeSet;
import io.druid.data.input.InputRow;

import java.util.List;
Expand Down Expand Up @@ -71,19 +71,11 @@ public int getPartitionNum()
@Override
public ShardSpecLookup getLookup(final List<ShardSpec> shardSpecs)
{

return new ShardSpecLookup()
{
@Override
public ShardSpec getShardSpec(long timestamp, InputRow row)
{
return shardSpecs.get(0);
}
};
return (long timestamp, InputRow row) -> shardSpecs.get(0);
}

@Override
public Map<String, Range<String>> getDomain()
public Map<String, RangeSet<String>> getDomain()
{
return ImmutableMap.of();
}
Expand Down
4 changes: 2 additions & 2 deletions api/src/main/java/io/druid/timeline/partition/ShardSpec.java
Original file line number Diff line number Diff line change
Expand Up @@ -21,7 +21,7 @@

import com.fasterxml.jackson.annotation.JsonSubTypes;
import com.fasterxml.jackson.annotation.JsonTypeInfo;
import com.google.common.collect.Range;
import com.google.common.collect.RangeSet;
import io.druid.data.input.InputRow;

import java.util.List;
Expand Down Expand Up @@ -50,5 +50,5 @@ public interface ShardSpec
*
* @return map of dimensions to its possible range. Dimensions with unknown possible range are not mapped
*/
Map<String, Range<String>> getDomain();
Map<String, RangeSet<String>> getDomain();
}
Loading