Skip to content

Commit

Permalink
lint and fix unused
Browse files Browse the repository at this point in the history
  • Loading branch information
zeroshade committed Oct 22, 2024
1 parent 22b4add commit ed50f25
Show file tree
Hide file tree
Showing 5 changed files with 12 additions and 16 deletions.
5 changes: 2 additions & 3 deletions cpp/src/arrow/c/abi.h
Original file line number Diff line number Diff line change
Expand Up @@ -331,7 +331,7 @@ struct ArrowAsyncProducer {
// defined callbacks, this is intended to be created by the consumer instead.
// The consumer passes this handler to the producer, which in turn uses the
// callbacks to inform the consumer of events in the stream.
struct ArrowAsyncDeviceStreamHandler {
struct ArrowAsyncDeviceStreamHandler {
// Handler for receiving a schema. The passed in stream_schema must be
// released or moved by the handler (producer is giving ownership of the schema to
// the handler, but not ownership of the top level object itself).
Expand Down Expand Up @@ -359,8 +359,7 @@ struct ArrowAsyncDeviceStreamHandler {
// A producer that receives a non-zero return here should stop producing and eventually
// call release instead.
int (*on_schema)(struct ArrowAsyncDeviceStreamHandler* self,
struct ArrowSchema* stream_schema,
const char* addl_metadata);
struct ArrowSchema* stream_schema, const char* addl_metadata);

// Handler for receiving data. This is called when data is available providing an
// ArrowAsyncTask struct to signify it. The producer indicates the end of the stream
Expand Down
13 changes: 7 additions & 6 deletions cpp/src/arrow/c/bridge.cc
Original file line number Diff line number Diff line change
Expand Up @@ -2715,7 +2715,7 @@ struct AsyncProducer {
ARROW_DISALLOW_COPY_AND_ASSIGN(PrivateTaskData);
};

Status operator()(const std::shared_ptr<RecordBatch>& record) {
Status operator()(const std::shared_ptr<RecordBatch>& record) {
std::unique_lock<std::mutex> lock(state_->mutex_);
if (state_->pending_requests_ == 0) {
state_->cv_.wait(lock, [this]() -> bool {
Expand Down Expand Up @@ -2789,7 +2789,7 @@ struct AsyncProducer {
return ret;
}

struct ArrowAsyncDeviceStreamHandler* handler_;
struct ArrowAsyncDeviceStreamHandler* handler_;
std::shared_ptr<State> state_;
};

Expand All @@ -2798,7 +2798,8 @@ struct AsyncProducer {
Future<AsyncRecordBatchGenerator> CreateAsyncDeviceStreamHandler(
struct ArrowAsyncDeviceStreamHandler* handler, internal::Executor* executor,
uint64_t queue_size, const DeviceMemoryMapper mapper) {
auto iterator = std::make_shared<AsyncRecordBatchIterator>(queue_size, std::move(mapper));
auto iterator =
std::make_shared<AsyncRecordBatchIterator>(queue_size, std::move(mapper));
return AsyncRecordBatchIterator::Make(*iterator, handler)
.Then([executor](std::shared_ptr<AsyncRecordBatchIterator::State> state)
-> Result<AsyncRecordBatchGenerator> {
Expand Down Expand Up @@ -2831,9 +2832,9 @@ Future<> ExportAsyncRecordBatchReader(
int status = handler->on_next_task(handler, nullptr, nullptr);
handler->release(handler);
if (status != 0) {
return Status::UnknownError(
"Received error from handler::on_next_task ", status);
}
return Status::UnknownError("Received error from handler::on_next_task ",
status);
}
return Future<>::MakeFinished();
},
[handler](const Status status) -> Future<> {
Expand Down
5 changes: 2 additions & 3 deletions cpp/src/arrow/c/bridge.h
Original file line number Diff line number Diff line change
Expand Up @@ -418,7 +418,7 @@ class AsyncErrorDetail : public StatusDetail {
: code_(code), message_(std::move(message)), metadata_(std::move(metadata)) {}
const char* type_id() const override { return "AsyncErrorDetail"; }
std::string ToString() const override { return message_; }

int code() const { return code_; }
const std::string& ErrorMetadata() const { return metadata_; }

private:
Expand Down Expand Up @@ -455,8 +455,7 @@ class Executor;
ARROW_EXPORT
Future<AsyncRecordBatchGenerator> CreateAsyncDeviceStreamHandler(
struct ArrowAsyncDeviceStreamHandler* handler, internal::Executor* executor,
uint64_t queue_size = 5,
const DeviceMemoryMapper mapper = DefaultDeviceMemoryMapper);
uint64_t queue_size = 5, const DeviceMemoryMapper mapper = DefaultDeviceMemoryMapper);

/// \brief Export an AsyncGenerator of record batches using a provided handler
///
Expand Down
3 changes: 1 addition & 2 deletions cpp/src/arrow/c/bridge_test.cc
Original file line number Diff line number Diff line change
Expand Up @@ -5363,8 +5363,7 @@ TEST_F(TestAsyncDeviceArrayStreamRoundTrip, Simple) {
ASSERT_FALSE(fut_gen.is_finished());

ASSERT_OK_AND_ASSIGN(auto fut, internal::GetCpuThreadPool()->Submit([&]() {
return ExportAsyncRecordBatchReader(orig_schema,
MakeVectorGenerator(batches),
return ExportAsyncRecordBatchReader(orig_schema, MakeVectorGenerator(batches),
device->device_type(), &handler);
}));

Expand Down
2 changes: 0 additions & 2 deletions cpp/src/arrow/record_batch.h
Original file line number Diff line number Diff line change
Expand Up @@ -298,14 +298,12 @@ struct ARROW_EXPORT RecordBatchWithMetadata {
std::shared_ptr<KeyValueMetadata> custom_metadata;
};


template <>
struct IterationTraits<RecordBatchWithMetadata> {
static RecordBatchWithMetadata End() { return {nullptr, nullptr}; }
static bool IsEnd(const RecordBatchWithMetadata& val) { return val.batch == nullptr; }
};


/// \brief Abstract interface for reading stream of record batches
class ARROW_EXPORT RecordBatchReader {
public:
Expand Down

0 comments on commit ed50f25

Please sign in to comment.