Indexing Service validates externally received taskId (#9666)

Addresses issues flagged by https://lgtm.com/rules/5970070/
This commit is contained in:
Suneet Saldanha 2020-04-10 10:36:26 -07:00 committed by GitHub
parent 1ced3b33fb
commit 642fe83897
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
2 changed files with 7 additions and 3 deletions

View File

@ -25,6 +25,7 @@ import com.google.common.base.Predicate;
import com.google.common.collect.Iterables;
import com.google.inject.Inject;
import com.sun.jersey.spi.container.ContainerRequest;
import org.apache.druid.indexer.TaskIdUtils;
import org.apache.druid.indexing.common.task.Task;
import org.apache.druid.indexing.overlord.TaskStorageQueryAdapter;
import org.apache.druid.java.util.common.StringUtils;
@ -81,6 +82,7 @@ public class TaskResourceFilter extends AbstractResourceFilter
).getPath()
);
taskId = StringUtils.urlDecode(taskId);
TaskIdUtils.validateId("taskId", taskId);
Optional<Task> taskOptional = taskStorageQueryAdapter.getTask(taskId);
if (!taskOptional.isPresent()) {

View File

@ -27,6 +27,7 @@ import com.google.common.collect.Lists;
import com.google.common.io.ByteSource;
import com.google.inject.Inject;
import com.sun.jersey.spi.container.ResourceFilters;
import org.apache.druid.indexer.TaskIdUtils;
import org.apache.druid.indexing.overlord.TaskRunner;
import org.apache.druid.indexing.overlord.TaskRunnerWorkItem;
import org.apache.druid.indexing.worker.Worker;
@ -184,10 +185,11 @@ public class WorkerResource
@Produces(HttpMediaType.TEXT_PLAIN_UTF8)
@ResourceFilters(StateResourceFilter.class)
public Response doGetLog(
@PathParam("taskid") String taskid,
@PathParam("taskid") String taskId,
@QueryParam("offset") @DefaultValue("0") long offset
)
{
TaskIdUtils.validateId("taskId", taskId);
if (!(taskRunner instanceof TaskLogStreamer)) {
return Response.status(501)
.entity(StringUtils.format(
@ -197,7 +199,7 @@ public class WorkerResource
.build();
}
try {
final Optional<ByteSource> stream = ((TaskLogStreamer) taskRunner).streamTaskLog(taskid, offset);
final Optional<ByteSource> stream = ((TaskLogStreamer) taskRunner).streamTaskLog(taskId, offset);
if (stream.isPresent()) {
return Response.ok(stream.get().openStream()).build();
@ -206,7 +208,7 @@ public class WorkerResource
}
}
catch (IOException e) {
log.warn(e, "Failed to read log for task: %s", taskid);
log.warn(e, "Failed to read log for task: %s", taskId);
return Response.serverError().build();
}
}