timely_communication/allocator/
process.rs
1use std::rc::Rc;
4use std::cell::RefCell;
5use std::sync::{Arc, Mutex};
6use std::any::Any;
7use std::time::Duration;
8use std::collections::{HashMap};
9use std::sync::mpsc::{Sender, Receiver};
10
11use crate::allocator::thread::{ThreadBuilder};
12use crate::allocator::{Allocate, AllocateBuilder, PeerBuilder, Thread};
13use crate::{Push, Pull};
14use crate::buzzer::Buzzer;
15
16pub struct ProcessBuilder {
18 inner: ThreadBuilder,
19 index: usize,
20 peers: usize,
21 channels: Arc<Mutex<HashMap<usize, Box<dyn Any+Send>>>>,
23
24 buzzers_send: Vec<Sender<Buzzer>>,
26 buzzers_recv: Vec<Receiver<Buzzer>>,
27
28 counters_send: Vec<Sender<usize>>,
29 counters_recv: Receiver<usize>,
30}
31
32impl AllocateBuilder for ProcessBuilder {
33 type Allocator = Process;
34 fn build(self) -> Self::Allocator {
35
36 for worker in self.buzzers_send.iter() {
38 let buzzer = Buzzer::default();
39 worker.send(buzzer).expect("Failed to send buzzer");
40 }
41 let mut buzzers = Vec::with_capacity(self.buzzers_recv.len());
42 for worker in self.buzzers_recv.iter() {
43 buzzers.push(worker.recv().expect("Failed to recv buzzer"));
44 }
45
46 Process {
47 inner: self.inner.build(),
48 index: self.index,
49 peers: self.peers,
50 channels: self.channels,
51 buzzers,
52 counters_send: self.counters_send,
53 counters_recv: self.counters_recv,
54 }
55 }
56}
57
58pub struct Process {
60 inner: Thread,
61 index: usize,
62 peers: usize,
63 channels: Arc<Mutex<HashMap<usize, Box<dyn Any+Send>>>>,
65 buzzers: Vec<Buzzer>,
66 counters_send: Vec<Sender<usize>>,
67 counters_recv: Receiver<usize>,
68}
69
70impl Process {
71 pub fn inner(&mut self) -> &mut Thread { &mut self.inner }
73}
74
75impl PeerBuilder for Process {
76 type Peer = ProcessBuilder;
77 fn new_vector(peers: usize, _refill: crate::allocator::BytesRefill) -> Vec<ProcessBuilder> {
79
80 let mut counters_send = Vec::with_capacity(peers);
81 let mut counters_recv = Vec::with_capacity(peers);
82 for _ in 0 .. peers {
83 let (send, recv) = std::sync::mpsc::channel();
84 counters_send.push(send);
85 counters_recv.push(recv);
86 }
87
88 let channels = Arc::new(Mutex::new(HashMap::with_capacity(peers)));
89
90 let (buzzers_send, buzzers_recv) = crate::promise_futures(peers, peers);
92
93 counters_recv
94 .into_iter()
95 .zip(buzzers_send)
96 .zip(buzzers_recv)
97 .enumerate()
98 .map(|(index, ((recv, bsend), brecv))| {
99 ProcessBuilder {
100 inner: ThreadBuilder,
101 index,
102 peers,
103 buzzers_send: bsend,
104 buzzers_recv: brecv,
105 channels: Arc::clone(&channels),
106 counters_send: counters_send.clone(),
107 counters_recv: recv,
108 }
109 })
110 .collect()
111 }
112}
113
114impl Allocate for Process {
115 fn index(&self) -> usize { self.index }
116 fn peers(&self) -> usize { self.peers }
117 fn allocate<T: Any+Send>(&mut self, identifier: usize) -> (Vec<Box<dyn Push<T>>>, Box<dyn Pull<T>>) {
118
119 let mut channels = self.channels.lock().expect("mutex error?");
124
125 let (sends, recv, empty) = {
126
127 let entry = channels.entry(identifier).or_insert_with(|| {
129
130 let mut pushers = Vec::with_capacity(self.peers);
131 let mut pullers = Vec::with_capacity(self.peers);
132 for buzzer in self.buzzers.iter() {
133 let (s, r): (Sender<T>, Receiver<T>) = std::sync::mpsc::channel();
134 pushers.push((Pusher { target: s }, buzzer.clone()));
136 pullers.push(Puller { source: r, current: None });
137 }
138
139 let mut to_box = Vec::with_capacity(pullers.len());
140 for recv in pullers.into_iter() {
141 to_box.push(Some((pushers.clone(), recv)));
142 }
143
144 Box::new(to_box)
145 });
146
147 let vector =
148 entry
149 .downcast_mut::<Vec<Option<(Vec<(Pusher<T>, Buzzer)>, Puller<T>)>>>()
150 .expect("failed to correctly cast channel");
151
152 let (sends, recv) =
153 vector[self.index]
154 .take()
155 .expect("channel already consumed");
156
157 let empty = vector.iter().all(|x| x.is_none());
158
159 (sends, recv, empty)
160 };
161
162 if empty { channels.remove(&identifier); }
165
166 use crate::allocator::counters::ArcPusher as CountPusher;
167 use crate::allocator::counters::Puller as CountPuller;
168
169 let sends =
170 sends.into_iter()
171 .zip(self.counters_send.iter())
172 .map(|((s,b), sender)| CountPusher::new(s, identifier, sender.clone(), b))
173 .map(|s| Box::new(s) as Box<dyn Push<T>>)
174 .collect::<Vec<_>>();
175
176 let recv = Box::new(CountPuller::new(recv, identifier, Rc::clone(self.inner.events()))) as Box<dyn Pull<T>>;
177
178 (sends, recv)
179 }
180
181 fn events(&self) -> &Rc<RefCell<Vec<usize>>> {
182 self.inner.events()
183 }
184
185 fn await_events(&self, duration: Option<Duration>) {
186 self.inner.await_events(duration);
187 }
188
189 fn receive(&mut self) {
190 let mut events = self.inner.events().borrow_mut();
191 while let Ok(index) = self.counters_recv.try_recv() {
192 events.push(index);
193 }
194 }
195}
196
197struct Pusher<T> {
199 target: Sender<T>,
200}
201
202impl<T> Clone for Pusher<T> {
203 fn clone(&self) -> Self {
204 Self {
205 target: self.target.clone(),
206 }
207 }
208}
209
210impl<T> Push<T> for Pusher<T> {
211 #[inline] fn push(&mut self, element: &mut Option<T>) {
212 if let Some(element) = element.take() {
213 let _ = self.target.send(element);
216 }
217 }
218}
219
220struct Puller<T> {
222 current: Option<T>,
223 source: Receiver<T>,
224}
225
226impl<T> Pull<T> for Puller<T> {
227 #[inline]
228 fn pull(&mut self) -> &mut Option<T> {
229 self.current = self.source.try_recv().ok();
230 &mut self.current
231 }
232}