Both a threaded or async i/o based implementation should be possible
using the same API. The current RPC API is synchronous, but we should
be able to add a single async API that permits multiple implementation
strategies.
Doug
On 06/15/2010 05:14 PM, FURUHASHI Sadayuki wrote:
Doug,
I see. I understood what you say. I think additional threads or pooled threads
are required for each I/Os, so Future should be RunnableFuture like following:
Client c1 = new Client("server1:1010");
Client c2 = new Client("server2:1010");
Client c3 = new Client("server3:1010");
// sends queries to each servers.
RunnableFuture<ResultChunk> f1 = c1.DoIt(query1, data1); // Since the
data is rather large,
RunnableFuture<ResultChunk> f2 = c2.DoIt(query2, data2); // asynchronous
(parallel) sending
RunnableFuture<ResultChunk> f3 = c3.DoIt(query3, data3); // and
receiving are wished.
// does I/Os on thread pool
ExecutorService pool = Executors.newFixedThreadPool(3);
pool.submit(f1);
pool.submit(f2);
pool.submit(f3);
try {
// gathers results.
ResultChunk result1 = f1.get();
ResultChunk result2 = f2.get();
ResultChunk result3 = f3.get();
// merges the results into one object.
Result result = new Result.merge(result1, result2, result3);
} finally {
pool.shutdown();
}
Although it will be acceptable on many cases, threads spend more resources.
Single-threaded event-driven I/O will be better as a client library. Codes will
be like following for that:
// sends queries to each servers.
FutureWithIO<ResultChunk> f1 = c1.DoIt(query1, data1); // Since the data
is rather large,
FutureWithIO<ResultChunk> f2 = c2.DoIt(query2, data2); // asynchronous
(parallel) sending
FutureWithIO<ResultChunk> f3 = c3.DoIt(query3, data3); // and receiving
are wished.
// introduces FutureWithIO class and IOMultiplexer class for event-driven
parallel I/O.
IOMultiplexer multiplexer = new IOMultiplexer();
multiplexer.add(f1);
multiplexer.add(f2);
multiplexer.add(f3);
// proceeds I/Os using Selector.
multiplexer.run();
// gathers results.
ResultChunk result1 = f1.get();
ResultChunk result2 = f2.get();
ResultChunk result3 = f3.get();
Thanks,
Sadayuki
On 2010/06/15, at 16:10, Doug Cutting wrote:
Sadayuki,
Couldn't a Transciever implementation return a Future immediately, so that the
requests could be sent in parallel?
Doug
On 06/15/2010 03:39 PM, FURUHASHI Sadayuki wrote:
Doug,
Thank you for your reply. I read codes and found that provided idea seems good.
But I guess that I/Os on f2 and f3 don't proceed while proceeding I/O on f1. As
a result, it might not utilize maximum network bandwidth of the servers thus
latency will increase.
I mean, I/Os seem not to run in parallel. Here is the picture:
provided idea will become like:
request 1 --send--> --receive--> // Sending and receiving
won't proceed in parallel.
request 2 --send--> --receive-->
request 3 --send--> --receive-->
server 1 ---doit-->
server 2 ---doit-->
server 3 ---doit-->
ideal sequence is:
request 1 --send--> --receive--> // Because the
network bandwidth of client is
request 2 -----send-----> ----receive---> // limited,
paralleled sending and receiving take
request 3 -------send--------> ----receive----> // longer time
compared with non-paralleled one.
server 1 ---doit-->
server 2 ---doit--> | |
server 3 ---doit--> +-------------+
| | latency increases
+------------------------------------+
I/O in parallel
This becomes problem only when data is large, and there are many active clients
than servers. I found that asynchronous calls of Thrift has similar design.
I'll try to test how much performance changes in actuality using another system.
Thanks,
Sadayuki
--
Sadayuki Furuhashi
[email protected]
http://msgpack.sourceforge.net/
http://kumofs.sourceforge.net/
On 2010/06/15, at 11:51, Doug Cutting wrote:
Sadayuki,
Avro does not currently support such an API, but it might not be difficult to
add one and it would be very useful to have.
The Transciever API might to be extended with a method like:
Future<List<ByteBuffer>> transceiveFuture(List<ByteBuffer>);
and the Requestor API might add a method like:
Future<Object> requestFuture(...)
If you are interested in pursuing this, please file an issue in Jira.
Thanks,
Doug
On 06/13/2010 09:35 PM, FURUHASHI Sadayuki wrote:
Hi,
I'm working on distributed systems and looking for flexible RPC system, then I
found Protocol Buffers, Thrift and Avro.
I'd like to compare and test these recently proposed systems.
Our system requires clients to communicate with other multiple servers in
parallel.
Here is the picture:
+----> Server 1 \
|
Client<----+----> Server 2 -- Servers proceed requirests in parallel,
| then the Client gathers the results.
+----> Server 3 /
The codes will be like following:
Client c1 = new Client("server1:1010");
Client c2 = new Client("server2:1010");
Client c3 = new Client("server3:1010");
// sends queries to each servers.
Future<ResultChunk> f1 = c1.DoIt(query1, data1); // Since the data is
rather large,
Future<ResultChunk> f2 = c2.DoIt(query2, data2); // asynchronous
(parallel) sending
Future<ResultChunk> f3 = c3.DoIt(query3, data3); // and receiving are
wished.
// gatehrs results.
ResultChunk result1 = f1.get();
ResultChunk result2 = f2.get();
ResultChunk result3 = f3.get();
// merges the results into one object.
Result result = new Result.merge(result1, result2, result3);
My question is that Avro supports these feature? Or is there plan to implement
it?
I couldn't find them on the source code. Design of RPC is still in under
construction?
Thanks for any pointers,
Sadayuki
--
Sadayuki Furuhashi
[email protected]